Connectors#

File System#

File Source#

FileEnumeratorProvider(...)

Factory for FileEnumerator which task is to discover all files to be read and to split them into a set of file source splits.

FileSplitAssignerProvider(j_file_split_assigner)

Factory for FileSplitAssigner which is responsible for deciding what split should be processed next by which node.

StreamFormat(j_stream_format)

A reader format that reads individual records from a stream.

BulkFormat(j_bulk_format)

The BulkFormat reads and decodes batches of records at a time.

FileSourceBuilder(j_file_source_builder)

The builder for the FileSource, to configure the various behaviors.

FileSource(j_file_source)

A unified data source that reads files - both in batch and in streaming mode.

File Sink#

BucketAssigner(j_bucket_assigner)

A BucketAssigner is used with a file sink to determine the bucket each incoming element should be put into.

RollingPolicy(j_rolling_policy)

The policy based on which a Bucket in the FileSink rolls its currently open part file and opens a new one.

DefaultRollingPolicy(j_rolling_policy)

The default implementation of the RollingPolicy.

OnCheckpointRollingPolicy(j_rolling_policy)

A RollingPolicy which rolls (ONLY) on every checkpoint.

OutputFileConfig(part_prefix, part_suffix)

Part file name configuration.

FileCompactStrategy(j_file_compact_strategy)

Strategy for compacting the files written in {@link FileSink} before committing.

FileCompactor(j_file_compactor)

The FileCompactor is responsible for compacting files into one file.

FileSink(j_file_sink[, transformer])

A unified sink that emits its input elements to FileSystem files within buckets.

StreamingFileSink(j_obj)

Sink that emits its input elements to FileSystem files within buckets.

Number Sequence#

NumberSequenceSource(start, end)

A data source that produces a sequence of numbers (longs).

Kafka#

Kakfa Producer and Consumer#

FlinkKafkaConsumer(topics, ...)

The Flink Kafka Consumer is a streaming data source that pulls a parallel data stream from Apache Kafka.

FlinkKafkaProducer(topic, ...[, ...])

Flink Sink to produce data into a Kafka topic.

Semantic(value)

Semantics that can be chosen.

Kafka Source and Sink#

KafkaSource(j_kafka_source)

The Source implementation of Kafka.

KafkaSourceBuilder()

The builder class for KafkaSource to make it easier for the users to construct a KafkaSource.

KafkaTopicPartition(topic, partition)

Corresponding to Java org.apache.kafka.common.TopicPartition class.

KafkaOffsetResetStrategy(value)

Corresponding to Java org.apache.kafka.client.consumer.OffsetResetStrategy class.

KafkaOffsetsInitializer(j_initializer)

An interface for users to specify the starting / stopping offset of a KafkaPartitionSplit.

KafkaSink(j_kafka_sink[, transformer])

Flink Sink to produce data into a Kafka topic.

KafkaSinkBuilder()

Builder to construct KafkaSink.

KafkaRecordSerializationSchema(...[, ...])

A serialization schema which defines how to convert the stream record to kafka producer record.

KafkaRecordSerializationSchemaBuilder()

Builder to construct KafkaRecordSerializationSchema.

KafkaTopicSelector()

Select topic for an incoming record

kinesis#

Kinesis Source#

KinesisShardAssigner(j_kinesis_shard_assigner)

Utility to map Kinesis shards to Flink subtask indices.

KinesisDeserializationSchema(...)

This is a deserialization schema specific for the Flink Kinesis Consumer.

WatermarkTracker(j_watermark_tracker)

The watermark tracker is responsible for aggregating watermarks across distributed operators.

FlinkKinesisConsumer(streams, deserializer, ...)

The Flink Kinesis Consumer is an exactly-once parallel streaming data source that subscribes to multiple AWS Kinesis streams within the same AWS service region, and can handle resharding of streams.

Kinesis Sink#

PartitionKeyGenerator(j_partition_key_generator)

This is a generator convert from an input element to the partition key, a string.

KinesisStreamsSink(j_kinesis_streams_sink)

A Kinesis Data Streams (KDS) Sink that performs async requests against a destination stream using the buffering protocol.

KinesisStreamsSinkBuilder()

Builder to construct KinesisStreamsSink.

KinesisFirehoseSink(j_kinesis_firehose_sink)

A Kinesis Data Firehose (KDF) Sink that performs async requests against a destination delivery stream using the buffering protocol.

KinesisFirehoseSinkBuilder()

Builder to construct KinesisFirehoseSink.

Pulsar#

Pulsar Source#

StartCursor(_j_start_cursor)

A factory class for users to specify the start position of a pulsar subscription.

StopCursor(_j_stop_cursor)

A factory class for users to specify the stop position of a pulsar subscription.

RangeGenerator(j_range_generator)

A generator for generating the TopicRange for given topic.

PulsarSource(j_pulsar_source)

The Source implementation of Pulsar.

PulsarSourceBuilder()

The builder class for PulsarSource to make it easier for the users to construct a PulsarSource.

Pulsar Sink#

TopicRoutingMode(value)

The routing policy for choosing the desired topic by the given message.

MessageDelayer(_j_message_delayer)

A delayer for Pulsar broker passing the sent message to the downstream consumer.

PulsarSink(j_pulsar_sink)

The Sink implementation of Pulsar.

PulsarSinkBuilder()

The builder class for PulsarSink to make it easier for the users to construct a PulsarSink.

Jdbc#

JdbcSink(j_jdbc_sink)

JdbcConnectionOptions(j_jdbc_connection_options)

JDBC connection options.

JdbcExecutionOptions(j_jdbc_execution_options)

JDBC sink batch options.

RMQ#

RMQConnectionConfig(j_rmq_connection_config)

Connection Configuration for RMQ.

RMQSource(connection_config, queue_name, ...)

RMQSink(connection_config, queue_name, ...)

Elasticsearch#

FlushBackoffType(value)

Used to control whether the sink should retry failed requests at all or with which kind back off strategy.

ElasticsearchEmitter(j_emitter)

Emitter which is used by sinks to prepare elements for sending them to Elasticsearch.

Elasticsearch6SinkBuilder()

Builder to construct an Elasticsearch 6 compatible ElasticsearchSink.

Elasticsearch7SinkBuilder()

Builder to construct an Elasticsearch 7 compatible ElasticsearchSink.

ElasticsearchSink(j_elasticsearch_sink)

Flink Sink to insert or update data in an Elasticsearch index.

Cassandra#

CassandraSink(j_cassandra_sink)

Sets the ClusterBuilder for this sink.

ConsistencyLevel(value)

The consistency level

MapperOptions()

This class is used to configure a Mapper after deployment.

ClusterBuilder(j_cluster_builder)

This class is used to configure a Cluster after deployment.

CassandraCommitter(j_checkpoint_committer)

CheckpointCommitter that saves information about completed checkpoints within a separate table in a cassandra database.

CassandraFailureHandler(...)

Handle a failed Throwable.