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/26 04:08:09 UTC

[03/12] beam git commit: Centralize primitive URNs in PTransformTranslation class

Centralize primitive URNs in PTransformTranslation class


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

Branch: refs/heads/master
Commit: 1dc134cf341d1eb4fa936f6fe7b83a3edbb64687
Parents: 26c9368
Author: Kenneth Knowles <kl...@google.com>
Authored: Fri May 19 20:54:36 2017 -0700
Committer: Kenneth Knowles <kl...@google.com>
Committed: Thu May 25 11:14:29 2017 -0700

----------------------------------------------------------------------
 .../runners/core/construction/PTransformTranslation.java    | 7 +++++++
 .../beam/runners/core/construction/ParDoTranslation.java    | 9 +++------
 2 files changed, 10 insertions(+), 6 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/beam/blob/1dc134cf/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/PTransformTranslation.java
----------------------------------------------------------------------
diff --git a/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/PTransformTranslation.java b/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/PTransformTranslation.java
index 86638de..8be023a 100644
--- a/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/PTransformTranslation.java
+++ b/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/PTransformTranslation.java
@@ -38,6 +38,13 @@ import org.apache.beam.sdk.values.TupleTag;
  * protocol buffers}.
  */
 public class PTransformTranslation {
+
+  public static final String PAR_DO_TRANSFORM_URN = "urn:beam:transform:pardo:v1";
+  public static final String FLATTEN_TRANSFORM_URN = "urn:beam:transform:flatten:v1";
+  public static final String GROUP_BY_KEY_TRANSFORM_URN = "urn:beam:transform:groupbykey:v1";
+  public static final String READ_TRANSFORM_URN = "urn:beam:transform:read:v1";
+  public static final String WINDOW_TRANSFORM_URN = "urn:beam:transform:window:v1";
+
   private static final Map<Class<? extends PTransform>, TransformPayloadTranslator>
       KNOWN_PAYLOAD_TRANSLATORS = loadTransformPayloadTranslators();
 

http://git-wip-us.apache.org/repos/asf/beam/blob/1dc134cf/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/ParDoTranslation.java
----------------------------------------------------------------------
diff --git a/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/ParDoTranslation.java b/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/ParDoTranslation.java
index 28d577f..83277bb 100644
--- a/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/ParDoTranslation.java
+++ b/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/ParDoTranslation.java
@@ -19,6 +19,7 @@
 package org.apache.beam.runners.core.construction;
 
 import static com.google.common.base.Preconditions.checkArgument;
+import static org.apache.beam.runners.core.construction.PTransformTranslation.PAR_DO_TRANSFORM_URN;
 
 import com.google.auto.service.AutoService;
 import com.google.auto.value.AutoValue;
@@ -75,10 +76,6 @@ import org.apache.beam.sdk.values.WindowingStrategy;
  */
 public class ParDoTranslation {
   /**
-   * The URN for a {@link ParDoPayload}.
-   */
-  public static final String PAR_DO_PAYLOAD_URN = "urn:beam:pardo:v1";
-  /**
    * The URN for an unknown Java {@link DoFn}.
    */
   public static final String CUSTOM_JAVA_DO_FN_URN = "urn:beam:dofn:javasdk:0.1";
@@ -108,7 +105,7 @@ public class ParDoTranslation {
         AppliedPTransform<?, ?, MultiOutput<?, ?>> transform, SdkComponents components) {
       ParDoPayload payload = toProto(transform.getTransform(), components);
       return RunnerApi.FunctionSpec.newBuilder()
-          .setUrn(PAR_DO_PAYLOAD_URN)
+          .setUrn(PAR_DO_TRANSFORM_URN)
           .setParameter(Any.pack(payload))
           .build();
     }
@@ -166,7 +163,7 @@ public class ParDoTranslation {
   public static RunnerApi.PCollection getMainInput(
       RunnerApi.PTransform ptransform, Components components) throws IOException {
     checkArgument(
-        ptransform.getSpec().getUrn().equals(PAR_DO_PAYLOAD_URN),
+        ptransform.getSpec().getUrn().equals(PAR_DO_TRANSFORM_URN),
         "Unexpected payload type %s",
         ptransform.getSpec().getUrn());
     ParDoPayload payload = ptransform.getSpec().getParameter().unpack(ParDoPayload.class);