title | nav-title | nav-parent_id | nav-pos |
---|---|---|---|
RabbitMQ Connector |
RabbitMQ |
connectors |
6 |
Flink's RabbitMQ connector defines a Maven dependency on the "RabbitMQ AMQP Java Client", is triple-licensed under the Mozilla Public License 1.1 ("MPL"), the GNU General Public License version 2 ("GPL") and the Apache License version 2 ("ASL").
Flink itself neither reuses source code from the "RabbitMQ AMQP Java Client" nor packages binaries from the "RabbitMQ AMQP Java Client".
Users that create and publish derivative work based on Flink's RabbitMQ connector (thereby re-distributing the "RabbitMQ AMQP Java Client") must be aware that this may be subject to conditions declared in the Mozilla Public License 1.1 ("MPL"), the GNU General Public License version 2 ("GPL") and the Apache License version 2 ("ASL").
This connector provides access to data streams from RabbitMQ. To use this connector, add the following dependency to your project:
{% highlight xml %} org.apache.flink flink-connector-rabbitmq{{ site.scala_version_suffix }} {{site.version }} {% endhighlight %}
Note that the streaming connectors are currently not part of the binary distribution. See linking with them for cluster execution here.
Follow the instructions from the RabbitMQ download page. After the installation the server automatically starts, and the application connecting to RabbitMQ can be launched.
This connector provides a RMQSource
class to consume messages from a RabbitMQ
queue. This source provides three different levels of guarantees, depending
on how it is configured with Flink:
- Exactly-once: In order to achieve exactly-once guarantees with the RabbitMQ source, the following is required -
- Enable checkpointing: With checkpointing enabled, messages are only acknowledged (hence, removed from the RabbitMQ queue) when checkpoints are completed.
- Use correlation ids: Correlation ids are a RabbitMQ application feature. You have to set it in the message properties when injecting messages into RabbitMQ. The correlation id is used by the source to deduplicate any messages that have been reprocessed when restoring from a checkpoint.
- Non-parallel source: The source must be non-parallel (parallelism set to 1) in order to achieve exactly-once. This limitation is mainly due to RabbitMQ's approach to dispatching messages from a single queue to multiple consumers.
-
At-least-once: When checkpointing is enabled, but correlation ids are not used or the source is parallel, the source only provides at-least-once guarantees.
-
No guarantee: If checkpointing isn't enabled, the source does not have any strong delivery guarantees. Under this setting, instead of collaborating with Flink's checkpointing, messages will be automatically acknowledged once the source receives and processes them.
Below is a code example for setting up an exactly-once RabbitMQ source. Inline comments explain which parts of the configuration can be ignored for more relaxed guarantees.
final RMQConnectionConfig connectionConfig = new RMQConnectionConfig.Builder() .setHost("localhost") .setPort(5000) ... .build();
final DataStream stream = env .addSource(new RMQSource( connectionConfig, // config for the RabbitMQ connection "queueName", // name of the RabbitMQ queue to consume true, // use correlation ids; can be false if only at-least-once is required new SimpleStringSchema())) // deserialization schema to turn messages into Java objects .setParallelism(1); // non-parallel source is only required for exactly-once {% endhighlight %}
val connectionConfig = new RMQConnectionConfig.Builder() .setHost("localhost") .setPort(5000) ... .build
val stream = env .addSource(new RMQSource[String]( connectionConfig, // config for the RabbitMQ connection "queueName", // name of the RabbitMQ queue to consume true, // use correlation ids; can be false if only at-least-once is required new SimpleStringSchema)) // deserialization schema to turn messages into Java objects .setParallelism(1) // non-parallel source is only required for exactly-once {% endhighlight %}
This connector provides a RMQSink
class for sending messages to a RabbitMQ
queue. Below is a code example for setting up a RabbitMQ sink.
final RMQConnectionConfig connectionConfig = new RMQConnectionConfig.Builder() .setHost("localhost") .setPort(5000) ... .build();
stream.addSink(new RMQSink( connectionConfig, // config for the RabbitMQ connection "queueName", // name of the RabbitMQ queue to send messages to new SimpleStringSchema())); // serialization schema to turn Java objects to messages {% endhighlight %}
val connectionConfig = new RMQConnectionConfig.Builder() .setHost("localhost") .setPort(5000) ... .build
stream.addSink(new RMQSink[String]( connectionConfig, // config for the RabbitMQ connection "queueName", // name of the RabbitMQ queue to send messages to new SimpleStringSchema)) // serialization schema to turn Java objects to messages {% endhighlight %}
More about RabbitMQ can be found here.
{% top %}