@PublicEvolving public abstract class AsyncSinkWriter<InputT,RequestEntryT extends Serializable> extends Object implements StatefulSink.StatefulSinkWriter<InputT,BufferedRequestState<RequestEntryT>>
At least once semantics is supported through prepareCommit
as outstanding requests are
flushed or completed prior to checkpointing.
Designed to be returned at createWriter
time by an AsyncSinkBase
.
There are configuration options to customize the buffer size etc.
SinkWriter.Context
Constructor and Description |
---|
AsyncSinkWriter(ElementConverter<InputT,RequestEntryT> elementConverter,
Sink.InitContext context,
AsyncSinkWriterConfiguration configuration,
Collection<BufferedRequestState<RequestEntryT>> states) |
AsyncSinkWriter(ElementConverter<InputT,RequestEntryT> elementConverter,
Sink.InitContext context,
int maxBatchSize,
int maxInFlightRequests,
int maxBufferedRequests,
long maxBatchSizeInBytes,
long maxTimeInBufferMS,
long maxRecordSizeInBytes)
Deprecated.
|
AsyncSinkWriter(ElementConverter<InputT,RequestEntryT> elementConverter,
Sink.InitContext context,
int maxBatchSize,
int maxInFlightRequests,
int maxBufferedRequests,
long maxBatchSizeInBytes,
long maxTimeInBufferMS,
long maxRecordSizeInBytes,
Collection<BufferedRequestState<RequestEntryT>> states)
Deprecated.
|
Modifier and Type | Method and Description |
---|---|
void |
close() |
void |
flush(boolean flush)
In flight requests will be retried if the sink is still healthy.
|
protected java.util.function.Consumer<Exception> |
getFatalExceptionCons() |
protected abstract long |
getSizeInBytes(RequestEntryT requestEntry)
This method allows the getting of the size of a
RequestEntryT in bytes. |
List<BufferedRequestState<RequestEntryT>> |
snapshotState(long checkpointId)
All in-flight requests that are relevant for the snapshot have been completed, but there may
still be request entries in the internal buffers that are yet to be sent to the endpoint.
|
protected abstract void |
submitRequestEntries(List<RequestEntryT> requestEntries,
java.util.function.Consumer<List<RequestEntryT>> requestToRetry)
This method specifies how to persist buffered request entries into the destination.
|
void |
write(InputT element,
SinkWriter.Context context)
Adds an element to the writer.
|
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
writeWatermark
@Deprecated public AsyncSinkWriter(ElementConverter<InputT,RequestEntryT> elementConverter, Sink.InitContext context, int maxBatchSize, int maxInFlightRequests, int maxBufferedRequests, long maxBatchSizeInBytes, long maxTimeInBufferMS, long maxRecordSizeInBytes)
AsyncSinkWriterConfiguration
.@Deprecated public AsyncSinkWriter(ElementConverter<InputT,RequestEntryT> elementConverter, Sink.InitContext context, int maxBatchSize, int maxInFlightRequests, int maxBufferedRequests, long maxBatchSizeInBytes, long maxTimeInBufferMS, long maxRecordSizeInBytes, Collection<BufferedRequestState<RequestEntryT>> states)
AsyncSinkWriterConfiguration
.public AsyncSinkWriter(ElementConverter<InputT,RequestEntryT> elementConverter, Sink.InitContext context, AsyncSinkWriterConfiguration configuration, Collection<BufferedRequestState<RequestEntryT>> states)
protected abstract void submitRequestEntries(List<RequestEntryT> requestEntries, java.util.function.Consumer<List<RequestEntryT>> requestToRetry)
The method is invoked with a set of request entries according to the buffering hints (and
the valid limits of the destination). The logic then needs to create and execute the request
asynchronously against the destination (ideally by batching together multiple request entries
to increase efficiency). The logic also needs to identify individual request entries that
were not persisted successfully and resubmit them using the requestToRetry
callback.
From a threading perspective, the mailbox thread will call this method and initiate the
asynchronous request to persist the requestEntries
. NOTE: The client must support
asynchronous requests and the method called to persist the records must asynchronously
execute and return a future with the results of that request. A thread from the destination
client thread pool should complete the request and submit the failed entries that should be
retried. The requestToRetry
will then trigger the mailbox thread to requeue the
unsuccessful elements.
An example implementation of this method is included:
{@code
requestEntries
- a set of request entries that should be sent to the destinationrequestToRetry
- the accept
method should be called on this Consumer once the
processing of the requestEntries
are complete. Any entries that encountered
difficulties in persisting should be re-queued through requestToRetry
by
including that element in the collection of RequestEntryT
s passed to the accept
method. All other elements are assumed to have been successfully persisted.protected abstract long getSizeInBytes(RequestEntryT requestEntry)
RequestEntryT
in bytes. The size in
this case is measured as the total bytes that is written to the destination as a result of
persisting this particular RequestEntryT
rather than the serialized length (which may
be the same).requestEntry
- the requestEntry for which we want to know the sizepublic void write(InputT element, SinkWriter.Context context) throws IOException, InterruptedException
SinkWriter
write
in interface SinkWriter<InputT>
element
- The input recordcontext
- The additional information about the input recordIOException
- if fail to add an element.InterruptedException
public void flush(boolean flush) throws InterruptedException
To this end, all in-flight requests need to completed before proceeding with the commit.
flush
in interface SinkWriter<InputT>
InterruptedException
public List<BufferedRequestState<RequestEntryT>> snapshotState(long checkpointId)
snapshotState
in interface StatefulSink.StatefulSinkWriter<InputT,BufferedRequestState<RequestEntryT extends Serializable>>
public void close()
close
in interface AutoCloseable
protected java.util.function.Consumer<Exception> getFatalExceptionCons()
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.