You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by lc...@apache.org on 2017/04/26 15:27:38 UTC

[1/2] beam git commit: [BEAM-1871] Move ByteStringCoder to sdks/java/extensions/protobuf

Repository: beam
Updated Branches:
  refs/heads/master 009469972 -> b29d1c83b


[BEAM-1871] Move ByteStringCoder to sdks/java/extensions/protobuf


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

Branch: refs/heads/master
Commit: 03d47845fc9de073d9bbdc242135cb55f52b56cc
Parents: 0094699
Author: Luke Cwik <lc...@google.com>
Authored: Tue Apr 25 14:23:11 2017 -0700
Committer: Luke Cwik <lc...@google.com>
Committed: Wed Apr 26 08:26:40 2017 -0700

----------------------------------------------------------------------
 .../apache/beam/sdk/coders/ByteStringCoder.java | 114 ----------------
 .../apache/beam/sdk/coders/CoderRegistry.java   |   2 -
 .../beam/sdk/coders/ByteStringCoderTest.java    | 128 ------------------
 sdks/java/extensions/protobuf/pom.xml           |   6 +
 .../extensions/protobuf/ByteStringCoder.java    | 118 +++++++++++++++++
 .../protobuf/ProtobufCoderRegistrar.java        |  39 ++++++
 .../protobuf/ByteStringCoderTest.java           | 131 +++++++++++++++++++
 .../sdk/io/gcp/bigtable/BigtableIOTest.java     |   2 +-
 8 files changed, 295 insertions(+), 245 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/ByteStringCoder.java
----------------------------------------------------------------------
diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/ByteStringCoder.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/ByteStringCoder.java
deleted file mode 100644
index 1b27b5b..0000000
--- a/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/ByteStringCoder.java
+++ /dev/null
@@ -1,114 +0,0 @@
-/*
- * 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.sdk.coders;
-
-import com.google.common.io.ByteStreams;
-import com.google.protobuf.ByteString;
-import java.io.IOException;
-import java.io.InputStream;
-import java.io.OutputStream;
-import org.apache.beam.sdk.util.VarInt;
-import org.apache.beam.sdk.values.TypeDescriptor;
-
-/**
- * A {@link Coder} for {@link ByteString} objects based on their encoded Protocol Buffer form.
- *
- * <p>When this code is used in a nested {@link Coder.Context}, the serialized {@link ByteString}
- * objects are first delimited by their size.
- */
-public class ByteStringCoder extends CustomCoder<ByteString> {
-
-  public static ByteStringCoder of() {
-    return INSTANCE;
-  }
-
-  /***************************/
-
-  private static final ByteStringCoder INSTANCE = new ByteStringCoder();
-  private static final TypeDescriptor<ByteString> TYPE_DESCRIPTOR =
-      new TypeDescriptor<ByteString>() {};
-
-  private ByteStringCoder() {}
-
-  @Override
-  public void encode(ByteString value, OutputStream outStream, Context context)
-      throws IOException, CoderException {
-    if (value == null) {
-      throw new CoderException("cannot encode a null ByteString");
-    }
-
-    if (!context.isWholeStream) {
-      // ByteString is not delimited, so write its size before its contents.
-      VarInt.encode(value.size(), outStream);
-    }
-    value.writeTo(outStream);
-  }
-
-  @Override
-  public ByteString decode(InputStream inStream, Context context) throws IOException {
-    if (context.isWholeStream) {
-      return ByteString.readFrom(inStream);
-    }
-
-    int size = VarInt.decodeInt(inStream);
-    // ByteString reads to the end of the input stream, so give it a limited stream of exactly
-    // the right length. Also set its chunk size so that the ByteString will contain exactly
-    // one chunk.
-    return ByteString.readFrom(ByteStreams.limit(inStream, size), size);
-  }
-
-  @Override
-  protected long getEncodedElementByteSize(ByteString value, Context context) throws Exception {
-    int size = value.size();
-
-    if (context.isWholeStream) {
-      return size;
-    }
-    return VarInt.getLength(size) + size;
-  }
-
-  @Override
-  public void verifyDeterministic() {}
-
-  /**
-   * {@inheritDoc}
-   *
-   * <p>Returns true; the encoded output of two invocations of {@link ByteStringCoder} in the same
-   * {@link Coder.Context} will be identical if and only if the original {@link ByteString} objects
-   * are equal according to {@link Object#equals}.
-   */
-  @Override
-  public boolean consistentWithEquals() {
-    return true;
-  }
-
-  /**
-   * {@inheritDoc}
-   *
-   * <p>Returns true. {@link ByteString#size} returns the size of an array and a {@link VarInt}.
-   */
-  @Override
-  public boolean isRegisterByteSizeObserverCheap(ByteString value, Context context) {
-    return true;
-  }
-
-  @Override
-  public TypeDescriptor<ByteString> getEncodedTypeDescriptor() {
-    return TYPE_DESCRIPTOR;
-  }
-}

http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/CoderRegistry.java
----------------------------------------------------------------------
diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/CoderRegistry.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/CoderRegistry.java
index e0b2b3a..b8f70ab 100644
--- a/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/CoderRegistry.java
+++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/coders/CoderRegistry.java
@@ -27,7 +27,6 @@ import com.google.common.collect.Lists;
 import com.google.common.collect.Maps;
 import com.google.common.collect.Multimap;
 import com.google.common.collect.Sets;
-import com.google.protobuf.ByteString;
 import java.lang.reflect.ParameterizedType;
 import java.lang.reflect.Type;
 import java.lang.reflect.TypeVariable;
@@ -92,7 +91,6 @@ public class CoderRegistry implements CoderProvider {
     // Register the standard coders first so they are choosen as the default
     Multimap<Class<?>, CoderFactory> codersToRegister = HashMultimap.create();
     codersToRegister.put(Byte.class, CoderFactories.fromStaticMethods(ByteCoder.class));
-    codersToRegister.put(ByteString.class, CoderFactories.fromStaticMethods(ByteStringCoder.class));
     codersToRegister.put(Double.class, CoderFactories.fromStaticMethods(DoubleCoder.class));
     codersToRegister.put(Instant.class, CoderFactories.fromStaticMethods(InstantCoder.class));
     codersToRegister.put(Integer.class, CoderFactories.fromStaticMethods(VarIntCoder.class));

http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/ByteStringCoderTest.java
----------------------------------------------------------------------
diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/ByteStringCoderTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/ByteStringCoderTest.java
deleted file mode 100644
index ace1527..0000000
--- a/sdks/java/core/src/test/java/org/apache/beam/sdk/coders/ByteStringCoderTest.java
+++ /dev/null
@@ -1,128 +0,0 @@
-/*
- * 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.sdk.coders;
-
-import static org.hamcrest.Matchers.equalTo;
-import static org.junit.Assert.assertEquals;
-import static org.junit.Assert.assertThat;
-import static org.junit.Assert.assertTrue;
-
-import com.google.common.collect.ImmutableList;
-import com.google.protobuf.ByteString;
-import java.util.Arrays;
-import java.util.List;
-import org.apache.beam.sdk.coders.Coder.Context;
-import org.apache.beam.sdk.testing.CoderProperties;
-import org.apache.beam.sdk.util.CoderUtils;
-import org.apache.beam.sdk.values.TypeDescriptor;
-import org.junit.Rule;
-import org.junit.Test;
-import org.junit.rules.ExpectedException;
-import org.junit.runner.RunWith;
-import org.junit.runners.JUnit4;
-
-/**
- * Test case for {@link ByteStringCoder}.
- */
-@RunWith(JUnit4.class)
-public class ByteStringCoderTest {
-
-  private static final ByteStringCoder TEST_CODER = ByteStringCoder.of();
-
-  private static final List<String> TEST_STRING_VALUES = Arrays.asList(
-      "", "a", "13", "hello",
-      "a longer string with spaces and all that",
-      "a string with a \n newline",
-      "???????????????");
-  private static final ImmutableList<ByteString> TEST_VALUES;
-  static {
-    ImmutableList.Builder<ByteString> builder = ImmutableList.<ByteString>builder();
-    for (String s : TEST_STRING_VALUES) {
-      builder.add(ByteString.copyFrom(s.getBytes()));
-    }
-    TEST_VALUES = builder.build();
-  }
-
-  /**
-   * Generated data to check that the wire format has not changed. To regenerate, see
-   * {@link org.apache.beam.sdk.coders.PrintBase64Encodings}.
-   */
-  private static final List<String> TEST_ENCODINGS = Arrays.asList(
-      "",
-      "YQ",
-      "MTM",
-      "aGVsbG8",
-      "YSBsb25nZXIgc3RyaW5nIHdpdGggc3BhY2VzIGFuZCBhbGwgdGhhdA",
-      "YSBzdHJpbmcgd2l0aCBhIAogbmV3bGluZQ",
-      "Pz8_Pz8_Pz8_Pz8_Pz8_");
-
-  @Rule
-  public ExpectedException thrown = ExpectedException.none();
-
-  @Test
-  public void testDecodeEncodeEqualInAllContexts() throws Exception {
-    for (ByteString value : TEST_VALUES) {
-      CoderProperties.coderDecodeEncodeEqual(TEST_CODER, value);
-    }
-  }
-
-  @Test
-  public void testWireFormatEncode() throws Exception {
-    CoderProperties.coderEncodesBase64(TEST_CODER, TEST_VALUES, TEST_ENCODINGS);
-  }
-
-  @Test
-  public void testCoderDeterministic() throws Throwable {
-    TEST_CODER.verifyDeterministic();
-  }
-
-  @Test
-  public void testConsistentWithEquals() {
-    assertTrue(TEST_CODER.consistentWithEquals());
-  }
-
-  @Test
-  public void testEncodeNullThrowsCoderException() throws Exception {
-    thrown.expect(CoderException.class);
-    thrown.expectMessage("cannot encode a null ByteString");
-
-    CoderUtils.encodeToBase64(TEST_CODER, null);
-  }
-
-  @Test
-  public void testNestedCoding() throws Throwable {
-    Coder<List<ByteString>> listCoder = ListCoder.of(TEST_CODER);
-    CoderProperties.coderDecodeEncodeContentsEqual(listCoder, TEST_VALUES);
-    CoderProperties.coderDecodeEncodeContentsInSameOrder(listCoder, TEST_VALUES);
-  }
-
-  @Test
-  public void testEncodedElementByteSizeInAllContexts() throws Throwable {
-    for (Context context : CoderProperties.ALL_CONTEXTS) {
-      for (ByteString value : TEST_VALUES) {
-        byte[] encoded = CoderUtils.encodeToByteArray(TEST_CODER, value, context);
-        assertEquals(encoded.length, TEST_CODER.getEncodedElementByteSize(value, context));
-      }
-    }
-  }
-
-  @Test
-  public void testEncodedTypeDescriptor() throws Exception {
-    assertThat(TEST_CODER.getEncodedTypeDescriptor(), equalTo(TypeDescriptor.of(ByteString.class)));
-  }
-}

http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/extensions/protobuf/pom.xml
----------------------------------------------------------------------
diff --git a/sdks/java/extensions/protobuf/pom.xml b/sdks/java/extensions/protobuf/pom.xml
index 9300fc7..ad220a6 100644
--- a/sdks/java/extensions/protobuf/pom.xml
+++ b/sdks/java/extensions/protobuf/pom.xml
@@ -95,6 +95,12 @@
 
     <!-- build dependencies -->
     <dependency>
+      <groupId>com.google.auto.service</groupId>
+      <artifactId>auto-service</artifactId>
+      <optional>true</optional>
+    </dependency>
+
+    <dependency>
       <groupId>com.google.auto.value</groupId>
       <artifactId>auto-value</artifactId>
       <scope>provided</scope>

http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoder.java
----------------------------------------------------------------------
diff --git a/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoder.java b/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoder.java
new file mode 100644
index 0000000..3057050
--- /dev/null
+++ b/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoder.java
@@ -0,0 +1,118 @@
+/*
+ * 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.sdk.extensions.protobuf;
+
+import com.google.common.io.ByteStreams;
+import com.google.protobuf.ByteString;
+import java.io.IOException;
+import java.io.InputStream;
+import java.io.OutputStream;
+import org.apache.beam.sdk.coders.Coder;
+import org.apache.beam.sdk.coders.Coder.Context;
+import org.apache.beam.sdk.coders.CoderException;
+import org.apache.beam.sdk.coders.CustomCoder;
+import org.apache.beam.sdk.util.VarInt;
+import org.apache.beam.sdk.values.TypeDescriptor;
+
+/**
+ * A {@link Coder} for {@link ByteString} objects based on their encoded Protocol Buffer form.
+ *
+ * <p>When this code is used in a nested {@link Coder.Context}, the serialized {@link ByteString}
+ * objects are first delimited by their size.
+ */
+public class ByteStringCoder extends CustomCoder<ByteString> {
+
+  public static ByteStringCoder of() {
+    return INSTANCE;
+  }
+
+  /***************************/
+
+  private static final ByteStringCoder INSTANCE = new ByteStringCoder();
+  private static final TypeDescriptor<ByteString> TYPE_DESCRIPTOR =
+      new TypeDescriptor<ByteString>() {};
+
+  private ByteStringCoder() {}
+
+  @Override
+  public void encode(ByteString value, OutputStream outStream, Context context)
+      throws IOException, CoderException {
+    if (value == null) {
+      throw new CoderException("cannot encode a null ByteString");
+    }
+
+    if (!context.isWholeStream) {
+      // ByteString is not delimited, so write its size before its contents.
+      VarInt.encode(value.size(), outStream);
+    }
+    value.writeTo(outStream);
+  }
+
+  @Override
+  public ByteString decode(InputStream inStream, Context context) throws IOException {
+    if (context.isWholeStream) {
+      return ByteString.readFrom(inStream);
+    }
+
+    int size = VarInt.decodeInt(inStream);
+    // ByteString reads to the end of the input stream, so give it a limited stream of exactly
+    // the right length. Also set its chunk size so that the ByteString will contain exactly
+    // one chunk.
+    return ByteString.readFrom(ByteStreams.limit(inStream, size), size);
+  }
+
+  @Override
+  protected long getEncodedElementByteSize(ByteString value, Context context) throws Exception {
+    int size = value.size();
+
+    if (context.isWholeStream) {
+      return size;
+    }
+    return VarInt.getLength(size) + size;
+  }
+
+  @Override
+  public void verifyDeterministic() {}
+
+  /**
+   * {@inheritDoc}
+   *
+   * <p>Returns true; the encoded output of two invocations of {@link ByteStringCoder} in the same
+   * {@link Coder.Context} will be identical if and only if the original {@link ByteString} objects
+   * are equal according to {@link Object#equals}.
+   */
+  @Override
+  public boolean consistentWithEquals() {
+    return true;
+  }
+
+  /**
+   * {@inheritDoc}
+   *
+   * <p>Returns true. {@link ByteString#size} returns the size of an array and a {@link VarInt}.
+   */
+  @Override
+  public boolean isRegisterByteSizeObserverCheap(ByteString value, Context context) {
+    return true;
+  }
+
+  @Override
+  public TypeDescriptor<ByteString> getEncodedTypeDescriptor() {
+    return TYPE_DESCRIPTOR;
+  }
+}

http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ProtobufCoderRegistrar.java
----------------------------------------------------------------------
diff --git a/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ProtobufCoderRegistrar.java b/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ProtobufCoderRegistrar.java
new file mode 100644
index 0000000..fb39a14
--- /dev/null
+++ b/sdks/java/extensions/protobuf/src/main/java/org/apache/beam/sdk/extensions/protobuf/ProtobufCoderRegistrar.java
@@ -0,0 +1,39 @@
+/*
+ * 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.sdk.extensions.protobuf;
+
+import com.google.auto.service.AutoService;
+import com.google.common.collect.ImmutableMap;
+import com.google.protobuf.ByteString;
+import java.util.Map;
+import org.apache.beam.sdk.coders.CoderFactories;
+import org.apache.beam.sdk.coders.CoderFactory;
+import org.apache.beam.sdk.coders.CoderRegistrar;
+
+/**
+ * A {@link CoderRegistrar} for standard types used with Google Protobuf.
+ */
+@AutoService(CoderRegistrar.class)
+public class ProtobufCoderRegistrar implements CoderRegistrar {
+  @Override
+  public Map<Class<?>, CoderFactory> getCoderFactoriesToUseForClasses() {
+    return ImmutableMap.<Class<?>, CoderFactory>of(
+        ByteString.class, CoderFactories.forCoder(ByteStringCoder.of()));
+  }
+}
+

http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/extensions/protobuf/src/test/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoderTest.java
----------------------------------------------------------------------
diff --git a/sdks/java/extensions/protobuf/src/test/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoderTest.java b/sdks/java/extensions/protobuf/src/test/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoderTest.java
new file mode 100644
index 0000000..8fdb851
--- /dev/null
+++ b/sdks/java/extensions/protobuf/src/test/java/org/apache/beam/sdk/extensions/protobuf/ByteStringCoderTest.java
@@ -0,0 +1,131 @@
+/*
+ * 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.sdk.extensions.protobuf;
+
+import static org.hamcrest.Matchers.equalTo;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertThat;
+import static org.junit.Assert.assertTrue;
+
+import com.google.common.collect.ImmutableList;
+import com.google.protobuf.ByteString;
+import java.util.Arrays;
+import java.util.List;
+import org.apache.beam.sdk.coders.Coder;
+import org.apache.beam.sdk.coders.Coder.Context;
+import org.apache.beam.sdk.coders.CoderException;
+import org.apache.beam.sdk.coders.ListCoder;
+import org.apache.beam.sdk.testing.CoderProperties;
+import org.apache.beam.sdk.util.CoderUtils;
+import org.apache.beam.sdk.values.TypeDescriptor;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.ExpectedException;
+import org.junit.runner.RunWith;
+import org.junit.runners.JUnit4;
+
+/**
+ * Test case for {@link ByteStringCoder}.
+ */
+@RunWith(JUnit4.class)
+public class ByteStringCoderTest {
+
+  private static final ByteStringCoder TEST_CODER = ByteStringCoder.of();
+
+  private static final List<String> TEST_STRING_VALUES = Arrays.asList(
+      "", "a", "13", "hello",
+      "a longer string with spaces and all that",
+      "a string with a \n newline",
+      "???????????????");
+  private static final ImmutableList<ByteString> TEST_VALUES;
+  static {
+    ImmutableList.Builder<ByteString> builder = ImmutableList.<ByteString>builder();
+    for (String s : TEST_STRING_VALUES) {
+      builder.add(ByteString.copyFrom(s.getBytes()));
+    }
+    TEST_VALUES = builder.build();
+  }
+
+  /**
+   * Generated data to check that the wire format has not changed. To regenerate, see
+   * {@link org.apache.beam.sdk.coders.PrintBase64Encodings}.
+   */
+  private static final List<String> TEST_ENCODINGS = Arrays.asList(
+      "",
+      "YQ",
+      "MTM",
+      "aGVsbG8",
+      "YSBsb25nZXIgc3RyaW5nIHdpdGggc3BhY2VzIGFuZCBhbGwgdGhhdA",
+      "YSBzdHJpbmcgd2l0aCBhIAogbmV3bGluZQ",
+      "Pz8_Pz8_Pz8_Pz8_Pz8_");
+
+  @Rule
+  public ExpectedException thrown = ExpectedException.none();
+
+  @Test
+  public void testDecodeEncodeEqualInAllContexts() throws Exception {
+    for (ByteString value : TEST_VALUES) {
+      CoderProperties.coderDecodeEncodeEqual(TEST_CODER, value);
+    }
+  }
+
+  @Test
+  public void testWireFormatEncode() throws Exception {
+    CoderProperties.coderEncodesBase64(TEST_CODER, TEST_VALUES, TEST_ENCODINGS);
+  }
+
+  @Test
+  public void testCoderDeterministic() throws Throwable {
+    TEST_CODER.verifyDeterministic();
+  }
+
+  @Test
+  public void testConsistentWithEquals() {
+    assertTrue(TEST_CODER.consistentWithEquals());
+  }
+
+  @Test
+  public void testEncodeNullThrowsCoderException() throws Exception {
+    thrown.expect(CoderException.class);
+    thrown.expectMessage("cannot encode a null ByteString");
+
+    CoderUtils.encodeToBase64(TEST_CODER, null);
+  }
+
+  @Test
+  public void testNestedCoding() throws Throwable {
+    Coder<List<ByteString>> listCoder = ListCoder.of(TEST_CODER);
+    CoderProperties.coderDecodeEncodeContentsEqual(listCoder, TEST_VALUES);
+    CoderProperties.coderDecodeEncodeContentsInSameOrder(listCoder, TEST_VALUES);
+  }
+
+  @Test
+  public void testEncodedElementByteSizeInAllContexts() throws Throwable {
+    for (Context context : CoderProperties.ALL_CONTEXTS) {
+      for (ByteString value : TEST_VALUES) {
+        byte[] encoded = CoderUtils.encodeToByteArray(TEST_CODER, value, context);
+        assertEquals(encoded.length, TEST_CODER.getEncodedElementByteSize(value, context));
+      }
+    }
+  }
+
+  @Test
+  public void testEncodedTypeDescriptor() throws Exception {
+    assertThat(TEST_CODER.getEncodedTypeDescriptor(), equalTo(TypeDescriptor.of(ByteString.class)));
+  }
+}

http://git-wip-us.apache.org/repos/asf/beam/blob/03d47845/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigtable/BigtableIOTest.java
----------------------------------------------------------------------
diff --git a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigtable/BigtableIOTest.java b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigtable/BigtableIOTest.java
index 43957e3..ed1b147 100644
--- a/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigtable/BigtableIOTest.java
+++ b/sdks/java/io/google-cloud-platform/src/test/java/org/apache/beam/sdk/io/gcp/bigtable/BigtableIOTest.java
@@ -74,10 +74,10 @@ import java.util.SortedMap;
 import java.util.TreeMap;
 import javax.annotation.Nullable;
 import org.apache.beam.sdk.Pipeline.PipelineExecutionException;
-import org.apache.beam.sdk.coders.ByteStringCoder;
 import org.apache.beam.sdk.coders.Coder;
 import org.apache.beam.sdk.coders.IterableCoder;
 import org.apache.beam.sdk.coders.KvCoder;
+import org.apache.beam.sdk.extensions.protobuf.ByteStringCoder;
 import org.apache.beam.sdk.extensions.protobuf.ProtoCoder;
 import org.apache.beam.sdk.io.BoundedSource.BoundedReader;
 import org.apache.beam.sdk.io.gcp.bigtable.BigtableIO.BigtableSource;


[2/2] beam git commit: [BEAM-1871] Move ByteStringCoder to sdks/java/extensions/protobuf

Posted by lc...@apache.org.
[BEAM-1871] Move ByteStringCoder to sdks/java/extensions/protobuf

This closes #2682


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

Branch: refs/heads/master
Commit: b29d1c83b985ab654b217fdbf85ee36df096cbc2
Parents: 0094699 03d4784
Author: Luke Cwik <lc...@google.com>
Authored: Wed Apr 26 08:27:29 2017 -0700
Committer: Luke Cwik <lc...@google.com>
Committed: Wed Apr 26 08:27:29 2017 -0700

----------------------------------------------------------------------
 .../apache/beam/sdk/coders/ByteStringCoder.java | 114 ----------------
 .../apache/beam/sdk/coders/CoderRegistry.java   |   2 -
 .../beam/sdk/coders/ByteStringCoderTest.java    | 128 ------------------
 sdks/java/extensions/protobuf/pom.xml           |   6 +
 .../extensions/protobuf/ByteStringCoder.java    | 118 +++++++++++++++++
 .../protobuf/ProtobufCoderRegistrar.java        |  39 ++++++
 .../protobuf/ByteStringCoderTest.java           | 131 +++++++++++++++++++
 .../sdk/io/gcp/bigtable/BigtableIOTest.java     |   2 +-
 8 files changed, 295 insertions(+), 245 deletions(-)
----------------------------------------------------------------------