You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by ke...@apache.org on 2017/05/23 18:18:09 UTC

[10/16] beam git commit: Remove writePCollectionViewData from the Beam codebase

Remove writePCollectionViewData from the Beam codebase


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

Branch: refs/heads/master
Commit: 32c6cb160f42e401f3e170cc8ed18d76c627d3e4
Parents: 5ac24e0
Author: Kenneth Knowles <kl...@google.com>
Authored: Mon May 22 16:26:00 2017 -0700
Committer: Kenneth Knowles <kl...@google.com>
Committed: Tue May 23 11:16:27 2017 -0700

----------------------------------------------------------------------
 .../apex/translation/utils/NoOpStepContext.java    | 13 -------------
 .../apache/beam/runners/core/BaseStepContext.java  | 14 --------------
 .../org/apache/beam/runners/core/StepContext.java  | 17 -----------------
 .../runners/direct/DirectExecutionContext.java     | 13 -------------
 .../functions/FlinkNoOpStepContext.java            | 16 +---------------
 .../wrappers/streaming/DoFnOperator.java           | 11 -----------
 .../spark/translation/SparkProcessContext.java     | 14 +-------------
 .../beam/fn/harness/fake/FakeStepContext.java      | 16 +---------------
 8 files changed, 3 insertions(+), 111 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/NoOpStepContext.java
----------------------------------------------------------------------
diff --git a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/NoOpStepContext.java b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/NoOpStepContext.java
index 51e843b..820b189 100644
--- a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/NoOpStepContext.java
+++ b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/NoOpStepContext.java
@@ -17,15 +17,10 @@
  */
 package org.apache.beam.runners.apex.translation.utils;
 
-import java.io.IOException;
 import java.io.Serializable;
 import org.apache.beam.runners.core.StateInternals;
 import org.apache.beam.runners.core.StepContext;
 import org.apache.beam.runners.core.TimerInternals;
-import org.apache.beam.sdk.coders.Coder;
-import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
-import org.apache.beam.sdk.util.WindowedValue;
-import org.apache.beam.sdk.values.TupleTag;
 
 /**
  * Serializable {@link StepContext} that does nothing.
@@ -44,14 +39,6 @@ public class NoOpStepContext implements StepContext, Serializable {
   }
 
   @Override
-  public <T, W extends BoundedWindow> void writePCollectionViewData(TupleTag<?> tag,
-      Iterable<WindowedValue<T>> data,
-      Coder<Iterable<WindowedValue<T>>> dataCoder, W window, Coder<W> windowCoder) throws
-      IOException {
-
-  }
-
-  @Override
   public StateInternals stateInternals() {
     return null;
   }

http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/runners/core-java/src/main/java/org/apache/beam/runners/core/BaseStepContext.java
----------------------------------------------------------------------
diff --git a/runners/core-java/src/main/java/org/apache/beam/runners/core/BaseStepContext.java b/runners/core-java/src/main/java/org/apache/beam/runners/core/BaseStepContext.java
index 014fe0d..e639c46 100644
--- a/runners/core-java/src/main/java/org/apache/beam/runners/core/BaseStepContext.java
+++ b/runners/core-java/src/main/java/org/apache/beam/runners/core/BaseStepContext.java
@@ -17,12 +17,6 @@
  */
 package org.apache.beam.runners.core;
 
-import java.io.IOException;
-import org.apache.beam.sdk.coders.Coder;
-import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
-import org.apache.beam.sdk.util.WindowedValue;
-import org.apache.beam.sdk.values.TupleTag;
-
 /**
  * Base class for implementations of {@link StepContext}.
  *
@@ -49,14 +43,6 @@ public abstract class BaseStepContext implements StepContext {
   }
 
   @Override
-  public <T, W extends BoundedWindow> void writePCollectionViewData(
-      TupleTag<?> tag,
-      Iterable<WindowedValue<T>> data, Coder<Iterable<WindowedValue<T>>> dataCoder,
-      W window, Coder<W> windowCoder) throws IOException {
-    throw new UnsupportedOperationException("Not implemented.");
-  }
-
-  @Override
   public abstract StateInternals stateInternals();
 
   @Override

http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/runners/core-java/src/main/java/org/apache/beam/runners/core/StepContext.java
----------------------------------------------------------------------
diff --git a/runners/core-java/src/main/java/org/apache/beam/runners/core/StepContext.java b/runners/core-java/src/main/java/org/apache/beam/runners/core/StepContext.java
index fd2575d..62a81f1 100644
--- a/runners/core-java/src/main/java/org/apache/beam/runners/core/StepContext.java
+++ b/runners/core-java/src/main/java/org/apache/beam/runners/core/StepContext.java
@@ -17,12 +17,6 @@
  */
 package org.apache.beam.runners.core;
 
-import java.io.IOException;
-import org.apache.beam.sdk.coders.Coder;
-import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
-import org.apache.beam.sdk.util.WindowedValue;
-import org.apache.beam.sdk.values.TupleTag;
-
 /**
  * Per-step, per-key context used for retrieving state.
  */
@@ -38,17 +32,6 @@ public interface StepContext {
    */
   String getTransformName();
 
-  /**
-   * Writes the given {@code PCollectionView} data to a globally accessible location.
-   */
-  <T, W extends BoundedWindow> void writePCollectionViewData(
-      TupleTag<?> tag,
-      Iterable<WindowedValue<T>> data,
-      Coder<Iterable<WindowedValue<T>>> dataCoder,
-      W window,
-      Coder<W> windowCoder)
-          throws IOException;
-
   StateInternals stateInternals();
 
   TimerInternals timerInternals();

http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectExecutionContext.java
----------------------------------------------------------------------
diff --git a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectExecutionContext.java b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectExecutionContext.java
index 2a75ef5..39174d6 100644
--- a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectExecutionContext.java
+++ b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectExecutionContext.java
@@ -17,7 +17,6 @@
  */
 package org.apache.beam.runners.direct;
 
-import java.io.IOException;
 import org.apache.beam.runners.core.BaseExecutionContext;
 import org.apache.beam.runners.core.ExecutionContext;
 import org.apache.beam.runners.core.StepContext;
@@ -25,10 +24,6 @@ import org.apache.beam.runners.core.TimerInternals;
 import org.apache.beam.runners.direct.DirectExecutionContext.DirectStepContext;
 import org.apache.beam.runners.direct.WatermarkManager.TimerUpdate;
 import org.apache.beam.runners.direct.WatermarkManager.TransformWatermarks;
-import org.apache.beam.sdk.coders.Coder;
-import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
-import org.apache.beam.sdk.util.WindowedValue;
-import org.apache.beam.sdk.values.TupleTag;
 
 /**
  * Execution Context for the {@link DirectRunner}.
@@ -112,14 +107,6 @@ class DirectExecutionContext
       return transformName;
     }
 
-    @Override
-    public <T, W extends BoundedWindow> void writePCollectionViewData(
-        TupleTag<?> tag,
-        Iterable<WindowedValue<T>> data, Coder<Iterable<WindowedValue<T>>> dataCoder,
-        W window, Coder<W> windowCoder) throws IOException {
-      throw new UnsupportedOperationException("Not implemented.");
-    }
-
     /**
      * Gets the timer update of the {@link TimerInternals} of this {@link DirectStepContext},
      * which is empty if the {@link TimerInternals} were never accessed.

http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkNoOpStepContext.java
----------------------------------------------------------------------
diff --git a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkNoOpStepContext.java b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkNoOpStepContext.java
index d999494..1ff322e 100644
--- a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkNoOpStepContext.java
+++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/functions/FlinkNoOpStepContext.java
@@ -17,14 +17,9 @@
  */
 package org.apache.beam.runners.flink.translation.functions;
 
-import java.io.IOException;
-import org.apache.beam.runners.core.StepContext;
 import org.apache.beam.runners.core.StateInternals;
+import org.apache.beam.runners.core.StepContext;
 import org.apache.beam.runners.core.TimerInternals;
-import org.apache.beam.sdk.coders.Coder;
-import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
-import org.apache.beam.sdk.util.WindowedValue;
-import org.apache.beam.sdk.values.TupleTag;
 
 /**
  * A {@link StepContext} for Flink Batch Runner execution.
@@ -42,15 +37,6 @@ public class FlinkNoOpStepContext implements StepContext {
   }
 
   @Override
-  public <T, W extends BoundedWindow> void writePCollectionViewData(
-      TupleTag<?> tag,
-      Iterable<WindowedValue<T>> data,
-      Coder<Iterable<WindowedValue<T>>> dataCoder,
-      W window,
-      Coder<W> windowCoder) throws IOException {
-  }
-
-  @Override
   public StateInternals stateInternals() {
     return null;
   }

http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java
----------------------------------------------------------------------
diff --git a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java
index 2bb9c20..4f8998e 100644
--- a/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java
+++ b/runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java
@@ -23,7 +23,6 @@ import com.google.common.base.Optional;
 import com.google.common.collect.Iterables;
 import java.io.DataInputStream;
 import java.io.DataOutputStream;
-import java.io.IOException;
 import java.io.Serializable;
 import java.util.ArrayList;
 import java.util.Collection;
@@ -688,16 +687,6 @@ public class DoFnOperator<InputT, FnOutputT, OutputT>
     }
 
     @Override
-    public <T, W extends BoundedWindow> void writePCollectionViewData(
-        TupleTag<?> tag,
-        Iterable<WindowedValue<T>> data,
-        Coder<Iterable<WindowedValue<T>>> dataCoder,
-        W window,
-        Coder<W> windowCoder) throws IOException {
-      throw new UnsupportedOperationException("Writing side-input data is not supported.");
-    }
-
-    @Override
     public StateInternals stateInternals() {
       return stateInternals;
     }

http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkProcessContext.java
----------------------------------------------------------------------
diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkProcessContext.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkProcessContext.java
index 31e616c..e693143 100644
--- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkProcessContext.java
+++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkProcessContext.java
@@ -20,19 +20,15 @@ package org.apache.beam.runners.spark.translation;
 
 import com.google.common.collect.AbstractIterator;
 import com.google.common.collect.Lists;
-import java.io.IOException;
 import java.util.Iterator;
 import org.apache.beam.runners.core.DoFnRunner;
 import org.apache.beam.runners.core.DoFnRunners.OutputManager;
-import org.apache.beam.runners.core.StepContext;
 import org.apache.beam.runners.core.StateInternals;
+import org.apache.beam.runners.core.StepContext;
 import org.apache.beam.runners.core.TimerInternals;
-import org.apache.beam.sdk.coders.Coder;
 import org.apache.beam.sdk.transforms.DoFn;
 import org.apache.beam.sdk.transforms.reflect.DoFnInvokers;
-import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
 import org.apache.beam.sdk.util.WindowedValue;
-import org.apache.beam.sdk.values.TupleTag;
 
 
 /**
@@ -110,14 +106,6 @@ class SparkProcessContext<FnInputT, FnOutputT, OutputT> {
     }
 
     @Override
-    public <T, W extends BoundedWindow> void writePCollectionViewData(
-        TupleTag<?> tag,
-        Iterable<WindowedValue<T>> data,
-        Coder<Iterable<WindowedValue<T>>> dataCoder,
-        W window,
-        Coder<W> windowCoder) throws IOException { }
-
-    @Override
     public StateInternals stateInternals() {
       return null;
     }

http://git-wip-us.apache.org/repos/asf/beam/blob/32c6cb16/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/fake/FakeStepContext.java
----------------------------------------------------------------------
diff --git a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/fake/FakeStepContext.java b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/fake/FakeStepContext.java
index 750c167..3f6a2da 100644
--- a/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/fake/FakeStepContext.java
+++ b/sdks/java/harness/src/main/java/org/apache/beam/fn/harness/fake/FakeStepContext.java
@@ -18,14 +18,9 @@
 
 package org.apache.beam.fn.harness.fake;
 
-import java.io.IOException;
-import org.apache.beam.runners.core.StepContext;
 import org.apache.beam.runners.core.StateInternals;
+import org.apache.beam.runners.core.StepContext;
 import org.apache.beam.runners.core.TimerInternals;
-import org.apache.beam.sdk.coders.Coder;
-import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
-import org.apache.beam.sdk.util.WindowedValue;
-import org.apache.beam.sdk.values.TupleTag;
 
 /**
  * A fake {@link StepContext} factory that performs no-ops.
@@ -42,15 +37,6 @@ public class FakeStepContext implements StepContext {
   }
 
   @Override
-  public <T, W extends BoundedWindow> void writePCollectionViewData(
-      TupleTag<?> tag,
-      Iterable<WindowedValue<T>> data,
-      Coder<Iterable<WindowedValue<T>>> dataCoder,
-      W window,
-      Coder<W> windowCoder) throws IOException {
-  }
-
-  @Override
   public StateInternals stateInternals() {
     throw new UnsupportedOperationException();
   }