@Internal public interface ChannelStateWriter extends Closeable
Modifier and Type | Interface and Description |
---|---|
static class |
ChannelStateWriter.ChannelStateWriteResult
Channel state write result.
|
static class |
ChannelStateWriter.NoOpChannelStateWriter
No-op implementation of
ChannelStateWriter . |
Modifier and Type | Field and Description |
---|---|
static ChannelStateWriter |
NO_OP |
static int |
SEQUENCE_NUMBER_RESTORED
Sequence number for the buffers that were saved during the previous execution attempt; then
restored; and now are to be saved again (as opposed to the buffers received from the upstream
or from the operator).
|
static int |
SEQUENCE_NUMBER_UNKNOWN
Signifies that buffer sequence number is unknown (e.g.
|
Modifier and Type | Method and Description |
---|---|
void |
abort(long checkpointId,
Throwable cause,
boolean cleanup)
Aborts the checkpoint and fails pending result for this checkpoint.
|
void |
addInputData(long checkpointId,
InputChannelInfo info,
int startSeqNum,
CloseableIterator<Buffer> data)
Add in-flight buffers from the
InputChannel . |
void |
addOutputData(long checkpointId,
ResultSubpartitionInfo info,
int startSeqNum,
Buffer... data)
Add in-flight buffers from the
ResultSubpartition . |
void |
addOutputDataFuture(long checkpointId,
ResultSubpartitionInfo info,
int startSeqNum,
CompletableFuture<List<Buffer>> data)
Add in-flight bufferFuture from the
ResultSubpartition . |
void |
finishInput(long checkpointId)
Finalize write of channel state data for the given checkpoint id.
|
void |
finishOutput(long checkpointId)
Finalize write of channel state data for the given checkpoint id.
|
ChannelStateWriter.ChannelStateWriteResult |
getAndRemoveWriteResult(long checkpointId)
Must be called after
start(long, CheckpointOptions) once. |
void |
start(long checkpointId,
CheckpointOptions checkpointOptions)
Initiate write of channel state for the given checkpoint id.
|
static final int SEQUENCE_NUMBER_RESTORED
static final int SEQUENCE_NUMBER_UNKNOWN
static final ChannelStateWriter NO_OP
void start(long checkpointId, CheckpointOptions checkpointOptions)
void addInputData(long checkpointId, InputChannelInfo info, int startSeqNum, CloseableIterator<Buffer> data)
InputChannel
. Must be
called after start(long, org.apache.flink.runtime.checkpoint.CheckpointOptions)
(long)} and before finishInput(long)
. Buffers are
recycled after they are written or exception occurs.startSeqNum
- sequence number of the 1st passed buffer. It is intended to use for
incremental snapshots. If no data is passed it is ignored.data
- zero or more data buffers ordered by their sequence numbersSEQUENCE_NUMBER_RESTORED
,
SEQUENCE_NUMBER_UNKNOWN
void addOutputData(long checkpointId, ResultSubpartitionInfo info, int startSeqNum, Buffer... data) throws IllegalArgumentException
ResultSubpartition
. Must be
called after start(long, org.apache.flink.runtime.checkpoint.CheckpointOptions)
and before finishOutput(long)
. Buffers are recycled
after they are written or exception occurs.startSeqNum
- sequence number of the 1st passed buffer. It is intended to use for
incremental snapshots. If no data is passed it is ignored.data
- zero or more data buffers ordered by their sequence numbersIllegalArgumentException
- if one or more passed buffers isn't
a buffer
SEQUENCE_NUMBER_RESTORED
,
SEQUENCE_NUMBER_UNKNOWN
void addOutputDataFuture(long checkpointId, ResultSubpartitionInfo info, int startSeqNum, CompletableFuture<List<Buffer>> data) throws IllegalArgumentException
ResultSubpartition
. Must be
called after start(long, org.apache.flink.runtime.checkpoint.CheckpointOptions)
and before finishOutput(long)
. Buffers are recycled
after they are written or exception occurs.
The method will be called when the unaligned checkpoint is enabled and received an aligned barrier.
IllegalArgumentException
void finishInput(long checkpointId)
start(long, CheckpointOptions)
and all of the input data of the given checkpoint added.
When both finishInput(long)
and finishOutput(long)
were called the results can be
(eventually) obtained using getAndRemoveWriteResult(long)
void finishOutput(long checkpointId)
start(long, CheckpointOptions)
and all of the output data of the given checkpoint added.
When both finishInput(long)
and finishOutput(long)
were called the results can be
(eventually) obtained using getAndRemoveWriteResult(long)
void abort(long checkpointId, Throwable cause, boolean cleanup)
cleanup
- true if getAndRemoveWriteResult(long)
is not supposed to be called
afterwards.ChannelStateWriter.ChannelStateWriteResult getAndRemoveWriteResult(long checkpointId) throws IllegalArgumentException
start(long, CheckpointOptions)
once.IllegalArgumentException
- if the passed checkpointId is not known.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.