Class PipelinedResultPartition
- java.lang.Object
-
- org.apache.flink.runtime.io.network.partition.ResultPartition
-
- org.apache.flink.runtime.io.network.partition.BufferWritingResultPartition
-
- org.apache.flink.runtime.io.network.partition.PipelinedResultPartition
-
- All Implemented Interfaces:
AutoCloseable
,AvailabilityProvider
,ResultPartitionWriter
,ChannelStateHolder
,CheckpointedResultPartition
public class PipelinedResultPartition extends BufferWritingResultPartition implements CheckpointedResultPartition, ChannelStateHolder
A result output of a task, pipelined (streamed) to the receivers.This result partition implementation is used both in batch and streaming. For streaming, it supports low latency transfers (ensure data is sent within x milliseconds) or unconstrained while for batch it transfers only once a buffer is full. Additionally, for streaming use this typically limits the length of the buffer backlog to not have too much data in flight, while for batch we do not constrain this.
Specifics of the PipelinedResultPartition
The PipelinedResultPartition cannot reconnect once a consumer disconnects (finished or errored). Once all consumers have disconnected (released the subpartition, notified via the call
onConsumedSubpartition(int)
) then the partition as a whole is disposed and all buffers are freed.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AvailabilityProvider.AvailabilityHelper
-
-
Field Summary
-
Fields inherited from class org.apache.flink.runtime.io.network.partition.BufferWritingResultPartition
subpartitions
-
Fields inherited from class org.apache.flink.runtime.io.network.partition.ResultPartition
bufferCompressor, bufferPool, LOG, numBuffersOut, numBytesOut, numSubpartitions, partitionId, partitionManager, partitionType, resultPartitionBytes
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Constructor Summary
Constructors Constructor Description PipelinedResultPartition(String owningTaskName, int partitionIndex, ResultPartitionID partitionId, ResultPartitionType partitionType, ResultSubpartition[] subpartitions, int numTargetKeyGroups, ResultPartitionManager partitionManager, BufferCompressor bufferCompressor, SupplierWithException<BufferPool,IOException> bufferPoolFactory)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
addRecovered(int subpartitionIndex, BufferConsumer bufferConsumer)
void
close()
Closes the partition writer which releases the allocated resource, for example the buffer pool.void
finishReadRecoveredState(boolean notifyAndBlockOnCompletion)
void
flush(int targetSubpartition)
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.ResultSubpartitionInfo
getCheckpointedSubpartitionInfo(int subpartitionIndex)
Gets the checkpointed subpartition info with the given subpartitionIndex.void
notifyEndOfData(StopMode mode)
Notifies the downstream tasks that thisResultPartitionWriter
have emitted all the user records.void
onSubpartitionAllDataProcessed(int subpartition)
The subpartition notifies that the corresponding downstream task have processed all the user records.BufferBuilder
requestBufferBuilderBlocking()
void
setChannelStateWriter(ChannelStateWriter channelStateWriter)
Injects theChannelStateWriter
.String
toString()
-
Methods inherited from class org.apache.flink.runtime.io.network.partition.BufferWritingResultPartition
abortCheckpoint, addToSubpartition, alignedBarrierTimeout, broadcastEvent, broadcastRecord, createSubpartitionView, emitRecord, finish, flushAllSubpartitions, flushSubpartition, getAllPartitions, getHardBackPressuredTimeMsPerSecond, getNumberOfQueuedBuffers, getNumberOfQueuedBuffers, getSizeOfQueuedBuffersUnsafe, releaseInternal, setMetricGroup, setupInternal
-
Methods inherited from class org.apache.flink.runtime.io.network.partition.ResultPartition
canBeCompressed, checkInProduceState, createSubpartitionView, fail, getAvailableFuture, getBufferPool, getFailureCause, getNumberOfSubpartitions, getNumTargetKeyGroups, getOwningTaskName, getPartitionId, getPartitionIndex, getPartitionManager, getPartitionType, getResultPartitionBytes, isFinished, isNumberOfPartitionConsumerUndefined, isNumberOfPartitionConsumerUndefined, isReleased, release, release, setMaxOverdraftBuffersPerGate, setup
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
-
Methods inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
isApproximatelyAvailable, isAvailable
-
-
-
-
Constructor Detail
-
PipelinedResultPartition
public PipelinedResultPartition(String owningTaskName, int partitionIndex, ResultPartitionID partitionId, ResultPartitionType partitionType, ResultSubpartition[] subpartitions, int numTargetKeyGroups, ResultPartitionManager partitionManager, @Nullable BufferCompressor bufferCompressor, SupplierWithException<BufferPool,IOException> bufferPoolFactory)
-
-
Method Detail
-
setChannelStateWriter
public void setChannelStateWriter(ChannelStateWriter channelStateWriter)
Description copied from interface:ChannelStateHolder
Injects theChannelStateWriter
. Must only be called once.- Specified by:
setChannelStateWriter
in interfaceChannelStateHolder
-
getCheckpointedSubpartitionInfo
public ResultSubpartitionInfo getCheckpointedSubpartitionInfo(int subpartitionIndex)
Description copied from interface:CheckpointedResultPartition
Gets the checkpointed subpartition info with the given subpartitionIndex.- Specified by:
getCheckpointedSubpartitionInfo
in interfaceCheckpointedResultPartition
-
flushAll
public void flushAll()
Description copied from interface:ResultPartitionWriter
Manually trigger the consumption of data from all subpartitions.- Specified by:
flushAll
in interfaceResultPartitionWriter
-
flush
public void flush(int targetSubpartition)
Description copied from interface:ResultPartitionWriter
Manually trigger the consumption of data from the given subpartitions.- Specified by:
flush
in interfaceResultPartitionWriter
-
notifyEndOfData
public void notifyEndOfData(StopMode mode) throws IOException
Description copied from interface:ResultPartitionWriter
Notifies the downstream tasks that thisResultPartitionWriter
have emitted all the user records.- Specified by:
notifyEndOfData
in interfaceResultPartitionWriter
- Overrides:
notifyEndOfData
in classResultPartition
- 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
public CompletableFuture<Void> getAllDataProcessedFuture()
Description copied from interface:ResultPartitionWriter
Gets the future indicating whether all the records has been processed by the downstream tasks.- Specified by:
getAllDataProcessedFuture
in interfaceResultPartitionWriter
- Overrides:
getAllDataProcessedFuture
in classResultPartition
-
onSubpartitionAllDataProcessed
public void onSubpartitionAllDataProcessed(int subpartition)
Description copied from class:ResultPartition
The subpartition notifies that the corresponding downstream task have processed all the user records.- Overrides:
onSubpartitionAllDataProcessed
in classResultPartition
- Parameters:
subpartition
- The index of the subpartition sending the notification.- See Also:
EndOfData
-
toString
public String toString()
- Overrides:
toString
in classResultPartition
-
finishReadRecoveredState
public void finishReadRecoveredState(boolean notifyAndBlockOnCompletion) throws IOException
- Specified by:
finishReadRecoveredState
in interfaceCheckpointedResultPartition
- Throws:
IOException
-
requestBufferBuilderBlocking
public BufferBuilder requestBufferBuilderBlocking() throws IOException, RuntimeException, InterruptedException
- Specified by:
requestBufferBuilderBlocking
in interfaceCheckpointedResultPartition
- Throws:
IOException
RuntimeException
InterruptedException
-
addRecovered
public void addRecovered(int subpartitionIndex, BufferConsumer bufferConsumer) throws IOException
- Specified by:
addRecovered
in interfaceCheckpointedResultPartition
- Throws:
IOException
-
close
public void close()
Description copied from interface:ResultPartitionWriter
Closes the partition writer which releases the allocated resource, for example the buffer pool.- Specified by:
close
in interfaceAutoCloseable
- Specified by:
close
in interfaceResultPartitionWriter
- Overrides:
close
in classBufferWritingResultPartition
-
-