From 46f744714aae4cd45ba6284e2c9414e4bba5cd3e Mon Sep 17 00:00:00 2001 From: Davor Bonaci Date: Mon, 25 Apr 2016 14:16:19 -0700 Subject: [PATCH 1/5] Update pom.xml for java8tests Java8tests module doesn't have sources, only tests. Hence, all dependencies should have scope of test. If not, dependency analysis correctly finds unused dependencies. --- sdks/java/java8tests/pom.xml | 3 +++ 1 file changed, 3 insertions(+) diff --git a/sdks/java/java8tests/pom.xml b/sdks/java/java8tests/pom.xml index cd7174a729fa..f750a1cc74c2 100644 --- a/sdks/java/java8tests/pom.xml +++ b/sdks/java/java8tests/pom.xml @@ -150,18 +150,21 @@ org.apache.beam java-sdk-all ${project.version} + test com.google.guava guava ${guava.version} + test joda-time joda-time ${joda.version} + test From c08f973cb5585885a2da93c66716dec87670ca30 Mon Sep 17 00:00:00 2001 From: Davor Bonaci Date: Mon, 25 Apr 2016 14:30:25 -0700 Subject: [PATCH 2/5] Increase visibility in PAssert and ZipFiles utilities This is needed for package reorganization in runners/google-cloud-dataflow. Those classes will have to move away from org.apache.beam.sdk.* packages. --- .../main/java/org/apache/beam/sdk/testing/PAssert.java | 8 ++++---- .../src/main/java/org/apache/beam/sdk/util/ZipFiles.java | 2 +- 2 files changed, 5 insertions(+), 5 deletions(-) diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/PAssert.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/PAssert.java index f328c5b49f9f..1265acdd9bd4 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/PAssert.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/PAssert.java @@ -98,8 +98,8 @@ public class PAssert { private static final Logger LOG = LoggerFactory.getLogger(PAssert.class); - static final String SUCCESS_COUNTER = "PAssertSuccess"; - static final String FAILURE_COUNTER = "PAssertFailure"; + public static final String SUCCESS_COUNTER = "PAssertSuccess"; + public static final String FAILURE_COUNTER = "PAssertFailure"; private static int assertCount = 0; @@ -576,7 +576,7 @@ public PCollectionView apply(PBegin input) { *

This is generally useful for assertion functions that * are serializable but whose underlying data may not have a coder. */ - static class OneSideInputAssert + public static class OneSideInputAssert extends PTransform implements Serializable { private final transient PTransform> createActual; private final SerializableFunction checkerFn; @@ -647,7 +647,7 @@ public void processElement(ProcessContext c) { * are not serializable, but have coders (provided * by the underlying {@link PCollection}s). */ - static class TwoSideInputAssert + public static class TwoSideInputAssert extends PTransform implements Serializable { private final transient PTransform> createActual; diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/util/ZipFiles.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/util/ZipFiles.java index 6d730278087a..038b9cb6fec9 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/util/ZipFiles.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/util/ZipFiles.java @@ -226,7 +226,7 @@ static void zipDirectory( * @throws IOException the zipping failed, e.g. because the input was not * readable. */ - static void zipDirectory( + public static void zipDirectory( File sourceDirectory, OutputStream outputStream) throws IOException { checkNotNull(sourceDirectory); From 9e19efdf37541ee082bed1ebfd6dd0b154de5f0a Mon Sep 17 00:00:00 2001 From: Davor Bonaci Date: Mon, 25 Apr 2016 15:02:27 -0700 Subject: [PATCH 3/5] Fix a few underlying checkstyle issues in java8 examples --- .../beam/examples/complete/game/GameStats.java | 12 ++++++++---- .../beam/examples/complete/game/HourlyTeamScore.java | 6 ++++-- .../beam/examples/complete/game/LeaderBoard.java | 6 ++++-- 3 files changed, 16 insertions(+), 8 deletions(-) diff --git a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java index d93c2aed8b39..2d1426497b36 100644 --- a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java +++ b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java @@ -207,8 +207,10 @@ static interface Options extends LeaderBoard.Options { c -> c.element().getValue())); tableConfigure.put("window_start", new WriteWindowedToBigQuery.FieldInfo>("STRING", - c -> { IntervalWindow w = (IntervalWindow) c.window(); - return fmt.print(w.start()); })); + c -> { + IntervalWindow w = (IntervalWindow) c.window(); + return fmt.print(w.start()); + })); tableConfigure.put("processing_time", new WriteWindowedToBigQuery.FieldInfo>( "STRING", c -> fmt.print(Instant.now()))); @@ -226,8 +228,10 @@ static interface Options extends LeaderBoard.Options { new HashMap>(); tableConfigure.put("window_start", new WriteWindowedToBigQuery.FieldInfo("STRING", - c -> { IntervalWindow w = (IntervalWindow) c.window(); - return fmt.print(w.start()); })); + c -> { + IntervalWindow w = (IntervalWindow) c.window(); + return fmt.print(w.start()); + })); tableConfigure.put("mean_duration", new WriteWindowedToBigQuery.FieldInfo("FLOAT", c -> c.element())); return tableConfigure; diff --git a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/HourlyTeamScore.java b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/HourlyTeamScore.java index 5ce7d95e4a54..b516a322455b 100644 --- a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/HourlyTeamScore.java +++ b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/HourlyTeamScore.java @@ -132,8 +132,10 @@ static interface Options extends UserScore.Options { c -> c.element().getValue())); tableConfig.put("window_start", new WriteWindowedToBigQuery.FieldInfo>("STRING", - c -> { IntervalWindow w = (IntervalWindow) c.window(); - return fmt.print(w.start()); })); + c -> { + IntervalWindow w = (IntervalWindow) c.window(); + return fmt.print(w.start()); + })); return tableConfig; } diff --git a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java index 594d2b8ce897..97958b08de60 100644 --- a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java +++ b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java @@ -143,8 +143,10 @@ static interface Options extends HourlyTeamScore.Options, DataflowExampleOptions c -> c.element().getValue())); tableConfigure.put("window_start", new WriteWindowedToBigQuery.FieldInfo>("STRING", - c -> { IntervalWindow w = (IntervalWindow) c.window(); - return fmt.print(w.start()); })); + c -> { + IntervalWindow w = (IntervalWindow) c.window(); + return fmt.print(w.start()); + })); tableConfigure.put("processing_time", new WriteWindowedToBigQuery.FieldInfo>( "STRING", c -> fmt.print(Instant.now()))); From 021909855fcf6729ce6ccb9b9ff76f1ca5af35db Mon Sep 17 00:00:00 2001 From: Davor Bonaci Date: Mon, 25 Apr 2016 14:16:03 -0700 Subject: [PATCH 4/5] Reorganize Java packages in the sources of the Google Cloud Dataflow runner Packages are moving from org.apache.beam.sdk to org.apache.beam.runners.dataflow. --- .../beam/examples/MinimalWordCount.java | 4 +-- .../org/apache/beam/examples/WordCount.java | 2 +- .../common/DataflowExampleOptions.java | 2 +- .../examples/common/DataflowExampleUtils.java | 8 ++--- .../common/ExampleBigQueryTableOptions.java | 2 +- ...mplePubsubTopicAndSubscriptionOptions.java | 2 +- .../common/ExamplePubsubTopicOptions.java | 2 +- .../examples/common/PubsubFileInjector.java | 2 +- .../beam/examples/complete/AutoComplete.java | 2 +- .../complete/StreamingWordExtract.java | 2 +- .../complete/TopWikipediaSessions.java | 2 +- .../beam/examples/cookbook/DeDupExample.java | 2 +- .../examples/cookbook/TriggerExample.java | 4 +-- .../org/apache/beam/examples/WordCountIT.java | 4 +-- .../beam/examples/MinimalWordCountJava8.java | 4 +-- .../examples/complete/game/GameStats.java | 2 +- .../examples/complete/game/LeaderBoard.java | 2 +- .../beam/runners/flink/examples/TFIDF.java | 1 + .../runners/flink/examples/WordCount.java | 9 +++++- .../examples/streaming/AutoComplete.java | 21 ++++++++++++-- .../examples/streaming/JoinExamples.java | 6 +++- .../KafkaWindowedWordCountExample.java | 11 +++++-- .../examples/streaming/WindowedWordCount.java | 13 +++++++-- .../runners/flink/FlinkPipelineOptions.java | 2 +- .../runners/flink/FlinkPipelineRunner.java | 4 +-- .../FlinkBatchPipelineTranslator.java | 2 +- .../FlinkStreamingPipelineTranslator.java | 2 +- runners/google-cloud-dataflow-java/pom.xml | 4 +-- .../BlockingDataflowPipelineRunner.java | 7 +++-- .../DataflowJobAlreadyExistsException.java | 2 +- .../DataflowJobAlreadyUpdatedException.java | 2 +- .../DataflowJobCancelledException.java | 2 +- .../dataflow}/DataflowJobException.java | 2 +- .../DataflowJobExecutionException.java | 2 +- .../DataflowJobUpdatedException.java | 2 +- .../dataflow}/DataflowPipelineJob.java | 10 ++++--- .../dataflow}/DataflowPipelineRegistrar.java | 8 +++-- .../dataflow}/DataflowPipelineRunner.java | 29 ++++++++++--------- .../DataflowPipelineRunnerHooks.java | 2 +- .../dataflow}/DataflowPipelineTranslator.java | 13 +++++---- .../dataflow}/DataflowServiceException.java | 2 +- .../dataflow/internal}/AssignWindows.java | 2 +- .../internal}/BigQueryIOTranslator.java | 4 +-- .../dataflow/internal}/CustomSources.java | 4 +-- .../DataflowAggregatorTransforms.java | 2 +- .../DataflowMetricUpdateExtractor.java | 2 +- .../internal}/PubsubIOTranslator.java | 8 ++--- .../dataflow/internal}/ReadTranslator.java | 8 ++--- .../dataflow/internal}/package-info.java | 4 +-- .../BlockingDataflowPipelineOptions.java | 9 ++++-- .../options/CloudDebuggerOptions.java | 5 +++- .../options/DataflowPipelineDebugOptions.java | 15 ++++++---- .../options/DataflowPipelineOptions.java | 15 ++++++++-- .../DataflowPipelineWorkerPoolOptions.java | 9 ++++-- .../options/DataflowProfilingOptions.java | 4 ++- .../options/DataflowWorkerHarnessOptions.java | 6 +++- .../options/DataflowWorkerLoggingOptions.java | 6 +++- .../testing/TestDataflowPipelineOptions.java | 6 ++-- .../testing/TestDataflowPipelineRunner.java | 14 +++++---- .../dataflow}/util/DataflowPathValidator.java | 5 ++-- .../dataflow}/util/DataflowTransport.java | 5 ++-- .../dataflow}/util/GcsStager.java | 6 ++-- .../dataflow}/util/MonitoringUtil.java | 5 ++-- .../dataflow}/util/PackageUtil.java | 7 ++++- .../dataflow}/util/Stager.java | 2 +- .../BlockingDataflowPipelineRunnerTest.java | 8 ++--- .../dataflow}/DataflowPipelineJobTest.java | 8 +++-- .../DataflowPipelineRegistrarTest.java | 7 +++-- .../dataflow}/DataflowPipelineRunnerTest.java | 15 +++++----- .../DataflowPipelineTranslatorTest.java | 14 +++++---- .../dataflow/internal}/CustomSourcesTest.java | 4 +-- .../dataflow}/io/DataflowTextIOTest.java | 7 +++-- .../DataflowPipelineDebugOptionsTest.java | 4 ++- .../options/DataflowPipelineOptionsTest.java | 3 +- .../options/DataflowProfilingOptionsTest.java | 4 ++- .../DataflowWorkerLoggingOptionsTest.java | 6 ++-- .../TestDataflowPipelineRunnerTest.java | 12 ++++---- .../transforms/DataflowGroupByKeyTest.java | 9 ++++-- .../transforms/DataflowViewTest.java | 9 ++++-- .../util/DataflowPathValidatorTest.java | 8 +++-- .../dataflow}/util/MonitoringUtilTest.java | 6 ++-- .../dataflow}/util/PackageUtilTest.java | 6 ++-- .../apache/beam/sdk/testing/TestPipeline.java | 2 +- .../src/main/java/MinimalWordCount.java | 4 +-- .../src/main/java/WindowedWordCount.java | 10 ++++--- .../src/main/java/WordCount.java | 2 +- .../java/common/DataflowExampleOptions.java | 2 +- .../java/common/DataflowExampleUtils.java | 9 +++--- .../common/ExampleBigQueryTableOptions.java | 5 ++-- .../common/ExamplePubsubTopicOptions.java | 2 +- .../main/java/common/PubsubFileInjector.java | 9 +++--- 91 files changed, 333 insertions(+), 201 deletions(-) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/BlockingDataflowPipelineRunner.java (96%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowJobAlreadyExistsException.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowJobAlreadyUpdatedException.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowJobCancelledException.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowJobException.java (96%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowJobExecutionException.java (96%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowJobUpdatedException.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineJob.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineRegistrar.java (87%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineRunner.java (99%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineRunnerHooks.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineTranslator.java (98%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowServiceException.java (96%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/AssignWindows.java (98%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/BigQueryIOTranslator.java (96%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/CustomSources.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/DataflowAggregatorTransforms.java (98%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/DataflowMetricUpdateExtractor.java (98%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/PubsubIOTranslator.java (93%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/ReadTranslator.java (93%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/package-info.java (85%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/BlockingDataflowPipelineOptions.java (82%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/CloudDebuggerOptions.java (92%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowPipelineDebugOptions.java (95%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowPipelineOptions.java (88%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowPipelineWorkerPoolOptions.java (96%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowProfilingOptions.java (92%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowWorkerHarnessOptions.java (90%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowWorkerLoggingOptions.java (97%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/testing/TestDataflowPipelineOptions.java (80%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/testing/TestDataflowPipelineRunner.java (95%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/util/DataflowPathValidator.java (95%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/util/DataflowTransport.java (95%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/util/GcsStager.java (90%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/util/MonitoringUtil.java (98%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/util/PackageUtil.java (98%) rename runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/{sdk => runners/dataflow}/util/Stager.java (95%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk/runners => runners/dataflow}/BlockingDataflowPipelineRunnerTest.java (97%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineJobTest.java (98%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineRegistrarTest.java (90%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineRunnerTest.java (99%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk/runners => runners/dataflow}/DataflowPipelineTranslatorTest.java (98%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk/runners/dataflow => runners/dataflow/internal}/CustomSourcesTest.java (98%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/io/DataflowTextIOTest.java (94%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowPipelineDebugOptionsTest.java (94%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowPipelineOptionsTest.java (97%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowProfilingOptionsTest.java (94%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/options/DataflowWorkerLoggingOptionsTest.java (91%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/testing/TestDataflowPipelineRunnerTest.java (97%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/transforms/DataflowGroupByKeyTest.java (92%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/transforms/DataflowViewTest.java (95%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/util/DataflowPathValidatorTest.java (92%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/util/MonitoringUtilTest.java (96%) rename runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/{sdk => runners/dataflow}/util/PackageUtilTest.java (98%) diff --git a/examples/java/src/main/java/org/apache/beam/examples/MinimalWordCount.java b/examples/java/src/main/java/org/apache/beam/examples/MinimalWordCount.java index 6c33ca66bee7..ec517c4bd69c 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/MinimalWordCount.java +++ b/examples/java/src/main/java/org/apache/beam/examples/MinimalWordCount.java @@ -17,11 +17,11 @@ */ package org.apache.beam.examples; +import org.apache.beam.runners.dataflow.BlockingDataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.BlockingDataflowPipelineRunner; import org.apache.beam.sdk.transforms.Count; import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.MapElements; diff --git a/examples/java/src/main/java/org/apache/beam/examples/WordCount.java b/examples/java/src/main/java/org/apache/beam/examples/WordCount.java index 9fb985687bcb..364d075c2b21 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/WordCount.java +++ b/examples/java/src/main/java/org/apache/beam/examples/WordCount.java @@ -17,9 +17,9 @@ */ package org.apache.beam.examples; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; diff --git a/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleOptions.java b/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleOptions.java index 5c62146930b4..2e8ef3d7ed7c 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleOptions.java +++ b/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleOptions.java @@ -17,7 +17,7 @@ */ package org.apache.beam.examples.common; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; diff --git a/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleUtils.java b/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleUtils.java index 93c2358d69ed..7ac71d300b99 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleUtils.java +++ b/examples/java/src/main/java/org/apache/beam/examples/common/DataflowExampleUtils.java @@ -17,18 +17,18 @@ */ package org.apache.beam.examples.common; +import org.apache.beam.runners.dataflow.DataflowPipelineJob; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.io.TextIO; import org.apache.beam.sdk.options.BigQueryOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; -import org.apache.beam.sdk.runners.DataflowPipelineJob; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.runners.DirectPipelineRunner; import org.apache.beam.sdk.transforms.IntraBundleParallelization; import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.util.AttemptBoundedExponentialBackOff; -import org.apache.beam.sdk.util.MonitoringUtil; import org.apache.beam.sdk.util.Transport; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; diff --git a/examples/java/src/main/java/org/apache/beam/examples/common/ExampleBigQueryTableOptions.java b/examples/java/src/main/java/org/apache/beam/examples/common/ExampleBigQueryTableOptions.java index 647d508c9d0e..36304a851995 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/common/ExampleBigQueryTableOptions.java +++ b/examples/java/src/main/java/org/apache/beam/examples/common/ExampleBigQueryTableOptions.java @@ -17,7 +17,7 @@ */ package org.apache.beam.examples.common; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; diff --git a/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicAndSubscriptionOptions.java b/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicAndSubscriptionOptions.java index b3da88d6c879..22bcf4ef0c1c 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicAndSubscriptionOptions.java +++ b/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicAndSubscriptionOptions.java @@ -17,7 +17,7 @@ */ package org.apache.beam.examples.common; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; diff --git a/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicOptions.java b/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicOptions.java index 17f5b28150b9..603e309fea3f 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicOptions.java +++ b/examples/java/src/main/java/org/apache/beam/examples/common/ExamplePubsubTopicOptions.java @@ -17,7 +17,7 @@ */ package org.apache.beam.examples.common; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; diff --git a/examples/java/src/main/java/org/apache/beam/examples/common/PubsubFileInjector.java b/examples/java/src/main/java/org/apache/beam/examples/common/PubsubFileInjector.java index b71b6f389351..82d58b6c474a 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/common/PubsubFileInjector.java +++ b/examples/java/src/main/java/org/apache/beam/examples/common/PubsubFileInjector.java @@ -17,9 +17,9 @@ */ package org.apache.beam.examples.common; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; diff --git a/examples/java/src/main/java/org/apache/beam/examples/complete/AutoComplete.java b/examples/java/src/main/java/org/apache/beam/examples/complete/AutoComplete.java index e1b658bd2dac..2f2283cdb41f 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/complete/AutoComplete.java +++ b/examples/java/src/main/java/org/apache/beam/examples/complete/AutoComplete.java @@ -20,6 +20,7 @@ import org.apache.beam.examples.common.DataflowExampleUtils; import org.apache.beam.examples.common.ExampleBigQueryTableOptions; import org.apache.beam.examples.common.ExamplePubsubTopicOptions; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.coders.AvroCoder; @@ -31,7 +32,6 @@ import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.transforms.Count; import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.Filter; diff --git a/examples/java/src/main/java/org/apache/beam/examples/complete/StreamingWordExtract.java b/examples/java/src/main/java/org/apache/beam/examples/complete/StreamingWordExtract.java index 7107e34ddc74..5de08da16119 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/complete/StreamingWordExtract.java +++ b/examples/java/src/main/java/org/apache/beam/examples/complete/StreamingWordExtract.java @@ -20,6 +20,7 @@ import org.apache.beam.examples.common.DataflowExampleUtils; import org.apache.beam.examples.common.ExampleBigQueryTableOptions; import org.apache.beam.examples.common.ExamplePubsubTopicOptions; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.io.BigQueryIO; @@ -27,7 +28,6 @@ import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.ParDo; diff --git a/examples/java/src/main/java/org/apache/beam/examples/complete/TopWikipediaSessions.java b/examples/java/src/main/java/org/apache/beam/examples/complete/TopWikipediaSessions.java index fa587ea05574..e7090fdf2f57 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/complete/TopWikipediaSessions.java +++ b/examples/java/src/main/java/org/apache/beam/examples/complete/TopWikipediaSessions.java @@ -17,10 +17,10 @@ */ package org.apache.beam.examples.complete; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.coders.TableRowJsonCoder; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptions; diff --git a/examples/java/src/main/java/org/apache/beam/examples/cookbook/DeDupExample.java b/examples/java/src/main/java/org/apache/beam/examples/cookbook/DeDupExample.java index 1d342457ce74..fe2bbc8573c3 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/cookbook/DeDupExample.java +++ b/examples/java/src/main/java/org/apache/beam/examples/cookbook/DeDupExample.java @@ -17,9 +17,9 @@ */ package org.apache.beam.examples.cookbook; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; diff --git a/examples/java/src/main/java/org/apache/beam/examples/cookbook/TriggerExample.java b/examples/java/src/main/java/org/apache/beam/examples/cookbook/TriggerExample.java index 72976e36bf97..28885a79bea3 100644 --- a/examples/java/src/main/java/org/apache/beam/examples/cookbook/TriggerExample.java +++ b/examples/java/src/main/java/org/apache/beam/examples/cookbook/TriggerExample.java @@ -22,16 +22,16 @@ import org.apache.beam.examples.common.ExampleBigQueryTableOptions; import org.apache.beam.examples.common.ExamplePubsubTopicOptions; import org.apache.beam.examples.common.PubsubFileInjector; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.io.BigQueryIO; import org.apache.beam.sdk.io.PubsubIO; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.DoFn.RequiresWindowAccess; import org.apache.beam.sdk.transforms.GroupByKey; diff --git a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java index a5ad707bcecc..56ca98c1a6b7 100644 --- a/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java +++ b/examples/java/src/test/java/org/apache/beam/examples/WordCountIT.java @@ -22,10 +22,10 @@ import static org.junit.Assert.assertNotNull; import org.apache.beam.examples.WordCount.WordCountOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.testing.TestDataflowPipelineRunner; import org.apache.beam.sdk.PipelineResult; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.testing.TestDataflowPipelineRunner; import org.apache.beam.sdk.testing.TestPipeline; import org.apache.beam.sdk.testing.TestPipelineOptions; import com.google.common.base.Joiner; diff --git a/examples/java8/src/main/java/org/apache/beam/examples/MinimalWordCountJava8.java b/examples/java8/src/main/java/org/apache/beam/examples/MinimalWordCountJava8.java index 493344e0767a..398d517baa54 100644 --- a/examples/java8/src/main/java/org/apache/beam/examples/MinimalWordCountJava8.java +++ b/examples/java8/src/main/java/org/apache/beam/examples/MinimalWordCountJava8.java @@ -17,11 +17,11 @@ */ package org.apache.beam.examples; +import org.apache.beam.runners.dataflow.BlockingDataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.BlockingDataflowPipelineRunner; import org.apache.beam.sdk.transforms.Count; import org.apache.beam.sdk.transforms.Filter; import org.apache.beam.sdk.transforms.FlatMapElements; diff --git a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java index 2d1426497b36..12dfdf9abebb 100644 --- a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java +++ b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/GameStats.java @@ -19,13 +19,13 @@ import org.apache.beam.examples.common.DataflowExampleUtils; import org.apache.beam.examples.complete.game.utils.WriteWindowedToBigQuery; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.io.PubsubIO; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.transforms.Aggregator; import org.apache.beam.sdk.transforms.Combine; import org.apache.beam.sdk.transforms.DoFn; diff --git a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java index 97958b08de60..a5d9fb93fb03 100644 --- a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java +++ b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/LeaderBoard.java @@ -21,6 +21,7 @@ import org.apache.beam.examples.common.DataflowExampleUtils; import org.apache.beam.examples.complete.game.utils.WriteToBigQuery; import org.apache.beam.examples.complete.game.utils.WriteWindowedToBigQuery; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.io.PubsubIO; @@ -28,7 +29,6 @@ import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.options.Validation; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.transforms.ParDo; import org.apache.beam.sdk.transforms.windowing.AfterProcessingTime; import org.apache.beam.sdk.transforms.windowing.AfterWatermark; diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/TFIDF.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/TFIDF.java index 3eb1327e51bd..0afde0a57978 100644 --- a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/TFIDF.java +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/TFIDF.java @@ -53,6 +53,7 @@ import org.apache.beam.sdk.values.PDone; import org.apache.beam.sdk.values.PInput; import org.apache.beam.sdk.values.TupleTag; + import org.slf4j.Logger; import org.slf4j.LoggerFactory; diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java index 04e11c1ebb80..702fb63ec68d 100644 --- a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java @@ -25,7 +25,14 @@ import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.transforms.*; +import org.apache.beam.sdk.transforms.Aggregator; +import org.apache.beam.sdk.transforms.Count; +import org.apache.beam.sdk.transforms.DoFn; +import org.apache.beam.sdk.transforms.MapElements; +import org.apache.beam.sdk.transforms.PTransform; +import org.apache.beam.sdk.transforms.ParDo; +import org.apache.beam.sdk.transforms.SimpleFunction; +import org.apache.beam.sdk.transforms.Sum; import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PCollection; diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/AutoComplete.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/AutoComplete.java index 2bc34903e565..9d1168ba40ee 100644 --- a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/AutoComplete.java +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/AutoComplete.java @@ -22,17 +22,32 @@ import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.coders.AvroCoder; import org.apache.beam.sdk.coders.DefaultCoder; -import org.apache.beam.sdk.io.*; +import org.apache.beam.sdk.io.Read; +import org.apache.beam.sdk.io.TextIO; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.transforms.*; +import org.apache.beam.sdk.transforms.Aggregator; +import org.apache.beam.sdk.transforms.Count; +import org.apache.beam.sdk.transforms.DoFn; +import org.apache.beam.sdk.transforms.Filter; +import org.apache.beam.sdk.transforms.Flatten; +import org.apache.beam.sdk.transforms.PTransform; +import org.apache.beam.sdk.transforms.ParDo; +import org.apache.beam.sdk.transforms.Partition; import org.apache.beam.sdk.transforms.Partition.PartitionFn; -import org.apache.beam.sdk.transforms.windowing.*; +import org.apache.beam.sdk.transforms.SerializableFunction; +import org.apache.beam.sdk.transforms.Sum; +import org.apache.beam.sdk.transforms.Top; +import org.apache.beam.sdk.transforms.windowing.AfterWatermark; +import org.apache.beam.sdk.transforms.windowing.FixedWindows; +import org.apache.beam.sdk.transforms.windowing.Window; +import org.apache.beam.sdk.transforms.windowing.WindowFn; import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.PCollectionList; + import org.joda.time.Duration; import java.io.IOException; diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/JoinExamples.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/JoinExamples.java index 8c8490b27085..d3e963d91255 100644 --- a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/JoinExamples.java +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/JoinExamples.java @@ -29,11 +29,15 @@ import org.apache.beam.sdk.transforms.join.CoGbkResult; import org.apache.beam.sdk.transforms.join.CoGroupByKey; import org.apache.beam.sdk.transforms.join.KeyedPCollectionTuple; -import org.apache.beam.sdk.transforms.windowing.*; +import org.apache.beam.sdk.transforms.windowing.AfterWatermark; +import org.apache.beam.sdk.transforms.windowing.FixedWindows; +import org.apache.beam.sdk.transforms.windowing.Window; +import org.apache.beam.sdk.transforms.windowing.WindowFn; import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; import org.apache.beam.sdk.values.TupleTag; + import org.joda.time.Duration; /** diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/KafkaWindowedWordCountExample.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/KafkaWindowedWordCountExample.java index 95210ad33ecf..abb9feaaac71 100644 --- a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/KafkaWindowedWordCountExample.java +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/KafkaWindowedWordCountExample.java @@ -25,10 +25,17 @@ import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.transforms.*; -import org.apache.beam.sdk.transforms.windowing.*; +import org.apache.beam.sdk.transforms.Aggregator; +import org.apache.beam.sdk.transforms.Count; +import org.apache.beam.sdk.transforms.DoFn; +import org.apache.beam.sdk.transforms.ParDo; +import org.apache.beam.sdk.transforms.Sum; +import org.apache.beam.sdk.transforms.windowing.AfterWatermark; +import org.apache.beam.sdk.transforms.windowing.FixedWindows; +import org.apache.beam.sdk.transforms.windowing.Window; import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PCollection; + import org.apache.flink.streaming.connectors.kafka.FlinkKafkaConsumer08; import org.apache.flink.streaming.util.serialization.SimpleStringSchema; import org.joda.time.Duration; diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/WindowedWordCount.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/WindowedWordCount.java index 9f77f8c92e79..e803e6ed80ea 100644 --- a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/WindowedWordCount.java +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/WindowedWordCount.java @@ -20,12 +20,19 @@ import org.apache.beam.runners.flink.FlinkPipelineRunner; import org.apache.beam.runners.flink.translation.wrappers.streaming.io.UnboundedSocketSource; import org.apache.beam.sdk.Pipeline; -import org.apache.beam.sdk.io.*; +import org.apache.beam.sdk.io.Read; +import org.apache.beam.sdk.io.TextIO; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.transforms.*; -import org.apache.beam.sdk.transforms.windowing.*; +import org.apache.beam.sdk.transforms.Aggregator; +import org.apache.beam.sdk.transforms.Count; +import org.apache.beam.sdk.transforms.DoFn; +import org.apache.beam.sdk.transforms.ParDo; +import org.apache.beam.sdk.transforms.Sum; +import org.apache.beam.sdk.transforms.windowing.AfterWatermark; +import org.apache.beam.sdk.transforms.windowing.SlidingWindows; +import org.apache.beam.sdk.transforms.windowing.Window; import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PCollection; diff --git a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java index bfb0d6aff22f..8c82abdcb9e0 100644 --- a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java +++ b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineOptions.java @@ -18,8 +18,8 @@ package org.apache.beam.runners.flink; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.ApplicationNameOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptions; diff --git a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineRunner.java b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineRunner.java index bb016fa4a611..a389d7a076c8 100644 --- a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineRunner.java +++ b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineRunner.java @@ -17,11 +17,11 @@ */ package org.apache.beam.runners.flink; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.options.PipelineOptionsValidator; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.runners.PipelineRunner; import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.values.PInput; @@ -46,7 +46,7 @@ * pipeline by first translating them to a Flink Plan and then executing them either locally * or on a Flink cluster, depending on the configuration. *

- * This is based on {@link org.apache.beam.sdk.runners.DataflowPipelineRunner}. + * This is based on {@link org.apache.beam.runners.dataflow.DataflowPipelineRunner}. */ public class FlinkPipelineRunner extends PipelineRunner { diff --git a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkBatchPipelineTranslator.java b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkBatchPipelineTranslator.java index 5ce828c7271f..456cf09457b6 100644 --- a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkBatchPipelineTranslator.java +++ b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkBatchPipelineTranslator.java @@ -30,7 +30,7 @@ /** * FlinkBatchPipelineTranslator knows how to translate Pipeline objects into Flink Jobs. - * This is based on {@link org.apache.beam.sdk.runners.DataflowPipelineTranslator} + * This is based on {@link org.apache.beam.runners.dataflow.DataflowPipelineTranslator} */ public class FlinkBatchPipelineTranslator extends FlinkPipelineTranslator { diff --git a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkStreamingPipelineTranslator.java b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkStreamingPipelineTranslator.java index 43598422c1ab..ebaf6ba0234b 100644 --- a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkStreamingPipelineTranslator.java +++ b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/translation/FlinkStreamingPipelineTranslator.java @@ -32,7 +32,7 @@ * {@link org.apache.beam.sdk.values.PCollection}-based job into a * {@link org.apache.flink.streaming.api.datastream.DataStream} one. * - * This is based on {@link org.apache.beam.sdk.runners.DataflowPipelineTranslator} + * This is based on {@link org.apache.beam.runners.dataflow.DataflowPipelineTranslator} * */ public class FlinkStreamingPipelineTranslator extends FlinkPipelineTranslator { diff --git a/runners/google-cloud-dataflow-java/pom.xml b/runners/google-cloud-dataflow-java/pom.xml index 789397521ad7..beb340c44201 100644 --- a/runners/google-cloud-dataflow-java/pom.xml +++ b/runners/google-cloud-dataflow-java/pom.xml @@ -177,8 +177,8 @@ Google Cloud Dataflow Java Runner ${project.version} Google Cloud Dataflow Runner for Java, version ${project.version} - org.apache.beam.sdk - -exclude org.apache.beam.sdk.runners.worker:org.apache.beam.sdk.runners.dataflow:org.apache.beam.sdk.util:org.apache.beam.sdk.runners.inprocess ${dataflow.javadoc_opts} + org.apache.beam.runners.dataflow + -exclude org.apache.beam.sdk.runners.dataflow.internal:org.apache.beam.sdk.runners.dataflow.testing:org.apache.beam.sdk.runners.dataflow.util ${dataflow.javadoc_opts} false true ]]> diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/BlockingDataflowPipelineRunner.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/BlockingDataflowPipelineRunner.java similarity index 96% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/BlockingDataflowPipelineRunner.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/BlockingDataflowPipelineRunner.java index 0f4126fad281..d8ee16a25bb1 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/BlockingDataflowPipelineRunner.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/BlockingDataflowPipelineRunner.java @@ -15,16 +15,17 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; +import org.apache.beam.runners.dataflow.options.BlockingDataflowPipelineOptions; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult.State; import org.apache.beam.sdk.annotations.Experimental; -import org.apache.beam.sdk.options.BlockingDataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsValidator; +import org.apache.beam.sdk.runners.PipelineRunner; import org.apache.beam.sdk.transforms.PTransform; -import org.apache.beam.sdk.util.MonitoringUtil; import org.apache.beam.sdk.values.PInput; import org.apache.beam.sdk.values.POutput; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobAlreadyExistsException.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobAlreadyExistsException.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobAlreadyExistsException.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobAlreadyExistsException.java index 728baa756278..2b73bf78bb95 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobAlreadyExistsException.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobAlreadyExistsException.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; /** * An exception that is thrown if the unique job name constraint of the Dataflow diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobAlreadyUpdatedException.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobAlreadyUpdatedException.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobAlreadyUpdatedException.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobAlreadyUpdatedException.java index 4d2025f5ceb0..be1163783746 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobAlreadyUpdatedException.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobAlreadyUpdatedException.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; /** * An exception that is thrown if the existing job has already been updated within the Dataflow diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobCancelledException.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobCancelledException.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobCancelledException.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobCancelledException.java index cc002f116877..de44a19516ae 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobCancelledException.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobCancelledException.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; /** * Signals that a job run by a {@link BlockingDataflowPipelineRunner} was updated during execution. diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobException.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobException.java similarity index 96% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobException.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobException.java index ed66fb74f6aa..74c0f80684a2 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobException.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobException.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import java.util.Objects; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobExecutionException.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobExecutionException.java similarity index 96% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobExecutionException.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobExecutionException.java index bc8a903e8b06..11b8723b75bd 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobExecutionException.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobExecutionException.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import javax.annotation.Nullable; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobUpdatedException.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobUpdatedException.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobUpdatedException.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobUpdatedException.java index 65c327f0f291..e30a0e78bc16 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowJobUpdatedException.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowJobUpdatedException.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; /** * Signals that a job run by a {@link BlockingDataflowPipelineRunner} was updated during execution. diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineJob.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineJob.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineJob.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineJob.java index 428a6edda2ad..19df0a194be5 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineJob.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineJob.java @@ -15,18 +15,20 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.apache.beam.sdk.util.TimeUtil.fromCloudTime; +import org.apache.beam.runners.dataflow.internal.DataflowAggregatorTransforms; +import org.apache.beam.runners.dataflow.internal.DataflowMetricUpdateExtractor; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; import org.apache.beam.sdk.PipelineResult; -import org.apache.beam.sdk.runners.dataflow.DataflowAggregatorTransforms; -import org.apache.beam.sdk.runners.dataflow.DataflowMetricUpdateExtractor; +import org.apache.beam.sdk.runners.AggregatorRetrievalException; +import org.apache.beam.sdk.runners.AggregatorValues; import org.apache.beam.sdk.transforms.Aggregator; import org.apache.beam.sdk.util.AttemptAndTimeBoundedExponentialBackOff; import org.apache.beam.sdk.util.AttemptBoundedExponentialBackOff; import org.apache.beam.sdk.util.MapAggregatorValues; -import org.apache.beam.sdk.util.MonitoringUtil; import com.google.api.client.googleapis.json.GoogleJsonResponseException; import com.google.api.client.util.BackOff; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRegistrar.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRegistrar.java similarity index 87% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRegistrar.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRegistrar.java index b0f72ed77484..d7d243f4df83 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRegistrar.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRegistrar.java @@ -15,12 +15,14 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; -import org.apache.beam.sdk.options.BlockingDataflowPipelineOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.BlockingDataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsRegistrar; +import org.apache.beam.sdk.runners.PipelineRunner; +import org.apache.beam.sdk.runners.PipelineRunnerRegistrar; import com.google.auto.service.AutoService; import com.google.common.collect.ImmutableList; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRunner.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunner.java similarity index 99% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRunner.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunner.java index d2f8bbedca4b..0fc095a14094 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRunner.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunner.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.apache.beam.sdk.util.StringUtils.approximatePTransformName; import static org.apache.beam.sdk.util.StringUtils.approximateSimpleName; @@ -24,6 +24,18 @@ import static com.google.common.base.Preconditions.checkArgument; import static com.google.common.base.Preconditions.checkState; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.JobSpecification; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.TransformTranslator; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.TranslationContext; +import org.apache.beam.runners.dataflow.internal.AssignWindows; +import org.apache.beam.runners.dataflow.internal.DataflowAggregatorTransforms; +import org.apache.beam.runners.dataflow.internal.PubsubIOTranslator; +import org.apache.beam.runners.dataflow.internal.ReadTranslator; +import org.apache.beam.runners.dataflow.options.DataflowPipelineDebugOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineWorkerPoolOptions; +import org.apache.beam.runners.dataflow.util.DataflowTransport; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.Pipeline.PipelineVisitor; import org.apache.beam.sdk.PipelineResult.State; @@ -52,19 +64,12 @@ import org.apache.beam.sdk.io.TextIO; import org.apache.beam.sdk.io.UnboundedSource; import org.apache.beam.sdk.io.Write; -import org.apache.beam.sdk.options.DataflowPipelineDebugOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; -import org.apache.beam.sdk.options.DataflowPipelineWorkerPoolOptions; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsValidator; import org.apache.beam.sdk.options.StreamingOptions; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.JobSpecification; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.TransformTranslator; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.TranslationContext; -import org.apache.beam.sdk.runners.dataflow.AssignWindows; -import org.apache.beam.sdk.runners.dataflow.DataflowAggregatorTransforms; -import org.apache.beam.sdk.runners.dataflow.PubsubIOTranslator; -import org.apache.beam.sdk.runners.dataflow.ReadTranslator; +import org.apache.beam.sdk.runners.AggregatorPipelineExtractor; +import org.apache.beam.sdk.runners.PipelineRunner; +import org.apache.beam.sdk.runners.TransformTreeNode; import org.apache.beam.sdk.runners.worker.IsmFormat; import org.apache.beam.sdk.runners.worker.IsmFormat.IsmRecord; import org.apache.beam.sdk.runners.worker.IsmFormat.IsmRecordCoder; @@ -89,10 +94,8 @@ import org.apache.beam.sdk.transforms.windowing.GlobalWindows; import org.apache.beam.sdk.transforms.windowing.Window; import org.apache.beam.sdk.util.CoderUtils; -import org.apache.beam.sdk.util.DataflowTransport; import org.apache.beam.sdk.util.IOChannelUtils; import org.apache.beam.sdk.util.InstanceBuilder; -import org.apache.beam.sdk.util.MonitoringUtil; import org.apache.beam.sdk.util.PCollectionViews; import org.apache.beam.sdk.util.PathValidator; import org.apache.beam.sdk.util.PropertyNames; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRunnerHooks.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunnerHooks.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRunnerHooks.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunnerHooks.java index 7ea44d7af72a..4d3796631744 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineRunnerHooks.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunnerHooks.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import org.apache.beam.sdk.annotations.Experimental; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineTranslator.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslator.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineTranslator.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslator.java index 5c0745f54ace..0f2d325a6982 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowPipelineTranslator.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslator.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.apache.beam.sdk.util.SerializableUtils.serializeToByteArray; import static org.apache.beam.sdk.util.StringUtils.byteArrayToJsonString; @@ -30,6 +30,11 @@ import static com.google.common.base.Preconditions.checkArgument; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner.GroupByKeyAndSortValuesOnly; +import org.apache.beam.runners.dataflow.internal.BigQueryIOTranslator; +import org.apache.beam.runners.dataflow.internal.PubsubIOTranslator; +import org.apache.beam.runners.dataflow.internal.ReadTranslator; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.Pipeline.PipelineVisitor; import org.apache.beam.sdk.coders.Coder; @@ -37,12 +42,8 @@ import org.apache.beam.sdk.io.BigQueryIO; import org.apache.beam.sdk.io.PubsubIO; import org.apache.beam.sdk.io.Read; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.StreamingOptions; -import org.apache.beam.sdk.runners.DataflowPipelineRunner.GroupByKeyAndSortValuesOnly; -import org.apache.beam.sdk.runners.dataflow.BigQueryIOTranslator; -import org.apache.beam.sdk.runners.dataflow.PubsubIOTranslator; -import org.apache.beam.sdk.runners.dataflow.ReadTranslator; +import org.apache.beam.sdk.runners.TransformTreeNode; import org.apache.beam.sdk.transforms.AppliedPTransform; import org.apache.beam.sdk.transforms.Combine; import org.apache.beam.sdk.transforms.DoFn; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowServiceException.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowServiceException.java similarity index 96% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowServiceException.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowServiceException.java index c1f0e7493102..5cd8a4d9be34 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/DataflowServiceException.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowServiceException.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import javax.annotation.Nullable; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/AssignWindows.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/AssignWindows.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java index b068f500d89b..1b18c2a3f386 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/AssignWindows.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; import org.apache.beam.sdk.coders.Coder; import org.apache.beam.sdk.transforms.DoFn; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/BigQueryIOTranslator.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/BigQueryIOTranslator.java similarity index 96% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/BigQueryIOTranslator.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/BigQueryIOTranslator.java index b0952a60a635..b344f0f788ec 100755 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/BigQueryIOTranslator.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/BigQueryIOTranslator.java @@ -15,10 +15,10 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator; import org.apache.beam.sdk.io.BigQueryIO; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator; import org.apache.beam.sdk.util.PropertyNames; import com.google.api.services.bigquery.model.TableReference; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/CustomSources.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/CustomSources.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/CustomSources.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/CustomSources.java index a3f9fda04ea0..73e5da02f464 100755 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/CustomSources.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/CustomSources.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; import static org.apache.beam.sdk.util.SerializableUtils.serializeToByteArray; import static org.apache.beam.sdk.util.Structs.addString; @@ -24,10 +24,10 @@ import static com.google.api.client.util.Base64.encodeBase64String; import static com.google.common.base.Preconditions.checkArgument; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.io.BoundedSource; import org.apache.beam.sdk.io.Source; import org.apache.beam.sdk.io.UnboundedSource; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.util.CloudObject; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/DataflowAggregatorTransforms.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/DataflowAggregatorTransforms.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/DataflowAggregatorTransforms.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/DataflowAggregatorTransforms.java index 088ce63d557a..7a08fde8fc17 100755 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/DataflowAggregatorTransforms.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/DataflowAggregatorTransforms.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.transforms.Aggregator; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/DataflowMetricUpdateExtractor.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/DataflowMetricUpdateExtractor.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/DataflowMetricUpdateExtractor.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/DataflowMetricUpdateExtractor.java index 35ef229ed3ae..8ab59fc911d6 100755 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/DataflowMetricUpdateExtractor.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/DataflowMetricUpdateExtractor.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; import org.apache.beam.sdk.transforms.Aggregator; import org.apache.beam.sdk.transforms.AppliedPTransform; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/PubsubIOTranslator.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/PubsubIOTranslator.java similarity index 93% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/PubsubIOTranslator.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/PubsubIOTranslator.java index e299bec3905e..976f948dd1c2 100755 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/PubsubIOTranslator.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/PubsubIOTranslator.java @@ -15,12 +15,12 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.TransformTranslator; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.TranslationContext; import org.apache.beam.sdk.io.PubsubIO; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.TransformTranslator; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.TranslationContext; import org.apache.beam.sdk.util.PropertyNames; import org.apache.beam.sdk.util.WindowedValue; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/ReadTranslator.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/ReadTranslator.java similarity index 93% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/ReadTranslator.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/ReadTranslator.java index 0c394a121b31..373738a71709 100755 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/ReadTranslator.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/ReadTranslator.java @@ -15,18 +15,18 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; import static org.apache.beam.sdk.util.Structs.addBoolean; import static org.apache.beam.sdk.util.Structs.addDictionary; import static org.apache.beam.sdk.util.Structs.addLong; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.TransformTranslator; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.TranslationContext; import org.apache.beam.sdk.io.FileBasedSource; import org.apache.beam.sdk.io.Read; import org.apache.beam.sdk.io.Source; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.TransformTranslator; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.TranslationContext; import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.util.PropertyNames; import org.apache.beam.sdk.values.PValue; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/package-info.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/package-info.java similarity index 85% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/package-info.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/package-info.java index 3630ea74fa37..f2e8459bba3d 100755 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/runners/dataflow/package-info.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/package-info.java @@ -16,6 +16,6 @@ * limitations under the License. */ /** - * Implementation of the {@link org.apache.beam.sdk.runners.DataflowPipelineRunner}. + * Implementation of the {@link org.apache.beam.runners.dataflow.DataflowPipelineRunner}. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/BlockingDataflowPipelineOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/BlockingDataflowPipelineOptions.java similarity index 82% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/BlockingDataflowPipelineOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/BlockingDataflowPipelineOptions.java index 5b52d3022c45..7fa5ad6a8266 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/BlockingDataflowPipelineOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/BlockingDataflowPipelineOptions.java @@ -15,9 +15,14 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; -import org.apache.beam.sdk.runners.BlockingDataflowPipelineRunner; +import org.apache.beam.runners.dataflow.BlockingDataflowPipelineRunner; +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.DefaultValueFactory; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.Hidden; +import org.apache.beam.sdk.options.PipelineOptions; import com.fasterxml.jackson.annotation.JsonIgnore; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/CloudDebuggerOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/CloudDebuggerOptions.java similarity index 92% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/CloudDebuggerOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/CloudDebuggerOptions.java index 0464fb4b8490..dbfbb16f3022 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/CloudDebuggerOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/CloudDebuggerOptions.java @@ -15,9 +15,12 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; import org.apache.beam.sdk.annotations.Experimental; +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.Hidden; import com.google.api.services.clouddebugger.v2.model.Debuggee; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineDebugOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineDebugOptions.java similarity index 95% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineDebugOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineDebugOptions.java index 553f667ddf3e..71c8a788326a 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineDebugOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineDebugOptions.java @@ -15,15 +15,20 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; +import org.apache.beam.runners.dataflow.util.DataflowPathValidator; +import org.apache.beam.runners.dataflow.util.DataflowTransport; +import org.apache.beam.runners.dataflow.util.GcsStager; +import org.apache.beam.runners.dataflow.util.Stager; import org.apache.beam.sdk.annotations.Experimental; -import org.apache.beam.sdk.util.DataflowPathValidator; -import org.apache.beam.sdk.util.DataflowTransport; -import org.apache.beam.sdk.util.GcsStager; +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.DefaultValueFactory; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.Hidden; +import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.util.InstanceBuilder; import org.apache.beam.sdk.util.PathValidator; -import org.apache.beam.sdk.util.Stager; import com.google.api.services.dataflow.Dataflow; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptions.java similarity index 88% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptions.java index 7f9d189dc63c..6e6ad96d0a49 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptions.java @@ -15,9 +15,20 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.sdk.options.ApplicationNameOptions; +import org.apache.beam.sdk.options.BigQueryOptions; +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.DefaultValueFactory; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.GcpOptions; +import org.apache.beam.sdk.options.GcsOptions; +import org.apache.beam.sdk.options.PipelineOptions; +import org.apache.beam.sdk.options.PubsubOptions; +import org.apache.beam.sdk.options.StreamingOptions; +import org.apache.beam.sdk.options.Validation; import com.google.common.base.MoreObjects; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineWorkerPoolOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineWorkerPoolOptions.java similarity index 96% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineWorkerPoolOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineWorkerPoolOptions.java index 6b44e7aa96f9..a29b328fe871 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowPipelineWorkerPoolOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineWorkerPoolOptions.java @@ -15,10 +15,15 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; import org.apache.beam.sdk.annotations.Experimental; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.DefaultValueFactory; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.Hidden; +import org.apache.beam.sdk.options.PipelineOptions; import com.fasterxml.jackson.annotation.JsonIgnore; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowProfilingOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowProfilingOptions.java similarity index 92% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowProfilingOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowProfilingOptions.java index da512305eb44..c7b4c91a0998 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowProfilingOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowProfilingOptions.java @@ -15,9 +15,11 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; import org.apache.beam.sdk.annotations.Experimental; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.Hidden; import java.util.HashMap; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowWorkerHarnessOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowWorkerHarnessOptions.java similarity index 90% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowWorkerHarnessOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowWorkerHarnessOptions.java index a28b756bfb03..e946e6b5b1bf 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowWorkerHarnessOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowWorkerHarnessOptions.java @@ -15,7 +15,11 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; + +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.Hidden; /** * Options that are used exclusively within the Dataflow worker harness. diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowWorkerLoggingOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowWorkerLoggingOptions.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowWorkerLoggingOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowWorkerLoggingOptions.java index dc840d8718b1..74ab59a28b2c 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/options/DataflowWorkerLoggingOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowWorkerLoggingOptions.java @@ -15,7 +15,11 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; + +import org.apache.beam.sdk.options.Default; +import org.apache.beam.sdk.options.Description; +import org.apache.beam.sdk.options.PipelineOptions; import com.google.common.base.Preconditions; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/testing/TestDataflowPipelineOptions.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineOptions.java similarity index 80% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/testing/TestDataflowPipelineOptions.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineOptions.java index f8913aff9870..e66ffc9498c2 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/testing/TestDataflowPipelineOptions.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineOptions.java @@ -15,9 +15,11 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.testing; +package org.apache.beam.runners.dataflow.testing; -import org.apache.beam.sdk.options.BlockingDataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.BlockingDataflowPipelineOptions; +import org.apache.beam.sdk.testing.TestPipeline; +import org.apache.beam.sdk.testing.TestPipelineOptions; /** * A set of options used to configure the {@link TestPipeline}. diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/testing/TestDataflowPipelineRunner.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java similarity index 95% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/testing/TestDataflowPipelineRunner.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java index d647b0db3a4a..3ab91f5511df 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/testing/TestDataflowPipelineRunner.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunner.java @@ -15,19 +15,21 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.testing; +package org.apache.beam.runners.dataflow.testing; +import org.apache.beam.runners.dataflow.DataflowJobExecutionException; +import org.apache.beam.runners.dataflow.DataflowPipelineJob; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; +import org.apache.beam.runners.dataflow.util.MonitoringUtil.JobMessagesHandler; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.PipelineResult.State; import org.apache.beam.sdk.options.PipelineOptions; -import org.apache.beam.sdk.runners.DataflowJobExecutionException; -import org.apache.beam.sdk.runners.DataflowPipelineJob; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.runners.PipelineRunner; +import org.apache.beam.sdk.testing.PAssert; +import org.apache.beam.sdk.testing.TestPipeline; import org.apache.beam.sdk.transforms.PTransform; -import org.apache.beam.sdk.util.MonitoringUtil; -import org.apache.beam.sdk.util.MonitoringUtil.JobMessagesHandler; import org.apache.beam.sdk.values.PInput; import org.apache.beam.sdk.values.POutput; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/DataflowPathValidator.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/DataflowPathValidator.java similarity index 95% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/DataflowPathValidator.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/DataflowPathValidator.java index d0388a3a9254..ddc5d6f7f226 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/DataflowPathValidator.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/DataflowPathValidator.java @@ -15,10 +15,11 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; +import org.apache.beam.sdk.util.PathValidator; import org.apache.beam.sdk.util.gcsfs.GcsPath; import com.google.common.base.Preconditions; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/DataflowTransport.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/DataflowTransport.java similarity index 95% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/DataflowTransport.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/DataflowTransport.java index 8fcfccf6688b..01996577a970 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/DataflowTransport.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/DataflowTransport.java @@ -15,12 +15,13 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; import static org.apache.beam.sdk.util.Transport.getJsonFactory; import static org.apache.beam.sdk.util.Transport.getTransport; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; +import org.apache.beam.sdk.util.RetryHttpRequestInitializer; import com.google.api.client.auth.oauth2.Credential; import com.google.api.client.http.HttpRequestInitializer; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/GcsStager.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/GcsStager.java similarity index 90% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/GcsStager.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/GcsStager.java index 4f1f6734c7f1..8e7cbbe2e754 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/GcsStager.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/GcsStager.java @@ -15,10 +15,10 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; -import org.apache.beam.sdk.options.DataflowPipelineDebugOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineDebugOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; import com.google.api.services.dataflow.model.DataflowPackage; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/MonitoringUtil.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/MonitoringUtil.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/MonitoringUtil.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/MonitoringUtil.java index 5afca52c1b1d..2eec9ccf9aa8 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/MonitoringUtil.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/MonitoringUtil.java @@ -15,12 +15,13 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; import static org.apache.beam.sdk.util.TimeUtil.fromCloudTime; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.PipelineResult.State; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.sdk.util.TimeUtil; import com.google.api.services.dataflow.Dataflow; import com.google.api.services.dataflow.Dataflow.Projects.Jobs.Messages; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/PackageUtil.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/PackageUtil.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/PackageUtil.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/PackageUtil.java index 7d492b4357e1..cff7e2b9c68f 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/PackageUtil.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/PackageUtil.java @@ -15,7 +15,12 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; + +import org.apache.beam.sdk.util.AttemptBoundedExponentialBackOff; +import org.apache.beam.sdk.util.IOChannelUtils; +import org.apache.beam.sdk.util.MimeTypes; +import org.apache.beam.sdk.util.ZipFiles; import com.google.api.client.util.BackOffUtils; import com.google.api.client.util.Sleeper; diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/Stager.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/Stager.java similarity index 95% rename from runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/Stager.java rename to runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/Stager.java index 9547ac1ffa0e..2172eb4711cf 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/sdk/util/Stager.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/util/Stager.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; import com.google.api.services.dataflow.model.DataflowPackage; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/BlockingDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/BlockingDataflowPipelineRunnerTest.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/BlockingDataflowPipelineRunnerTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/BlockingDataflowPipelineRunnerTest.java index 13e120b7eb65..bc570e1f1903 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/BlockingDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/BlockingDataflowPipelineRunnerTest.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.hamcrest.CoreMatchers.equalTo; import static org.junit.Assert.assertEquals; @@ -24,15 +24,15 @@ import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.testing.TestDataflowPipelineOptions; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult.State; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.ExpectedLogs; -import org.apache.beam.sdk.testing.TestDataflowPipelineOptions; import org.apache.beam.sdk.testing.TestPipeline; -import org.apache.beam.sdk.util.MonitoringUtil; import org.apache.beam.sdk.util.NoopPathValidator; import org.apache.beam.sdk.util.TestCredential; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineJobTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineJobTest.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineJobTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineJobTest.java index d496f38b6e22..80b7e7b12454 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineJobTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineJobTest.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.hamcrest.MatcherAssert.assertThat; import static org.hamcrest.Matchers.allOf; @@ -32,8 +32,11 @@ import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; +import org.apache.beam.runners.dataflow.internal.DataflowAggregatorTransforms; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; import org.apache.beam.sdk.PipelineResult.State; -import org.apache.beam.sdk.runners.dataflow.DataflowAggregatorTransforms; +import org.apache.beam.sdk.runners.AggregatorRetrievalException; +import org.apache.beam.sdk.runners.AggregatorValues; import org.apache.beam.sdk.testing.FastNanoClockAndSleeper; import org.apache.beam.sdk.transforms.Aggregator; import org.apache.beam.sdk.transforms.AppliedPTransform; @@ -41,7 +44,6 @@ import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.transforms.Sum; import org.apache.beam.sdk.util.AttemptBoundedExponentialBackOff; -import org.apache.beam.sdk.util.MonitoringUtil; import org.apache.beam.sdk.values.PInput; import org.apache.beam.sdk.values.POutput; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineRegistrarTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineRegistrarTest.java similarity index 90% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineRegistrarTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineRegistrarTest.java index 67ed2fd09490..cf9a95ae14fb 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineRegistrarTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineRegistrarTest.java @@ -15,14 +15,15 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.junit.Assert.assertEquals; import static org.junit.Assert.fail; -import org.apache.beam.sdk.options.BlockingDataflowPipelineOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.BlockingDataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsRegistrar; +import org.apache.beam.sdk.runners.PipelineRunnerRegistrar; import com.google.common.collect.ImmutableList; import com.google.common.collect.Lists; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineRunnerTest.java similarity index 99% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineRunnerTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineRunnerTest.java index f888c5b04a9a..79e281e70f73 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineRunnerTest.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.apache.beam.sdk.util.WindowedValue.valueInGlobalWindow; @@ -38,6 +38,12 @@ import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner.BatchViewAsList; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner.BatchViewAsMap; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner.BatchViewAsMultimap; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner.TransformedMap; +import org.apache.beam.runners.dataflow.options.DataflowPipelineDebugOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.Pipeline.PipelineVisitor; import org.apache.beam.sdk.coders.BigEndianIntegerCoder; @@ -49,15 +55,10 @@ import org.apache.beam.sdk.io.BigQueryIO; import org.apache.beam.sdk.io.Read; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineDebugOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptions.CheckEnabled; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner.BatchViewAsList; -import org.apache.beam.sdk.runners.DataflowPipelineRunner.BatchViewAsMap; -import org.apache.beam.sdk.runners.DataflowPipelineRunner.BatchViewAsMultimap; -import org.apache.beam.sdk.runners.DataflowPipelineRunner.TransformedMap; +import org.apache.beam.sdk.runners.TransformTreeNode; import org.apache.beam.sdk.runners.dataflow.TestCountingSource; import org.apache.beam.sdk.runners.worker.IsmFormat; import org.apache.beam.sdk.runners.worker.IsmFormat.IsmRecord; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineTranslatorTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineTranslatorTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java index 27c0acc65917..3a39e41e2b7b 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/DataflowPipelineTranslatorTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners; +package org.apache.beam.runners.dataflow; import static org.apache.beam.sdk.util.Structs.addObject; import static org.apache.beam.sdk.util.Structs.getDictionary; @@ -35,16 +35,17 @@ import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; +import org.apache.beam.runners.dataflow.DataflowPipelineTranslator.TranslationContext; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineWorkerPoolOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.coders.Coder; import org.apache.beam.sdk.coders.StringUtf8Coder; import org.apache.beam.sdk.coders.VarIntCoder; import org.apache.beam.sdk.coders.VoidCoder; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; -import org.apache.beam.sdk.options.DataflowPipelineWorkerPoolOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineTranslator.TranslationContext; +import org.apache.beam.sdk.runners.RecordingPipelineVisitor; import org.apache.beam.sdk.transforms.Count; import org.apache.beam.sdk.transforms.Create; import org.apache.beam.sdk.transforms.DoFn; @@ -194,8 +195,9 @@ public void testSettingOfSdkPipelineOptions() throws IOException { Map settings = new HashMap<>(); settings.put("appName", "DataflowPipelineTranslatorTest"); settings.put("project", "some-project"); - settings.put("pathValidatorClass", "org.apache.beam.sdk.util.DataflowPathValidator"); - settings.put("runner", "org.apache.beam.sdk.runners.DataflowPipelineRunner"); + settings.put("pathValidatorClass", + "org.apache.beam.runners.dataflow.util.DataflowPathValidator"); + settings.put("runner", "org.apache.beam.runners.dataflow.DataflowPipelineRunner"); settings.put("jobName", "some-job-name"); settings.put("tempLocation", "gs://somebucket/some/path"); settings.put("stagingLocation", "gs://somebucket/some/path/staging"); diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/dataflow/CustomSourcesTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/internal/CustomSourcesTest.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/dataflow/CustomSourcesTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/internal/CustomSourcesTest.java index 2acede3a5a33..ed86be22d8a9 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/runners/dataflow/CustomSourcesTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/internal/CustomSourcesTest.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.runners.dataflow; +package org.apache.beam.runners.dataflow.internal; import static org.apache.beam.sdk.testing.SourceTestUtils.readFromSource; import static org.hamcrest.MatcherAssert.assertThat; @@ -26,12 +26,12 @@ import static org.junit.Assert.assertNull; import static org.junit.Assert.assertTrue; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.coders.BigEndianIntegerCoder; import org.apache.beam.sdk.coders.Coder; import org.apache.beam.sdk.io.BoundedSource; import org.apache.beam.sdk.io.Read; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.ExpectedLogs; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/io/DataflowTextIOTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/io/DataflowTextIOTest.java similarity index 94% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/io/DataflowTextIOTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/io/DataflowTextIOTest.java index 6b9fbb423649..bfc99e8f06cc 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/io/DataflowTextIOTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/io/DataflowTextIOTest.java @@ -15,12 +15,13 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.io; +package org.apache.beam.runners.dataflow.io; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.testing.TestDataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; +import org.apache.beam.sdk.io.TextIO; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; -import org.apache.beam.sdk.testing.TestDataflowPipelineOptions; import org.apache.beam.sdk.util.GcsUtil; import org.apache.beam.sdk.util.TestCredential; import org.apache.beam.sdk.util.gcsfs.GcsPath; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowPipelineDebugOptionsTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowPipelineDebugOptionsTest.java similarity index 94% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowPipelineDebugOptionsTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowPipelineDebugOptionsTest.java index c3f3a1857e7d..71b6b5721458 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowPipelineDebugOptionsTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowPipelineDebugOptionsTest.java @@ -15,12 +15,14 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; import static org.hamcrest.Matchers.hasEntry; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertThat; +import org.apache.beam.sdk.options.PipelineOptionsFactory; + import org.junit.Test; import org.junit.runner.RunWith; import org.junit.runners.JUnit4; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowPipelineOptionsTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptionsTest.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowPipelineOptionsTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptionsTest.java index c9eac567c145..e7db40f64df8 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowPipelineOptionsTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptionsTest.java @@ -15,11 +15,12 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertTrue; +import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.ResetDateTimeProvider; import org.apache.beam.sdk.testing.RestoreSystemProperties; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowProfilingOptionsTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowProfilingOptionsTest.java similarity index 94% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowProfilingOptionsTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowProfilingOptionsTest.java index 18c808519d52..58f851468460 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowProfilingOptionsTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowProfilingOptionsTest.java @@ -15,11 +15,13 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; import static org.junit.Assert.assertThat; import static org.junit.Assert.assertTrue; +import org.apache.beam.sdk.options.PipelineOptionsFactory; + import com.fasterxml.jackson.databind.ObjectMapper; import org.hamcrest.Matchers; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowWorkerLoggingOptionsTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowWorkerLoggingOptionsTest.java similarity index 91% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowWorkerLoggingOptionsTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowWorkerLoggingOptionsTest.java index 47d518daae3b..5b12ad5f07ad 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/options/DataflowWorkerLoggingOptionsTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/options/DataflowWorkerLoggingOptionsTest.java @@ -15,13 +15,13 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.options; +package org.apache.beam.runners.dataflow.options; -import static org.apache.beam.sdk.options.DataflowWorkerLoggingOptions.Level.WARN; +import static org.apache.beam.runners.dataflow.options.DataflowWorkerLoggingOptions.Level.WARN; import static org.junit.Assert.assertEquals; -import org.apache.beam.sdk.options.DataflowWorkerLoggingOptions.WorkerLogLevelOverrides; +import org.apache.beam.runners.dataflow.options.DataflowWorkerLoggingOptions.WorkerLogLevelOverrides; import com.google.common.collect.ImmutableMap; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/testing/TestDataflowPipelineRunnerTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java similarity index 97% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/testing/TestDataflowPipelineRunnerTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java index 185ab514649e..d6de50158c55 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/testing/TestDataflowPipelineRunnerTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/testing/TestDataflowPipelineRunnerTest.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.testing; +package org.apache.beam.runners.dataflow.testing; import static org.hamcrest.Matchers.containsString; import static org.junit.Assert.assertEquals; @@ -30,15 +30,17 @@ import static org.mockito.Mockito.verify; import static org.mockito.Mockito.when; +import org.apache.beam.runners.dataflow.DataflowPipelineJob; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; +import org.apache.beam.runners.dataflow.util.MonitoringUtil.JobMessagesHandler; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult.State; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineJob; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; +import org.apache.beam.sdk.testing.PAssert; +import org.apache.beam.sdk.testing.TestPipeline; import org.apache.beam.sdk.transforms.Create; import org.apache.beam.sdk.util.GcsUtil; -import org.apache.beam.sdk.util.MonitoringUtil; -import org.apache.beam.sdk.util.MonitoringUtil.JobMessagesHandler; import org.apache.beam.sdk.util.NoopPathValidator; import org.apache.beam.sdk.util.TestCredential; import org.apache.beam.sdk.util.TimeUtil; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/transforms/DataflowGroupByKeyTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/transforms/DataflowGroupByKeyTest.java similarity index 92% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/transforms/DataflowGroupByKeyTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/transforms/DataflowGroupByKeyTest.java index a381f68ae782..f0e677edef45 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/transforms/DataflowGroupByKeyTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/transforms/DataflowGroupByKeyTest.java @@ -15,15 +15,18 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.transforms; +package org.apache.beam.runners.dataflow.transforms; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.coders.BigEndianIntegerCoder; import org.apache.beam.sdk.coders.KvCoder; import org.apache.beam.sdk.coders.StringUtf8Coder; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; +import org.apache.beam.sdk.transforms.Create; +import org.apache.beam.sdk.transforms.GroupByKey; +import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.transforms.windowing.Sessions; import org.apache.beam.sdk.transforms.windowing.Window; import org.apache.beam.sdk.util.NoopPathValidator; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/transforms/DataflowViewTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/transforms/DataflowViewTest.java similarity index 95% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/transforms/DataflowViewTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/transforms/DataflowViewTest.java index b86de7ed4381..d78750076aeb 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/transforms/DataflowViewTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/transforms/DataflowViewTest.java @@ -15,12 +15,15 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.transforms; +package org.apache.beam.runners.dataflow.transforms; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; +import org.apache.beam.sdk.transforms.Create; +import org.apache.beam.sdk.transforms.PTransform; +import org.apache.beam.sdk.transforms.View; import org.apache.beam.sdk.transforms.windowing.FixedWindows; import org.apache.beam.sdk.transforms.windowing.InvalidWindows; import org.apache.beam.sdk.transforms.windowing.Window; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/DataflowPathValidatorTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/DataflowPathValidatorTest.java similarity index 92% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/DataflowPathValidatorTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/DataflowPathValidatorTest.java index b459c47bc38e..55879866aea4 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/DataflowPathValidatorTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/DataflowPathValidatorTest.java @@ -15,15 +15,17 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; import static org.mockito.Matchers.any; import static org.mockito.Matchers.anyString; import static org.mockito.Mockito.when; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; +import org.apache.beam.sdk.util.GcsUtil; +import org.apache.beam.sdk.util.TestCredential; import org.apache.beam.sdk.util.gcsfs.GcsPath; import org.junit.Before; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/MonitoringUtilTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/MonitoringUtilTest.java similarity index 96% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/MonitoringUtilTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/MonitoringUtilTest.java index bdc0bc3339cb..ee1532da80e3 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/MonitoringUtilTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/MonitoringUtilTest.java @@ -15,16 +15,18 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; import static org.junit.Assert.assertEquals; import static org.mockito.Mockito.mock; import static org.mockito.Mockito.verify; import static org.mockito.Mockito.when; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.PipelineResult.State; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; +import org.apache.beam.sdk.util.TestCredential; +import org.apache.beam.sdk.util.TimeUtil; import com.google.api.services.dataflow.Dataflow; import com.google.api.services.dataflow.model.JobMessage; diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/PackageUtilTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/PackageUtilTest.java similarity index 98% rename from runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/PackageUtilTest.java rename to runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/PackageUtilTest.java index 21bc60e702bf..41ad05dc044b 100644 --- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/sdk/util/PackageUtilTest.java +++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/util/PackageUtilTest.java @@ -15,7 +15,7 @@ * See the License for the specific language governing permissions and * limitations under the License. */ -package org.apache.beam.sdk.util; +package org.apache.beam.runners.dataflow.util; import static org.hamcrest.Matchers.equalTo; import static org.hamcrest.collection.IsIterableContainingInAnyOrder.containsInAnyOrder; @@ -32,11 +32,13 @@ import static org.mockito.Mockito.verifyNoMoreInteractions; import static org.mockito.Mockito.when; +import org.apache.beam.runners.dataflow.util.PackageUtil.PackageAttributes; import org.apache.beam.sdk.options.GcsOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; import org.apache.beam.sdk.testing.ExpectedLogs; import org.apache.beam.sdk.testing.FastNanoClockAndSleeper; -import org.apache.beam.sdk.util.PackageUtil.PackageAttributes; +import org.apache.beam.sdk.util.GcsUtil; +import org.apache.beam.sdk.util.IOChannelUtils; import org.apache.beam.sdk.util.gcsfs.GcsPath; import com.google.api.client.googleapis.json.GoogleJsonError.ErrorInfo; diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java index cdd3b8460a4b..a51a24e81270 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/testing/TestPipeline.java @@ -56,7 +56,7 @@ *

  • System property "beamTestPipelineOptions" must contain a JSON delimited list of pipeline * options. For example: *
    {@code [
    - *     "--runner=org.apache.beam.sdk.testing.TestDataflowPipelineRunner",
    + *     "--runner=org.apache.beam.runners.dataflow.testing.TestDataflowPipelineRunner",
      *     "--project=mygcpproject",
      *     "--stagingLocation=gs://mygcsbucket/path"
      *     ]}
    diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/MinimalWordCount.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/MinimalWordCount.java index 4b2bb7b0207e..3e4fc860afe3 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/MinimalWordCount.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/MinimalWordCount.java @@ -17,11 +17,11 @@ */ package ${package}; +import org.apache.beam.runners.dataflow.BlockingDataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; -import org.apache.beam.sdk.runners.BlockingDataflowPipelineRunner; import org.apache.beam.sdk.transforms.Count; import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.ParDo; diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WindowedWordCount.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WindowedWordCount.java index f99752141859..7dea9feb12c6 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WindowedWordCount.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WindowedWordCount.java @@ -17,11 +17,8 @@ */ package ${package}; -import com.google.api.services.bigquery.model.TableFieldSchema; -import com.google.api.services.bigquery.model.TableReference; -import com.google.api.services.bigquery.model.TableRow; -import com.google.api.services.bigquery.model.TableSchema; import ${package}.common.DataflowExampleUtils; + import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.io.BigQueryIO; @@ -37,6 +34,11 @@ import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PCollection; +import com.google.api.services.bigquery.model.TableFieldSchema; +import com.google.api.services.bigquery.model.TableReference; +import com.google.api.services.bigquery.model.TableRow; +import com.google.api.services.bigquery.model.TableSchema; + import org.joda.time.Duration; import org.joda.time.Instant; import org.slf4j.Logger; diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WordCount.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WordCount.java index 8c89283138a8..fc1f4b55f9d2 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WordCount.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/WordCount.java @@ -17,9 +17,9 @@ */ package ${package}; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleOptions.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleOptions.java index 9e7136d94f79..e3bf7c56d691 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleOptions.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleOptions.java @@ -17,7 +17,7 @@ */ package ${package}.common; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleUtils.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleUtils.java index 0ad449a1d7a3..4914d4c0a47e 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleUtils.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/DataflowExampleUtils.java @@ -17,6 +17,11 @@ */ package ${package}.common; +import org.apache.beam.runners.dataflow.DataflowPipelineJob; +import org.apache.beam.runners.dataflow.DataflowPipelineRunner; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.util.MonitoringUtil; + import com.google.api.client.googleapis.json.GoogleJsonResponseException; import com.google.api.client.googleapis.services.AbstractGoogleClientRequest; import com.google.api.services.bigquery.Bigquery; @@ -34,12 +39,8 @@ import org.apache.beam.sdk.PipelineResult; import org.apache.beam.sdk.io.TextIO; import org.apache.beam.sdk.options.BigQueryOptions; -import org.apache.beam.sdk.options.DataflowPipelineOptions; -import org.apache.beam.sdk.runners.DataflowPipelineJob; -import org.apache.beam.sdk.runners.DataflowPipelineRunner; import org.apache.beam.sdk.runners.DirectPipelineRunner; import org.apache.beam.sdk.transforms.IntraBundleParallelization; -import org.apache.beam.sdk.util.MonitoringUtil; import org.apache.beam.sdk.util.Transport; import com.google.common.collect.Lists; import com.google.common.collect.Sets; diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExampleBigQueryTableOptions.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExampleBigQueryTableOptions.java index 9e65c4fab645..279f2e098dde 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExampleBigQueryTableOptions.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExampleBigQueryTableOptions.java @@ -17,13 +17,14 @@ */ package ${package}.common; -import com.google.api.services.bigquery.model.TableSchema; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptions; +import com.google.api.services.bigquery.model.TableSchema; + /** * Options that can be used to configure BigQuery tables in Dataflow examples. * The project defaults to the project being used to run the example. diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExamplePubsubTopicOptions.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExamplePubsubTopicOptions.java index b5acf69bccb8..8a7c9cf0b7f0 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExamplePubsubTopicOptions.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/ExamplePubsubTopicOptions.java @@ -17,7 +17,7 @@ */ package ${package}.common; -import org.apache.beam.sdk.options.DataflowPipelineOptions; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.DefaultValueFactory; import org.apache.beam.sdk.options.Description; diff --git a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/PubsubFileInjector.java b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/PubsubFileInjector.java index d8bab1f78c48..5c182b2bb9e3 100644 --- a/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/PubsubFileInjector.java +++ b/sdks/java/maven-archetypes/examples/src/main/resources/archetype-resources/src/main/java/common/PubsubFileInjector.java @@ -17,12 +17,9 @@ */ package ${package}.common; -import com.google.api.services.pubsub.Pubsub; -import com.google.api.services.pubsub.model.PublishRequest; -import com.google.api.services.pubsub.model.PubsubMessage; +import org.apache.beam.runners.dataflow.options.DataflowPipelineOptions; import org.apache.beam.sdk.Pipeline; import org.apache.beam.sdk.io.TextIO; -import org.apache.beam.sdk.options.DataflowPipelineOptions; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.PipelineOptionsFactory; @@ -30,6 +27,10 @@ import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.IntraBundleParallelization; import org.apache.beam.sdk.util.Transport; + +import com.google.api.services.pubsub.Pubsub; +import com.google.api.services.pubsub.model.PublishRequest; +import com.google.api.services.pubsub.model.PubsubMessage; import com.google.common.collect.ImmutableMap; import java.io.IOException; From 0fafd4e89f387b73c33aac65d42a6a367e9dd738 Mon Sep 17 00:00:00 2001 From: Davor Bonaci Date: Tue, 26 Apr 2016 16:38:06 -0700 Subject: [PATCH 5/5] Update Dataflow worker harness container image to match package changes in this pull request --- .../apache/beam/runners/dataflow/DataflowPipelineRunner.java | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunner.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunner.java index 0fc095a14094..ec4a60c305be 100644 --- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunner.java +++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineRunner.java @@ -217,9 +217,9 @@ public class DataflowPipelineRunner extends PipelineRunner // Default Docker container images that execute Dataflow worker harness, residing in Google // Container Registry, separately for Batch and Streaming. public static final String BATCH_WORKER_HARNESS_CONTAINER_IMAGE - = "dataflow.gcr.io/v1beta3/beam-java-batch:beam-master-20160422"; + = "dataflow.gcr.io/v1beta3/beam-java-batch:beam-master-20160426"; public static final String STREAMING_WORKER_HARNESS_CONTAINER_IMAGE - = "dataflow.gcr.io/v1beta3/beam-java-streaming:beam-master-20160422"; + = "dataflow.gcr.io/v1beta3/beam-java-streaming:beam-master-20160426"; // The limit of CreateJob request size. private static final int CREATE_JOB_REQUEST_LIMIT_BYTES = 10 * 1024 * 1024;