You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by am...@apache.org on 2016/12/09 16:11:50 UTC

[1/2] incubator-beam git commit: Remove misc uses of OldDoFn

Repository: incubator-beam
Updated Branches:
  refs/heads/master 63d197cd0 -> d9a631173


Remove misc uses of OldDoFn


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/e48b0e6b
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/e48b0e6b
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/e48b0e6b

Branch: refs/heads/master
Commit: e48b0e6bc20d8eba2968decf7ac2b4ee7503a4df
Parents: 63d197c
Author: Kenneth Knowles <kl...@google.com>
Authored: Thu Dec 8 23:33:40 2016 -0800
Committer: Sela <an...@paypal.com>
Committed: Fri Dec 9 18:00:39 2016 +0200

----------------------------------------------------------------------
 .../game/utils/WriteWindowedToBigQuery.java        |  5 +----
 .../runners/dataflow/internal/AssignWindows.java   |  8 ++++----
 .../dataflow/DataflowPipelineTranslatorTest.java   | 17 +++++++++--------
 3 files changed, 14 insertions(+), 16 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/e48b0e6b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/utils/WriteWindowedToBigQuery.java
----------------------------------------------------------------------
diff --git a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/utils/WriteWindowedToBigQuery.java b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/utils/WriteWindowedToBigQuery.java
index c32289f..7a4fb2c 100644
--- a/examples/java8/src/main/java/org/apache/beam/examples/complete/game/utils/WriteWindowedToBigQuery.java
+++ b/examples/java8/src/main/java/org/apache/beam/examples/complete/game/utils/WriteWindowedToBigQuery.java
@@ -23,7 +23,6 @@ import org.apache.beam.sdk.io.gcp.bigquery.BigQueryIO;
 import org.apache.beam.sdk.io.gcp.bigquery.BigQueryIO.Write.CreateDisposition;
 import org.apache.beam.sdk.io.gcp.bigquery.BigQueryIO.Write.WriteDisposition;
 import org.apache.beam.sdk.transforms.DoFn;
-import org.apache.beam.sdk.transforms.OldDoFn.RequiresWindowAccess;
 import org.apache.beam.sdk.transforms.ParDo;
 import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
 import org.apache.beam.sdk.values.PCollection;
@@ -43,9 +42,7 @@ public class WriteWindowedToBigQuery<T>
   }
 
   /** Convert each key/score pair into a BigQuery TableRow. */
-  protected class BuildRowFn extends DoFn<T, TableRow>
-      implements RequiresWindowAccess {
-
+  protected class BuildRowFn extends DoFn<T, TableRow> {
     @ProcessElement
     public void processElement(ProcessContext c, BoundedWindow window) {
 

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/e48b0e6b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java
----------------------------------------------------------------------
diff --git a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java
index 68ee7bc..27fe13d 100644
--- a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java
+++ b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/internal/AssignWindows.java
@@ -18,7 +18,7 @@
 package org.apache.beam.runners.dataflow.internal;
 
 import org.apache.beam.sdk.coders.Coder;
-import org.apache.beam.sdk.transforms.OldDoFn;
+import org.apache.beam.sdk.transforms.DoFn;
 import org.apache.beam.sdk.transforms.PTransform;
 import org.apache.beam.sdk.transforms.ParDo;
 import org.apache.beam.sdk.transforms.windowing.Window;
@@ -63,9 +63,9 @@ public class AssignWindows<T> extends PTransform<PCollection<T>, PCollection<T>>
     } else {
       // If the windowFn didn't change, we just run a pass-through transform and then set the
       // new windowing strategy.
-      return input.apply("Identity", ParDo.of(new OldDoFn<T, T>() {
-        @Override
-        public void processElement(OldDoFn<T, T>.ProcessContext c) throws Exception {
+      return input.apply("Identity", ParDo.of(new DoFn<T, T>() {
+        @ProcessElement
+        public void processElement(ProcessContext c) throws Exception {
           c.output(c.element());
         }
       })).setWindowingStrategyInternal(outputStrategy);

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/e48b0e6b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
----------------------------------------------------------------------
diff --git a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
index ac4f2df..8d0b83a 100644
--- a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
+++ b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
@@ -70,7 +70,7 @@ import org.apache.beam.sdk.options.PipelineOptionsFactory;
 import org.apache.beam.sdk.options.ValueProvider;
 import org.apache.beam.sdk.transforms.Count;
 import org.apache.beam.sdk.transforms.Create;
-import org.apache.beam.sdk.transforms.OldDoFn;
+import org.apache.beam.sdk.transforms.DoFn;
 import org.apache.beam.sdk.transforms.PTransform;
 import org.apache.beam.sdk.transforms.ParDo;
 import org.apache.beam.sdk.transforms.Sum;
@@ -508,7 +508,7 @@ public class DataflowPipelineTranslatorTest implements Serializable {
   }
 
   /**
-   * Returns a Step for a OldDoFn by creating and translating a pipeline.
+   * Returns a Step for a {@link DoFn} by creating and translating a pipeline.
    */
   private static Step createPredefinedStep() throws Exception {
     DataflowPipelineOptions options = buildPipelineOptions();
@@ -533,8 +533,9 @@ public class DataflowPipelineTranslatorTest implements Serializable {
     return step;
   }
 
-  private static class NoOpFn extends OldDoFn<String, String> {
-    @Override public void processElement(ProcessContext c) throws Exception {
+  private static class NoOpFn extends DoFn<String, String> {
+    @ProcessElement
+    public void processElement(ProcessContext c) throws Exception {
       c.output(c.element());
     }
   }
@@ -899,8 +900,8 @@ public class DataflowPipelineTranslatorTest implements Serializable {
     DataflowPipelineTranslator translator = DataflowPipelineTranslator.fromOptions(options);
     Pipeline pipeline = Pipeline.create(options);
 
-    OldDoFn<Integer, Integer> fn1 = new OldDoFn<Integer, Integer>() {
-      @Override
+    DoFn<Integer, Integer> fn1 = new DoFn<Integer, Integer>() {
+      @ProcessElement
       public void processElement(ProcessContext c) throws Exception {
         c.output(c.element());
       }
@@ -915,8 +916,8 @@ public class DataflowPipelineTranslatorTest implements Serializable {
       }
     };
 
-    OldDoFn<Integer, Integer> fn2 = new OldDoFn<Integer, Integer>() {
-      @Override
+    DoFn<Integer, Integer> fn2 = new DoFn<Integer, Integer>() {
+      @ProcessElement
       public void processElement(ProcessContext c) throws Exception {
         c.output(c.element());
       }


[2/2] incubator-beam git commit: This closes #1561

Posted by am...@apache.org.
This closes #1561


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/d9a63117
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/d9a63117
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/d9a63117

Branch: refs/heads/master
Commit: d9a6311734064b1c7171b943eeb511c4d648187a
Parents: 63d197c e48b0e6
Author: Sela <an...@paypal.com>
Authored: Fri Dec 9 18:01:12 2016 +0200
Committer: Sela <an...@paypal.com>
Committed: Fri Dec 9 18:01:12 2016 +0200

----------------------------------------------------------------------
 .../game/utils/WriteWindowedToBigQuery.java        |  5 +----
 .../runners/dataflow/internal/AssignWindows.java   |  8 ++++----
 .../dataflow/DataflowPipelineTranslatorTest.java   | 17 +++++++++--------
 3 files changed, 14 insertions(+), 16 deletions(-)
----------------------------------------------------------------------