Skip to content
This repository was archived by the owner on Nov 11, 2022. It is now read-only.
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -16,9 +16,10 @@

package com.google.cloud.dataflow.examples;

import com.google.cloud.dataflow.examples.WordCount.WordCountOptions;
import com.google.cloud.dataflow.sdk.Pipeline;
import com.google.cloud.dataflow.sdk.io.TextIO;
import com.google.cloud.dataflow.sdk.options.Default;
import com.google.cloud.dataflow.sdk.options.Description;
import com.google.cloud.dataflow.sdk.options.PipelineOptionsFactory;
import com.google.cloud.dataflow.sdk.testing.DataflowAssert;
import com.google.cloud.dataflow.sdk.transforms.Aggregator;
Expand Down Expand Up @@ -150,6 +151,21 @@ public void processElement(ProcessContext c) {
}
}

/**
* Options supported by {@link DebuggingWordCount}.
*
* <p>Inherits standard configuration options and all options defined in
* {@link WordCount.WordCountOptions}.
*/
public static interface WordCountOptions extends WordCount.WordCountOptions {

@Description("Regex filter pattern to use in DebuggingWordCount. "
+ "Only words matching this pattern will be counted.")
@Default.String("Flourish|stomach")
String getFilterPattern();
void setFilterPattern(String value);
}

public static void main(String[] args) {
WordCountOptions options = PipelineOptionsFactory.fromArgs(args).withValidation()
.as(WordCountOptions.class);
Expand All @@ -158,7 +174,7 @@ public static void main(String[] args) {
PCollection<KV<String, Long>> filteredWords =
p.apply(TextIO.Read.named("ReadLines").from(options.getInputFile()))
.apply(new WordCount.CountWords())
.apply(ParDo.of(new FilterTextFn("Flourish|stomach")));
.apply(ParDo.of(new FilterTextFn(options.getFilterPattern())));

/**
* Concept #4: DataflowAssert is a set of convenient PTransforms in the style of
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -16,9 +16,11 @@

package ${package};

import ${package}.WordCount.WordCountOptions;
import ${package}.WordCount;
import com.google.cloud.dataflow.sdk.Pipeline;
import com.google.cloud.dataflow.sdk.io.TextIO;
import com.google.cloud.dataflow.sdk.options.Default;
import com.google.cloud.dataflow.sdk.options.Description;
import com.google.cloud.dataflow.sdk.options.PipelineOptionsFactory;
import com.google.cloud.dataflow.sdk.testing.DataflowAssert;
import com.google.cloud.dataflow.sdk.transforms.Aggregator;
Expand Down Expand Up @@ -149,6 +151,21 @@ public void processElement(ProcessContext c) {
}
}
}

/**
* Options supported by {@link DebuggingWordCount}.
*
* <p>Inherits standard configuration options and all options defined in
* {@link WordCount.WordCountOptions}.
*/
public static interface WordCountOptions extends WordCount.WordCountOptions {

@Description("Regex filter pattern to use in DebuggingWordCount. "
+ "Only words matching this pattern will be counted.")
@Default.String("Flourish|stomach")
String getFilterPattern();
void setFilterPattern(String value);
}

public static void main(String[] args) {
WordCountOptions options = PipelineOptionsFactory.fromArgs(args).withValidation()
Expand All @@ -158,7 +175,7 @@ public static void main(String[] args) {
PCollection<KV<String, Long>> filteredWords =
p.apply(TextIO.Read.named("ReadLines").from(options.getInputFile()))
.apply(new WordCount.CountWords())
.apply(ParDo.of(new FilterTextFn("Flourish|stomach")));
.apply(ParDo.of(new FilterTextFn(options.getFilterPattern())));

/**
* Concept #4: DataflowAssert is a set of convenient PTransforms in the style of
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -29,15 +29,23 @@
@Hidden
public interface CloudDebuggerOptions {

/**
* Whether to enable the Cloud Debugger snapshot agent for the current job.
*/
/** Whether to enable the Cloud Debugger snapshot agent for the current job. */
@Description("Whether to enable the Cloud Debugger snapshot agent for the current job.")
boolean getEnableCloudDebugger();
void setEnableCloudDebugger(boolean enabled);

@Description("The Cloud Debugger debugee to associate with. This should not be set directly.")
/** The Cloud Debugger debuggee to associate with. This should not be set directly. */
@Description("The Cloud Debugger debuggee to associate with. This should not be set directly.")
@Hidden
@Nullable Debuggee getDebuggee();
void setDebuggee(Debuggee debuggee);

/** The maximum cost (as a ratio of CPU time) allowed for evaluating conditional snapshots. */
@Description(
"The maximum cost (as a ratio of CPU time) allowed for evaluating conditional snapshots. "
+ "Should be a double between 0 and 1. "
+ "Snapshots will be cancelled if evaluating conditions takes more than this ratio of time.")
@Default.Double(0.01)
double getMaxConditionCost();
void setMaxConditionCost(double maxConditionCost);
}