Repository: incubator-beam
Updated Branches:
  refs/heads/master 4da935b65 -> 419b6f4ae


BigtableIO.Read: use PBegin, rather than PInput

Sources should start from the beginning of a pipeline.


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

Branch: refs/heads/master
Commit: d91bc091f6a82b78a5e14981525da815fc6046bb
Parents: 4da935b
Author: Dan Halperin <dhalp...@google.com>
Authored: Thu Mar 3 18:40:12 2016 -0800
Committer: Luke Cwik <lc...@google.com>
Committed: Fri Mar 4 11:51:26 2016 -0800

----------------------------------------------------------------------
 .../google/cloud/dataflow/sdk/io/bigtable/BigtableIO.java    | 8 ++++----
 1 file changed, 4 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/d91bc091/sdk/src/main/java/com/google/cloud/dataflow/sdk/io/bigtable/BigtableIO.java
----------------------------------------------------------------------
diff --git 
a/sdk/src/main/java/com/google/cloud/dataflow/sdk/io/bigtable/BigtableIO.java 
b/sdk/src/main/java/com/google/cloud/dataflow/sdk/io/bigtable/BigtableIO.java
index c3f233f..562d253 100644
--- 
a/sdk/src/main/java/com/google/cloud/dataflow/sdk/io/bigtable/BigtableIO.java
+++ 
b/sdk/src/main/java/com/google/cloud/dataflow/sdk/io/bigtable/BigtableIO.java
@@ -41,9 +41,9 @@ import com.google.cloud.dataflow.sdk.runners.PipelineRunner;
 import com.google.cloud.dataflow.sdk.transforms.PTransform;
 import com.google.cloud.dataflow.sdk.util.DataflowReleaseInfo;
 import com.google.cloud.dataflow.sdk.values.KV;
+import com.google.cloud.dataflow.sdk.values.PBegin;
 import com.google.cloud.dataflow.sdk.values.PCollection;
 import com.google.cloud.dataflow.sdk.values.PDone;
-import com.google.cloud.dataflow.sdk.values.PInput;
 import com.google.common.base.MoreObjects;
 import com.google.common.collect.ImmutableList;
 import com.google.common.util.concurrent.FutureCallback;
@@ -176,7 +176,7 @@ public class BigtableIO {
    * @see BigtableIO
    */
   @Experimental
-  public static class Read extends PTransform<PInput, PCollection<Row>> {
+  public static class Read extends PTransform<PBegin, PCollection<Row>> {
     /**
      * Returns a new {@link BigtableIO.Read} that will read from the Cloud 
Bigtable cluster
      * indicated by the given options, and using any other specified 
customizations.
@@ -241,14 +241,14 @@ public class BigtableIO {
     }
 
     @Override
-    public PCollection<Row> apply(PInput input) {
+    public PCollection<Row> apply(PBegin input) {
       BigtableSource source =
           new BigtableSource(getBigtableService(), tableId, filter, 
ByteKeyRange.ALL_KEYS, null);
       return 
input.getPipeline().apply(com.google.cloud.dataflow.sdk.io.Read.from(source));
     }
 
     @Override
-    public void validate(PInput input) {
+    public void validate(PBegin input) {
       checkArgument(options != null, "BigtableOptions not specified");
       checkArgument(!tableId.isEmpty(), "Table ID not specified");
       try {

Reply via email to