Adds PubsubIO.writeStrings(), writeProtos(), writeAvros()

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

Branch: refs/heads/master
Commit: f0651145fea31854ab83fc064a3c7866251cc0a4
Parents: 079353d
Author: Eugene Kirpichov <kirpic...@google.com>
Authored: Thu Apr 20 17:54:03 2017 -0700
Committer: Eugene Kirpichov <kirpic...@google.com>
Committed: Sat Apr 29 13:15:48 2017 -0700

----------------------------------------------------------------------
 .../apache/beam/sdk/io/gcp/pubsub/PubsubIO.java | 28 ++++++++++++++++++--
 1 file changed, 26 insertions(+), 2 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/beam/blob/f0651145/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubIO.java
----------------------------------------------------------------------
diff --git 
a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubIO.java
 
b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubIO.java
index 9604864..3a7522e 100644
--- 
a/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubIO.java
+++ 
b/sdks/java/io/google-cloud-platform/src/main/java/org/apache/beam/sdk/io/gcp/pubsub/PubsubIO.java
@@ -474,8 +474,8 @@ public class PubsubIO {
   }
 
   /**
-   * Returns A {@link PTransform} that continuously reads binary encoded 
protos of the given type
-   * from a Google Cloud Pub/Sub stream.
+   * Returns A {@link PTransform} that continuously reads binary encoded 
protobuf messages of the
+   * given type from a Google Cloud Pub/Sub stream.
    */
   public static <T extends Message> Read<T> readProtos(Class<T> messageClass) {
     return PubsubIO.<T>read().withCoder(ProtoCoder.of(messageClass));
@@ -494,6 +494,30 @@ public class PubsubIO {
     return new AutoValue_PubsubIO_Write.Builder<T>().build();
   }
 
+  /**
+   * Returns A {@link PTransform} that writes UTF-8 encoded strings to a 
Google Cloud Pub/Sub
+   * stream.
+   */
+  public static Write<String> writeStrings() {
+    return PubsubIO.<String>write().withCoder(StringUtf8Coder.of());
+  }
+
+  /**
+   * Returns A {@link PTransform} that writes binary encoded protobuf messages 
of a given type
+   * to a Google Cloud Pub/Sub stream.
+   */
+  public static <T extends Message> Write<T> writeProtos(Class<T> 
messageClass) {
+    return PubsubIO.<T>write().withCoder(ProtoCoder.of(messageClass));
+  }
+
+  /**
+   * Returns A {@link PTransform} that writes binary encoded Avro messages of 
a given type
+   * to a Google Cloud Pub/Sub stream.
+   */
+  public static <T extends Message> Write<T> writeAvros(Class<T> clazz) {
+    return PubsubIO.<T>write().withCoder(AvroCoder.of(clazz));
+  }
+
   /** Implementation of {@link #read}. */
   @AutoValue
   public abstract static class Read<T> extends PTransform<PBegin, 
PCollection<T>> {

Reply via email to