Interface ResultPartitionWriter
-
- All Superinterfaces:
AutoCloseable
,AvailabilityProvider
- All Known Implementing Classes:
BoundedBlockingResultPartition
,BufferWritingResultPartition
,PipelinedResultPartition
,ResultPartition
,SortMergeResultPartition
,TieredResultPartition
public interface ResultPartitionWriter extends AutoCloseable, AvailabilityProvider
A record-oriented runtime result writer API for producing results.If
close()
is called beforefail(Throwable)
orfinish()
, it abruptly triggers failure and cancellation of production. In this casefail(Throwable)
still needs to be called afterwards to fully release all resources associated the partition and propagate failure cause to the consumer if possible.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AvailabilityProvider.AvailabilityHelper
-
-
Field Summary
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Method Summary
All Methods Instance Methods Abstract Methods Modifier and Type Method Description void
abortCheckpoint(long checkpointId, CheckpointException cause)
Abort the checkpoint.void
alignedBarrierTimeout(long checkpointId)
Timeout the aligned barrier to unaligned barrier.void
broadcastEvent(AbstractEvent event, boolean isPriorityEvent)
Writes the givenAbstractEvent
to all subpartitions.void
broadcastRecord(ByteBuffer record)
Writes the given serialized record to all subpartitions.void
close()
Closes the partition writer which releases the allocated resource, for example the buffer pool.ResultSubpartitionView
createSubpartitionView(ResultSubpartitionIndexSet indexSet, BufferAvailabilityListener availabilityListener)
Returns a reader for the subpartition with the given index range.void
emitRecord(ByteBuffer record, int targetSubpartition)
Writes the given serialized record to the target subpartition.void
fail(Throwable throwable)
Fail the production of the partition.void
finish()
Successfully finish the production of the partition.void
flush(int subpartitionIndex)
Manually trigger the consumption of data from the given subpartitions.void
flushAll()
Manually trigger the consumption of data from all subpartitions.CompletableFuture<Void>
getAllDataProcessedFuture()
Gets the future indicating whether all the records has been processed by the downstream tasks.int
getNumberOfSubpartitions()
int
getNumTargetKeyGroups()
ResultPartitionID
getPartitionId()
boolean
isFinished()
boolean
isReleased()
void
notifyEndOfData(StopMode mode)
Notifies the downstream tasks that thisResultPartitionWriter
have emitted all the user records.void
release(Throwable cause)
Releases the partition writer which releases the produced data and no reader can consume the partition any more.void
setMaxOverdraftBuffersPerGate(int maxOverdraftBuffersPerGate)
Sets the max overdraft buffer size of per gate.void
setMetricGroup(TaskIOMetricGroup metrics)
Sets the metric group for theResultPartitionWriter
.void
setup()
Setup partition, potentially heavy-weight, blocking operation comparing to just creation.-
Methods inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
getAvailableFuture, isApproximatelyAvailable, isAvailable
-
-
-
-
Method Detail
-
setup
void setup() throws IOException
Setup partition, potentially heavy-weight, blocking operation comparing to just creation.- Throws:
IOException
-
getPartitionId
ResultPartitionID getPartitionId()
-
getNumberOfSubpartitions
int getNumberOfSubpartitions()
-
getNumTargetKeyGroups
int getNumTargetKeyGroups()
-
setMaxOverdraftBuffersPerGate
void setMaxOverdraftBuffersPerGate(int maxOverdraftBuffersPerGate)
Sets the max overdraft buffer size of per gate.
-
emitRecord
void emitRecord(ByteBuffer record, int targetSubpartition) throws IOException
Writes the given serialized record to the target subpartition.- Throws:
IOException
-
broadcastRecord
void broadcastRecord(ByteBuffer record) throws IOException
Writes the given serialized record to all subpartitions. One can also achieve the same effect by emitting the same record to all subpartitions one by one, however, this method can have better performance for the underlying implementation can do some optimizations, for example coping the given serialized record only once to a shared channel which can be consumed by all subpartitions.- Throws:
IOException
-
broadcastEvent
void broadcastEvent(AbstractEvent event, boolean isPriorityEvent) throws IOException
Writes the givenAbstractEvent
to all subpartitions.- Throws:
IOException
-
alignedBarrierTimeout
void alignedBarrierTimeout(long checkpointId) throws IOException
Timeout the aligned barrier to unaligned barrier.- Throws:
IOException
-
abortCheckpoint
void abortCheckpoint(long checkpointId, CheckpointException cause)
Abort the checkpoint.
-
notifyEndOfData
void notifyEndOfData(StopMode mode) throws IOException
Notifies the downstream tasks that thisResultPartitionWriter
have emitted all the user records.- Parameters:
mode
- tells if we should flush all records or not (it is false in case of stop-with-savepoint (--no-drain))- Throws:
IOException
-
getAllDataProcessedFuture
CompletableFuture<Void> getAllDataProcessedFuture()
Gets the future indicating whether all the records has been processed by the downstream tasks.
-
setMetricGroup
void setMetricGroup(TaskIOMetricGroup metrics)
Sets the metric group for theResultPartitionWriter
.
-
createSubpartitionView
ResultSubpartitionView createSubpartitionView(ResultSubpartitionIndexSet indexSet, BufferAvailabilityListener availabilityListener) throws IOException
Returns a reader for the subpartition with the given index range.- Throws:
IOException
-
flushAll
void flushAll()
Manually trigger the consumption of data from all subpartitions.
-
flush
void flush(int subpartitionIndex)
Manually trigger the consumption of data from the given subpartitions.
-
fail
void fail(@Nullable Throwable throwable)
Fail the production of the partition.This method propagates non-
null
failure causes to consumers on a best-effort basis. This call also leads to the release of all resources associated with the partition. Closing of the partition is still needed afterwards if it has not been done before.- Parameters:
throwable
- failure cause
-
finish
void finish() throws IOException
Successfully finish the production of the partition.Closing of partition is still needed afterwards.
- Throws:
IOException
-
isFinished
boolean isFinished()
-
release
void release(Throwable cause)
Releases the partition writer which releases the produced data and no reader can consume the partition any more.
-
isReleased
boolean isReleased()
-
close
void close() throws Exception
Closes the partition writer which releases the allocated resource, for example the buffer pool.- Specified by:
close
in interfaceAutoCloseable
- Throws:
Exception
-
-