Repository: beam
Updated Branches:
  refs/heads/master ca657c4f7 -> 217f085f2


Add GroupByKey translation


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

Branch: refs/heads/master
Commit: 3dd2fb1e951bbfde8053bccef32aa73c51f9845d
Parents: 82b81e5
Author: Kenneth Knowles <k...@google.com>
Authored: Thu May 25 06:45:27 2017 -0700
Committer: Kenneth Knowles <k...@google.com>
Committed: Thu Jun 1 12:39:39 2017 -0700

----------------------------------------------------------------------
 .../construction/GroupByKeyTranslation.java     | 62 ++++++++++++++++++++
 .../construction/GroupByKeyTranslationTest.java | 44 ++++++++++++++
 2 files changed, 106 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/beam/blob/3dd2fb1e/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/GroupByKeyTranslation.java
----------------------------------------------------------------------
diff --git 
a/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/GroupByKeyTranslation.java
 
b/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/GroupByKeyTranslation.java
new file mode 100644
index 0000000..db73461
--- /dev/null
+++ 
b/runners/core-construction-java/src/main/java/org/apache/beam/runners/core/construction/GroupByKeyTranslation.java
@@ -0,0 +1,62 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.beam.runners.core.construction;
+
+import com.google.auto.service.AutoService;
+import java.util.Collections;
+import java.util.Map;
+import 
org.apache.beam.runners.core.construction.PTransformTranslation.TransformPayloadTranslator;
+import org.apache.beam.sdk.common.runner.v1.RunnerApi;
+import org.apache.beam.sdk.common.runner.v1.RunnerApi.FunctionSpec;
+import org.apache.beam.sdk.runners.AppliedPTransform;
+import org.apache.beam.sdk.transforms.GroupByKey;
+import org.apache.beam.sdk.transforms.PTransform;
+
+/**
+ * Utility methods for translating a {@link GroupByKey} to and from {@link 
RunnerApi}
+ * representations.
+ */
+public class GroupByKeyTranslation {
+
+  static class GroupByKeyTranslator implements 
TransformPayloadTranslator<GroupByKey<?, ?>> {
+    @Override
+    public String getUrn(GroupByKey<?, ?> transform) {
+      return PTransformTranslation.GROUP_BY_KEY_TRANSFORM_URN;
+    }
+
+    @Override
+    public FunctionSpec translate(
+        AppliedPTransform<?, ?, GroupByKey<?, ?>> transform, SdkComponents 
components) {
+      return FunctionSpec.newBuilder()
+          .setUrn(getUrn(transform.getTransform()))
+          .build();
+    }
+  }
+
+
+  /** Registers {@link GroupByKeyTranslator}. */
+  @AutoService(TransformPayloadTranslatorRegistrar.class)
+  public static class Registrar implements TransformPayloadTranslatorRegistrar 
{
+    @Override
+    public Map<? extends Class<? extends PTransform>, ? extends 
TransformPayloadTranslator>
+        getTransformPayloadTranslators() {
+      return Collections.singletonMap(GroupByKey.class, new 
GroupByKeyTranslator());
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/beam/blob/3dd2fb1e/runners/core-construction-java/src/test/java/org/apache/beam/runners/core/construction/GroupByKeyTranslationTest.java
----------------------------------------------------------------------
diff --git 
a/runners/core-construction-java/src/test/java/org/apache/beam/runners/core/construction/GroupByKeyTranslationTest.java
 
b/runners/core-construction-java/src/test/java/org/apache/beam/runners/core/construction/GroupByKeyTranslationTest.java
new file mode 100644
index 0000000..22681f7
--- /dev/null
+++ 
b/runners/core-construction-java/src/test/java/org/apache/beam/runners/core/construction/GroupByKeyTranslationTest.java
@@ -0,0 +1,44 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.beam.runners.core.construction;
+
+import static 
org.apache.beam.runners.core.construction.PTransformTranslation.GROUP_BY_KEY_TRANSFORM_URN;
+import static org.hamcrest.Matchers.equalTo;
+import static org.junit.Assert.assertThat;
+
+import org.apache.beam.sdk.transforms.GroupByKey;
+import org.junit.Test;
+import org.junit.runner.RunWith;
+import org.junit.runners.JUnit4;
+
+/** Tests for {@link GroupByKeyTranslation}. */
+@RunWith(JUnit4.class)
+public class GroupByKeyTranslationTest {
+
+  /**
+   * Tests that the translator is registered so the URN can be retrieved (the 
only thing you can
+   * meaningfully do with a {@link GroupByKey}).
+   */
+  @Test
+  public void testUrnRetrievable() throws Exception {
+    assertThat(
+        PTransformTranslation.urnForTransform(GroupByKey.create()),
+        equalTo(GROUP_BY_KEY_TRANSFORM_URN));
+  }
+}

Reply via email to