public class MqttIO
extends java.lang.Object
MqttIO source returns an unbounded PCollection containing MQTT message payloads (as
byte[]).
To configure a MQTT source, you have to provide a MQTT connection configuration including
ClientId, a ServerURI, a Topic pattern, and optionally username
and password to connect to the MQTT broker. The following example illustrates various
options for configuring the source:
pipeline.apply(
MqttIO.read()
.withConnectionConfiguration(MqttIO.ConnectionConfiguration.create(
"tcp://host:11883",
"my_topic"))
The readWithMetadata method extends the functionality of the basic read method
by returning a PCollection of metadata that includes both the topic name and the payload.
The metadata is encapsulated in a container class MqttRecord that includes the topic name
and payload. This allows you to implement business logic that can differ depending on the topic
from which the message was received.
PCollection<MqttRecord> records = pipeline.apply(
MqttIO.readWithMetadata()
.withConnectionConfiguration(MqttIO.ConnectionConfiguration.create(
"tcp://host:11883",
"my_topic_pattern"))
By using the topic information, you can apply different processing logic depending on the source topic, enhancing the flexibility of message processing.
{@code
pipeline
.apply(MqttIO.readWithMetadata()
.withConnectionConfiguration(MqttIO.ConnectionConfiguration.create(
"tcp://host:1883", "my_topic_pattern")))
.apply(ParDo.of(new DoFn() { | Modifier and Type | Class and Description |
|---|---|
static class |
MqttIO.ConnectionConfiguration
A POJO describing a MQTT connection.
|
static class |
MqttIO.Read<T>
A
PTransform to read from a MQTT broker. |
static class |
MqttIO.Write<InputT>
A
PTransform to write and send a message to a MQTT server. |
| Modifier and Type | Method and Description |
|---|---|
static <InputT> MqttIO.Write<InputT> |
dynamicWrite() |
static MqttIO.Read<byte[]> |
read() |
static MqttIO.Read<MqttRecord> |
readWithMetadata() |
static MqttIO.Write<byte[]> |
write() |
public static MqttIO.Read<byte[]> read()
public static MqttIO.Read<MqttRecord> readWithMetadata()
public static MqttIO.Write<byte[]> write()
public static <InputT> MqttIO.Write<InputT> dynamicWrite()