public class PubsubIO
extends java.lang.Object
PTransform
s for Cloud Pub/Sub streams. These transforms create and consume
unbounded PCollections
.
In order to use local emulator for Pubsub you should use PubsubOptions#setPubsubRootUrl(String)
method to set host and port of your local emulator.
Permission requirements depend on the PipelineRunner
that is used to execute the Beam
pipeline. Please refer to the documentation of corresponding PipelineRunners
for more details.
// Read from a specific topic; a subscription will be created at pipeline start time.
PCollection<PubsubMessage> messages = PubsubIO.readMessages().fromTopic(topic);
// Read from a subscription.
PCollection<PubsubMessage> messages = PubsubIO.readMessages().fromSubscription(subscription);
// Read messages including attributes. All PubSub attributes will be included in the PubsubMessage.
PCollection<PubsubMessage> messages = PubsubIO.readMessagesWithAttributes().fromTopic(topic);
// Examples of reading different types from PubSub.
PCollection<String> strings = PubsubIO.readStrings().fromTopic(topic);
PCollection<MyProto> protos = PubsubIO.readProtos(MyProto.class).fromTopic(topic);
PCollection<MyType> avros = PubsubIO.readAvros(MyType.class).fromTopic(topic);
PubsubIO.Write.to(String)
method can be used. For example:
avros.apply(PubsubIO.writeAvros(MyType.class).to(topic));
protos.apply(PubsubIO.writeProtos(MyProto.class).to(topic));
strings.apply(PubsubIO.writeStrings().to(topic));
Dynamic topic destinations can be accomplished by specifying a function to extract the topic from
the record using the PubsubIO.Write.to(SerializableFunction)
method. For example:
avros.apply(PubsubIO.writeAvros(MyType.class).
to((ValueInSingleWindow<Event> quote) -> {
String country = quote.getCountry();
return "projects/myproject/topics/events_" + country;
});
Dynamic topics can also be specified by writing PubsubMessage
objects containing the
topic and writing using the writeMessagesDynamic()
method. For example:
events.apply(MapElements.into(new TypeDescriptor<PubsubMessage>() {})
.via(e -> new PubsubMessage(
e.toByteString(), Collections.emptyMap()).withTopic(e.getCountry())))
.apply(PubsubIO.writeMessagesDynamic());
PubsubIO.Read.withTimestampAttribute(java.lang.String)
. See the Javadoc for that method for the timestamp format.Modifier and Type | Class and Description |
---|---|
static class |
PubsubIO.PubsubSubscription
Class representing a Cloud Pub/Sub Subscription.
|
static class |
PubsubIO.PubsubTopic
Class representing a Cloud Pub/Sub Topic.
|
static class |
PubsubIO.Read<T>
Implementation of read methods.
|
static class |
PubsubIO.Write<T>
Implementation of write methods.
|
Modifier and Type | Method and Description |
---|---|
static PubsubIO.Read<GenericRecord> |
readAvroGenericRecords(Schema avroSchema)
Returns a
PTransform that continuously reads binary encoded Avro messages into the Avro
GenericRecord type. |
static <T> PubsubIO.Read<T> |
readAvros(java.lang.Class<T> clazz)
Returns A
PTransform that continuously reads binary encoded Avro messages of the given
type from a Google Cloud Pub/Sub stream. |
static <T> PubsubIO.Read<T> |
readAvrosWithBeamSchema(java.lang.Class<T> clazz)
Returns a
PTransform that continuously reads binary encoded Avro messages of the
specific type. |
static PubsubIO.Read<PubsubMessage> |
readMessages()
Returns A
PTransform that continuously reads from a Google Cloud Pub/Sub stream. |
static PubsubIO.Read<PubsubMessage> |
readMessagesWithAttributes()
Returns A
PTransform that continuously reads from a Google Cloud Pub/Sub stream. |
static PubsubIO.Read<PubsubMessage> |
readMessagesWithAttributesAndMessageId()
Returns A
PTransform that continuously reads from a Google Cloud Pub/Sub stream. |
static PubsubIO.Read<PubsubMessage> |
readMessagesWithAttributesAndMessageIdAndOrderingKey()
Returns A
PTransform that continuously reads from a Google Cloud Pub/Sub stream. |
static <T> PubsubIO.Read<T> |
readMessagesWithCoderAndParseFn(Coder<T> coder,
SimpleFunction<PubsubMessage,T> parseFn)
Returns A
PTransform that continuously reads from a Google Cloud Pub/Sub stream,
mapping each PubsubMessage into type T using the supplied parse function and coder. |
static PubsubIO.Read<PubsubMessage> |
readMessagesWithMessageId()
Returns A
PTransform that continuously reads from a Google Cloud Pub/Sub stream. |
static PubsubIO.Read<com.google.protobuf.DynamicMessage> |
readProtoDynamicMessages(com.google.protobuf.Descriptors.Descriptor descriptor)
Similar to
readProtoDynamicMessages(ProtoDomain, String) but for when the
Descriptors.Descriptor is already known. |
static PubsubIO.Read<com.google.protobuf.DynamicMessage> |
readProtoDynamicMessages(ProtoDomain domain,
java.lang.String fullMessageName)
Returns a
PTransform that continuously reads binary encoded protobuf messages for the
type specified by fullMessageName . |
static <T extends com.google.protobuf.Message> |
readProtos(java.lang.Class<T> messageClass)
Returns A
PTransform that continuously reads binary encoded protobuf messages of the
given type from a Google Cloud Pub/Sub stream. |
static PubsubIO.Read<java.lang.String> |
readStrings()
Returns A
PTransform that continuously reads UTF-8 encoded strings from a Google Cloud
Pub/Sub stream. |
static <T> PubsubIO.Write<T> |
writeAvros(java.lang.Class<T> clazz)
Returns A
PTransform that writes binary encoded Avro messages of a given type to a
Google Cloud Pub/Sub stream. |
static <T> PubsubIO.Write<T> |
writeAvros(java.lang.Class<T> clazz,
SerializableFunction<ValueInSingleWindow<T>,java.util.Map<java.lang.String,java.lang.String>> attributeFn)
Returns A
PTransform that writes binary encoded Avro messages of a given type to a
Google Cloud Pub/Sub stream. |
static PubsubIO.Write<PubsubMessage> |
writeMessages()
Returns A
PTransform that writes to a Google Cloud Pub/Sub stream. |
static PubsubIO.Write<PubsubMessage> |
writeMessagesDynamic()
Enables dynamic destination topics.
|
static <T extends com.google.protobuf.Message> |
writeProtos(java.lang.Class<T> messageClass)
Returns A
PTransform that writes binary encoded protobuf messages of a given type to a
Google Cloud Pub/Sub stream. |
static <T extends com.google.protobuf.Message> |
writeProtos(java.lang.Class<T> messageClass,
SerializableFunction<ValueInSingleWindow<T>,java.util.Map<java.lang.String,java.lang.String>> attributeFn)
Returns A
PTransform that writes binary encoded protobuf messages of a given type to a
Google Cloud Pub/Sub stream. |
static PubsubIO.Write<java.lang.String> |
writeStrings()
Returns A
PTransform that writes UTF-8 encoded strings to a Google Cloud Pub/Sub
stream. |
public static PubsubIO.Read<PubsubMessage> readMessages()
PTransform
that continuously reads from a Google Cloud Pub/Sub stream. The
messages will only contain a payload
, but no attributes
.public static PubsubIO.Read<PubsubMessage> readMessagesWithMessageId()
PTransform
that continuously reads from a Google Cloud Pub/Sub stream. The
messages will only contain a payload
with the messageId
from PubSub, but no attributes
.public static PubsubIO.Read<PubsubMessage> readMessagesWithAttributes()
PTransform
that continuously reads from a Google Cloud Pub/Sub stream. The
messages will contain both a payload
and attributes
.public static PubsubIO.Read<PubsubMessage> readMessagesWithAttributesAndMessageId()
PTransform
that continuously reads from a Google Cloud Pub/Sub stream. The
messages will contain both a payload
and attributes
, along with the messageId
from PubSub.public static PubsubIO.Read<PubsubMessage> readMessagesWithAttributesAndMessageIdAndOrderingKey()
PTransform
that continuously reads from a Google Cloud Pub/Sub stream. The
messages will contain a payload
, attributes
, along with the messageId
and {PubsubMessage#getOrderingKey() orderingKey} from PubSub.public static PubsubIO.Read<java.lang.String> readStrings()
PTransform
that continuously reads UTF-8 encoded strings from a Google Cloud
Pub/Sub stream.public static <T extends com.google.protobuf.Message> PubsubIO.Read<T> readProtos(java.lang.Class<T> messageClass)
PTransform
that continuously reads binary encoded protobuf messages of the
given type from a Google Cloud Pub/Sub stream.public static PubsubIO.Read<com.google.protobuf.DynamicMessage> readProtoDynamicMessages(ProtoDomain domain, java.lang.String fullMessageName)
PTransform
that continuously reads binary encoded protobuf messages for the
type specified by fullMessageName
.
This is primarily here for cases where the message type cannot be known at compile time. If
it can be known, prefer readProtos(Class)
, as DynamicMessage
tends to
perform worse than concrete types.
Beam will infer a schema for the DynamicMessage
schema. Note that some proto schema
features are not supported by all sinks.
domain
- The ProtoDomain
that contains the target message and its dependencies.fullMessageName
- The full name of the message for lookup in domain
.public static PubsubIO.Read<com.google.protobuf.DynamicMessage> readProtoDynamicMessages(com.google.protobuf.Descriptors.Descriptor descriptor)
readProtoDynamicMessages(ProtoDomain, String)
but for when the
Descriptors.Descriptor
is already known.public static <T> PubsubIO.Read<T> readAvros(java.lang.Class<T> clazz)
PTransform
that continuously reads binary encoded Avro messages of the given
type from a Google Cloud Pub/Sub stream.public static <T> PubsubIO.Read<T> readMessagesWithCoderAndParseFn(Coder<T> coder, SimpleFunction<PubsubMessage,T> parseFn)
PTransform
that continuously reads from a Google Cloud Pub/Sub stream,
mapping each PubsubMessage
into type T using the supplied parse function and coder.public static PubsubIO.Read<GenericRecord> readAvroGenericRecords(Schema avroSchema)
PTransform
that continuously reads binary encoded Avro messages into the Avro
GenericRecord
type.
Beam will infer a schema for the Avro schema. This allows the output to be used by SQL and by the schema-transform library.
public static <T> PubsubIO.Read<T> readAvrosWithBeamSchema(java.lang.Class<T> clazz)
PTransform
that continuously reads binary encoded Avro messages of the
specific type.
Beam will infer a schema for the Avro schema. This allows the output to be used by SQL and by the schema-transform library.
public static PubsubIO.Write<PubsubMessage> writeMessages()
PTransform
that writes to a Google Cloud Pub/Sub stream.public static PubsubIO.Write<PubsubMessage> writeMessagesDynamic()
PubsubMessage
elements are each expected to
contain a destination topic, which can be set using PubsubMessage.withTopic(java.lang.String)
. If PubsubIO.Write.to(java.lang.String)
is called, that will be used instead to generate the topic and the value returned by
PubsubMessage.getTopic()
will be ignored.public static PubsubIO.Write<java.lang.String> writeStrings()
PTransform
that writes UTF-8 encoded strings to a Google Cloud Pub/Sub
stream.public static <T extends com.google.protobuf.Message> PubsubIO.Write<T> writeProtos(java.lang.Class<T> messageClass)
PTransform
that writes binary encoded protobuf messages of a given type to a
Google Cloud Pub/Sub stream.public static <T extends com.google.protobuf.Message> PubsubIO.Write<T> writeProtos(java.lang.Class<T> messageClass, SerializableFunction<ValueInSingleWindow<T>,java.util.Map<java.lang.String,java.lang.String>> attributeFn)
PTransform
that writes binary encoded protobuf messages of a given type to a
Google Cloud Pub/Sub stream.public static <T> PubsubIO.Write<T> writeAvros(java.lang.Class<T> clazz)
PTransform
that writes binary encoded Avro messages of a given type to a
Google Cloud Pub/Sub stream.public static <T> PubsubIO.Write<T> writeAvros(java.lang.Class<T> clazz, SerializableFunction<ValueInSingleWindow<T>,java.util.Map<java.lang.String,java.lang.String>> attributeFn)
PTransform
that writes binary encoded Avro messages of a given type to a
Google Cloud Pub/Sub stream.