Class FileSink<IN>
- Type Parameters:
IN- Type of the elements in the input of the sink that are also the elements to be written to its output
- All Implemented Interfaces:
Serializable,org.apache.flink.api.common.SupportsConcurrentExecutionAttempts,org.apache.flink.api.connector.sink2.Sink<IN>,org.apache.flink.api.connector.sink2.SupportsCommitter<FileSinkCommittable>,org.apache.flink.api.connector.sink2.SupportsWriterState<IN,,FileWriterBucketState> org.apache.flink.api.connector.sink2.SupportsWriterState.WithCompatibleState,org.apache.flink.streaming.api.connector.sink2.SupportsPreCommitTopology<FileSinkCommittable,FileSinkCommittable>
FileSystem files within buckets. This
sink achieves exactly-once semantics for both BATCH and STREAMING.
When creating the sink a basePath must be specified. The base directory contains one
directory for every bucket. The bucket directories themselves contain several part files, with at
least one for each parallel subtask of the sink which is writing data to that bucket. These part
files contain the actual output data.
The sink uses a BucketAssigner to determine in which bucket directory each element
should be written to inside the base directory. The BucketAssigner can, for example, roll
on every checkpoint or use time or a property of the element to determine the bucket directory.
The default BucketAssigner is a DateTimeBucketAssigner which will create one new
bucket every hour. You can specify a custom BucketAssigner using the
setBucketAssigner(bucketAssigner) method, after calling forRowFormat(Path, Encoder) or forBulkFormat(Path, BulkWriter.Factory).
The names of the part files could be defined using OutputFileConfig. This
configuration contains a part prefix and a part suffix that will be used with a random uid
assigned to each subtask of the sink and a rolling counter to determine the file names. For
example with a prefix "prefix" and a suffix ".ext", a file named
"prefix-81fc4980-a6af-41c8-9937-9939408a734b-17.ext" contains the data from subtask with uid
81fc4980-a6af-41c8-9937-9939408a734b of the sink and is the 17th part-file
created by that subtask.
Part files roll based on the user-specified RollingPolicy. By default, a DefaultRollingPolicy is used for row-encoded sink output; a OnCheckpointRollingPolicy is
used for bulk-encoded sink output.
In some scenarios, the open buckets are required to change based on time. In these cases, the
user can specify a bucketCheckInterval (by default 1m) and the sink will check
periodically and roll the part file if the specified rolling policy says so.
Part files can be in one of three states: in-progress, pending or
finished. The reason for this is how the sink works to provide exactly-once semantics and
fault-tolerance. The part file that is currently being written to is in-progress. Once a
part file is closed for writing it becomes pending. When a checkpoint is successful (for
STREAMING) or at the end of the job (for BATCH) the currently pending files will
be moved to finished.
For STREAMING in order to guarantee exactly-once semantics in case of a failure, the
sink should roll back to the state it had when that last successful checkpoint occurred. To this
end, when restoring, the restored files in pending state are transferred into the
finished state while any in-progress files are rolled back, so that they do not contain
data that arrived after the checkpoint from which we restore.
FileSink also support compacting small files to accelerate the access speed of the resulted
files. Compaction could be enabled via enableCompact. Once enabled, the compaction could
only be disabled via calling disableCompact explicitly, otherwise there might be data
loss.
- See Also:
-
Nested Class Summary
Nested ClassesModifier and TypeClassDescriptionstatic classFileSink.BulkFormatBuilder<IN,T extends FileSink.BulkFormatBuilder<IN, T>> A builder for configuring the sink for bulk-encoding formats, e.g.static final classBuilder for the vanillaFileSinkusing a bulk format.static final classBuilder for the vanillaFileSinkusing a row format.static classFileSink.RowFormatBuilder<IN,T extends FileSink.RowFormatBuilder<IN, T>> A builder for configuring the sink for row-wise encoding formats.Nested classes/interfaces inherited from interface org.apache.flink.api.connector.sink2.SupportsWriterState
org.apache.flink.api.connector.sink2.SupportsWriterState.WithCompatibleState -
Method Summary
Modifier and TypeMethodDescriptionorg.apache.flink.streaming.api.datastream.DataStream<org.apache.flink.streaming.api.connector.sink2.CommittableMessage<FileSinkCommittable>>addPreCommitTopology(org.apache.flink.streaming.api.datastream.DataStream<org.apache.flink.streaming.api.connector.sink2.CommittableMessage<FileSinkCommittable>> committableStream) org.apache.flink.api.connector.sink2.Committer<FileSinkCommittable>createCommitter(org.apache.flink.api.connector.sink2.CommitterInitContext context) createWriter(org.apache.flink.api.connector.sink2.WriterInitContext context) static <IN> FileSink.DefaultBulkFormatBuilder<IN>forBulkFormat(org.apache.flink.core.fs.Path basePath, org.apache.flink.api.common.serialization.BulkWriter.Factory<IN> bulkWriterFactory) static <IN> FileSink.DefaultRowFormatBuilder<IN>forRowFormat(org.apache.flink.core.fs.Path basePath, org.apache.flink.api.common.serialization.Encoder<IN> encoder) org.apache.flink.core.io.SimpleVersionedSerializer<FileSinkCommittable>org.apache.flink.core.io.SimpleVersionedSerializer<FileSinkCommittable>org.apache.flink.core.io.SimpleVersionedSerializer<FileWriterBucketState>restoreWriter(org.apache.flink.api.connector.sink2.WriterInitContext context, Collection<FileWriterBucketState> recoveredState)
-
Method Details
-
createWriter
public FileWriter<IN> createWriter(org.apache.flink.api.connector.sink2.WriterInitContext context) throws IOException - Specified by:
createWriterin interfaceorg.apache.flink.api.connector.sink2.Sink<IN>- Throws:
IOException
-
restoreWriter
public FileWriter<IN> restoreWriter(org.apache.flink.api.connector.sink2.WriterInitContext context, Collection<FileWriterBucketState> recoveredState) throws IOException - Specified by:
restoreWriterin interfaceorg.apache.flink.api.connector.sink2.SupportsWriterState<IN,FileWriterBucketState> - Throws:
IOException
-
getWriterStateSerializer
public org.apache.flink.core.io.SimpleVersionedSerializer<FileWriterBucketState> getWriterStateSerializer()- Specified by:
getWriterStateSerializerin interfaceorg.apache.flink.api.connector.sink2.SupportsWriterState<IN,FileWriterBucketState>
-
createCommitter
public org.apache.flink.api.connector.sink2.Committer<FileSinkCommittable> createCommitter(org.apache.flink.api.connector.sink2.CommitterInitContext context) throws IOException - Specified by:
createCommitterin interfaceorg.apache.flink.api.connector.sink2.SupportsCommitter<IN>- Throws:
IOException
-
getCommittableSerializer
public org.apache.flink.core.io.SimpleVersionedSerializer<FileSinkCommittable> getCommittableSerializer()- Specified by:
getCommittableSerializerin interfaceorg.apache.flink.api.connector.sink2.SupportsCommitter<IN>
-
getWriteResultSerializer
public org.apache.flink.core.io.SimpleVersionedSerializer<FileSinkCommittable> getWriteResultSerializer()- Specified by:
getWriteResultSerializerin interfaceorg.apache.flink.streaming.api.connector.sink2.SupportsPreCommitTopology<FileSinkCommittable,FileSinkCommittable>
-
getCompatibleWriterStateNames
- Specified by:
getCompatibleWriterStateNamesin interfaceorg.apache.flink.api.connector.sink2.SupportsWriterState.WithCompatibleState
-
forRowFormat
public static <IN> FileSink.DefaultRowFormatBuilder<IN> forRowFormat(org.apache.flink.core.fs.Path basePath, org.apache.flink.api.common.serialization.Encoder<IN> encoder) -
forBulkFormat
public static <IN> FileSink.DefaultBulkFormatBuilder<IN> forBulkFormat(org.apache.flink.core.fs.Path basePath, org.apache.flink.api.common.serialization.BulkWriter.Factory<IN> bulkWriterFactory) -
addPreCommitTopology
public org.apache.flink.streaming.api.datastream.DataStream<org.apache.flink.streaming.api.connector.sink2.CommittableMessage<FileSinkCommittable>> addPreCommitTopology(org.apache.flink.streaming.api.datastream.DataStream<org.apache.flink.streaming.api.connector.sink2.CommittableMessage<FileSinkCommittable>> committableStream) - Specified by:
addPreCommitTopologyin interfaceorg.apache.flink.streaming.api.connector.sink2.SupportsPreCommitTopology<FileSinkCommittable,FileSinkCommittable>
-