mirror of
https://github.com/google/nomulus.git
synced 2025-05-16 01:17:14 +02:00
Prepare billing pipeline for production
This makes a few cosmetic changes that prepares the pipeline for production. Namely: - Converts file names to include the input yearMonth, mostly mirroring the original invoicing pipeline. - Factors out the yearMonth logic from the reporting module to the more common backend module. We will likely use the default yearMonth logic in other backend tasks (such as spec11 reporting). - Adds the "withTemplateCompatability" flag to the Bigquery read, which allows multiple uses of the same template. - Adds the 'billing' task queue, which retries up to 5 times every 3 minutes, which is about the rate we desire for checking if the pipeline is complete. - Adds a shell 'invoicing upload' class, which tests the retry semantics we want for post-generation work (e-mailing the invoice to crr-tech, and publishing detail reports) While this cl may look big, it's mostly just a refactor and setting up boilerplate needed to frame the upload logic. ------------- Created by MOE: https://github.com/google/moe MOE_MIGRATED_REVID=179849586
This commit is contained in:
parent
53ed6035c4
commit
552ab12314
24 changed files with 436 additions and 115 deletions
|
@ -17,6 +17,7 @@ package google.registry.beam;
|
|||
import google.registry.beam.BillingEvent.InvoiceGroupingKey;
|
||||
import google.registry.beam.BillingEvent.InvoiceGroupingKey.InvoiceGroupingKeyCoder;
|
||||
import google.registry.config.RegistryConfig.Config;
|
||||
import java.io.Serializable;
|
||||
import javax.inject.Inject;
|
||||
import org.apache.beam.runners.dataflow.DataflowRunner;
|
||||
import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions;
|
||||
|
@ -29,6 +30,7 @@ import org.apache.beam.sdk.io.gcp.bigquery.BigQueryIO;
|
|||
import org.apache.beam.sdk.options.Description;
|
||||
import org.apache.beam.sdk.options.PipelineOptionsFactory;
|
||||
import org.apache.beam.sdk.options.ValueProvider;
|
||||
import org.apache.beam.sdk.options.ValueProvider.NestedValueProvider;
|
||||
import org.apache.beam.sdk.transforms.Count;
|
||||
import org.apache.beam.sdk.transforms.MapElements;
|
||||
import org.apache.beam.sdk.transforms.PTransform;
|
||||
|
@ -49,7 +51,7 @@ import org.apache.beam.sdk.values.TypeDescriptors;
|
|||
*
|
||||
* @see <a href="https://cloud.google.com/dataflow/docs/templates/overview">Dataflow Templates</a>
|
||||
*/
|
||||
public class InvoicingPipeline {
|
||||
public class InvoicingPipeline implements Serializable {
|
||||
|
||||
@Inject @Config("projectId") String projectId;
|
||||
@Inject @Config("apacheBeamBucketUrl") String beamBucket;
|
||||
|
@ -66,12 +68,12 @@ public class InvoicingPipeline {
|
|||
|
||||
/** Deploys the invoicing pipeline as a template on GCS, for a given projectID and GCS bucket. */
|
||||
public void deploy() {
|
||||
// We can't store options as a member variable due to serialization concerns.
|
||||
InvoicingPipelineOptions options = PipelineOptionsFactory.as(InvoicingPipelineOptions.class);
|
||||
options.setProject(projectId);
|
||||
options.setRunner(DataflowRunner.class);
|
||||
options.setStagingLocation(beamBucket + "/staging");
|
||||
options.setTemplateLocation(beamBucket + "/templates/invoicing");
|
||||
|
||||
Pipeline p = Pipeline.create(options);
|
||||
|
||||
PCollection<BillingEvent> billingEvents =
|
||||
|
@ -81,9 +83,9 @@ public class InvoicingPipeline {
|
|||
.fromQuery(InvoicingUtils.makeQueryProvider(options.getYearMonth(), projectId))
|
||||
.withCoder(SerializableCoder.of(BillingEvent.class))
|
||||
.usingStandardSql()
|
||||
.withoutValidation());
|
||||
|
||||
applyTerminalTransforms(billingEvents);
|
||||
.withoutValidation()
|
||||
.withTemplateCompatibility());
|
||||
applyTerminalTransforms(billingEvents, options.getYearMonth());
|
||||
p.run();
|
||||
}
|
||||
|
||||
|
@ -92,13 +94,15 @@ public class InvoicingPipeline {
|
|||
*
|
||||
* <p>This is factored out purely to facilitate testing.
|
||||
*/
|
||||
void applyTerminalTransforms(PCollection<BillingEvent> billingEvents) {
|
||||
void applyTerminalTransforms(
|
||||
PCollection<BillingEvent> billingEvents, ValueProvider<String> yearMonthProvider) {
|
||||
billingEvents.apply(
|
||||
"Write events to separate CSVs keyed by registrarId_tld pair", writeDetailReports());
|
||||
"Write events to separate CSVs keyed by registrarId_tld pair",
|
||||
writeDetailReports(yearMonthProvider));
|
||||
|
||||
billingEvents
|
||||
.apply("Generate overall invoice rows", new GenerateInvoiceRows())
|
||||
.apply("Write overall invoice to CSV", writeInvoice());
|
||||
.apply("Write overall invoice to CSV", writeInvoice(yearMonthProvider));
|
||||
}
|
||||
|
||||
/** Transform that converts a {@code BillingEvent} into an invoice CSV row. */
|
||||
|
@ -121,19 +125,23 @@ public class InvoicingPipeline {
|
|||
}
|
||||
|
||||
/** Returns an IO transform that writes the overall invoice to a single CSV file. */
|
||||
private TextIO.Write writeInvoice() {
|
||||
private TextIO.Write writeInvoice(ValueProvider<String> yearMonthProvider) {
|
||||
return TextIO.write()
|
||||
.to(beamBucket + "/results/overall_invoice")
|
||||
.to(
|
||||
NestedValueProvider.of(
|
||||
yearMonthProvider,
|
||||
yearMonth -> String.format("%s/results/CRR-INV-%s", beamBucket, yearMonth)))
|
||||
.withHeader(InvoiceGroupingKey.invoiceHeader())
|
||||
.withoutSharding()
|
||||
.withSuffix(".csv");
|
||||
}
|
||||
|
||||
/** Returns an IO transform that writes detail reports to registrar-tld keyed CSV files. */
|
||||
private TextIO.TypedWrite<BillingEvent, Params> writeDetailReports() {
|
||||
private TextIO.TypedWrite<BillingEvent, Params> writeDetailReports(
|
||||
ValueProvider<String> yearMonthProvider) {
|
||||
return TextIO.<BillingEvent>writeCustomType()
|
||||
.to(
|
||||
InvoicingUtils.makeDestinationFunction(beamBucket + "/results"),
|
||||
InvoicingUtils.makeDestinationFunction(beamBucket + "/results", yearMonthProvider),
|
||||
InvoicingUtils.makeEmptyDestinationParams(beamBucket + "/results"))
|
||||
.withFormatFunction(BillingEvent::toCsv)
|
||||
.withoutSharding()
|
||||
|
|
Loading…
Add table
Add a link
Reference in a new issue