Class TieredResultPartition
- java.lang.Object
-
- org.apache.flink.runtime.io.network.partition.ResultPartition
-
- org.apache.flink.runtime.io.network.partition.hybrid.tiered.shuffle.TieredResultPartition
-
- All Implemented Interfaces:
AutoCloseable
,AvailabilityProvider
,ResultPartitionWriter
public class TieredResultPartition extends ResultPartition
TieredResultPartition
appends records and events to the tiered storage, which supports the upstream dynamically switches storage tier for writing shuffle data, and the downstream will read data from the relevant tier.
-
-
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.ResultPartition
bufferCompressor, bufferPool, LOG, numBuffersOut, numBytesOut, numSubpartitions, partitionManager, partitionType, resultPartitionBytes
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Constructor Summary
Constructors Constructor Description TieredResultPartition(String owningTaskName, int partitionIndex, ResultPartitionID partitionId, ResultPartitionType partitionType, int numSubpartitions, int numTargetKeyGroups, ResultPartitionManager partitionManager, BufferCompressor bufferCompressor, SupplierWithException<BufferPool,IOException> bufferPoolFactory, TieredStorageProducerClient tieredStorageProducerClient, TieredStorageResourceRegistry tieredStorageResourceRegistry, TieredStorageNettyServiceImpl nettyService, List<TieredStorageMemorySpec> tieredStorageMemorySpecs, TieredStorageMemoryManager storageMemoryManager)
-
Method Summary
All Methods Instance Methods Concrete 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.protected ResultSubpartitionView
createSubpartitionView(int subpartitionId, BufferAvailabilityListener availabilityListener)
Returns a reader for the subpartition with the given index.void
emitRecord(ByteBuffer record, int consumerId)
Writes the given serialized record to the target subpartition.void
finish()
Finishes the result 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.CompletableFuture<?>
getAvailableFuture()
int
getNumberOfQueuedBuffers()
Returns the total number of queued buffers of all subpartitions.int
getNumberOfQueuedBuffers(int targetSubpartition)
Returns the number of queued buffers of the given target subpartition.long
getSizeOfQueuedBuffersUnsafe()
Returns the total size in bytes of queued buffers of all subpartitions.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.protected void
releaseInternal()
Releases all produced data including both those stored in memory and persisted on disk.void
setMetricGroup(TaskIOMetricGroup metrics)
Sets the metric group for theResultPartitionWriter
.protected void
setupInternal()
Do the subclass's own setup operation.-
Methods inherited from class org.apache.flink.runtime.io.network.partition.ResultPartition
canBeCompressed, checkInProduceState, createSubpartitionView, fail, getBufferPool, getFailureCause, getNumberOfSubpartitions, getNumTargetKeyGroups, getOwningTaskName, getPartitionId, getPartitionIndex, getPartitionManager, getPartitionType, getResultPartitionBytes, isFinished, isNumberOfPartitionConsumerUndefined, isNumberOfPartitionConsumerUndefined, isReleased, release, release, setMaxOverdraftBuffersPerGate, setup, toString
-
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
-
TieredResultPartition
public TieredResultPartition(String owningTaskName, int partitionIndex, ResultPartitionID partitionId, ResultPartitionType partitionType, int numSubpartitions, int numTargetKeyGroups, ResultPartitionManager partitionManager, @Nullable BufferCompressor bufferCompressor, SupplierWithException<BufferPool,IOException> bufferPoolFactory, TieredStorageProducerClient tieredStorageProducerClient, TieredStorageResourceRegistry tieredStorageResourceRegistry, TieredStorageNettyServiceImpl nettyService, List<TieredStorageMemorySpec> tieredStorageMemorySpecs, TieredStorageMemoryManager storageMemoryManager)
-
-
Method Detail
-
setupInternal
protected void setupInternal() throws IOException
Description copied from class:ResultPartition
Do the subclass's own setup operation.- Specified by:
setupInternal
in classResultPartition
- Throws:
IOException
-
setMetricGroup
public void setMetricGroup(TaskIOMetricGroup metrics)
Description copied from interface:ResultPartitionWriter
Sets the metric group for theResultPartitionWriter
.- Specified by:
setMetricGroup
in interfaceResultPartitionWriter
- Overrides:
setMetricGroup
in classResultPartition
-
emitRecord
public void emitRecord(ByteBuffer record, int consumerId) throws IOException
Description copied from interface:ResultPartitionWriter
Writes the given serialized record to the target subpartition.- Throws:
IOException
-
broadcastRecord
public void broadcastRecord(ByteBuffer record) throws IOException
Description copied from interface:ResultPartitionWriter
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
public void broadcastEvent(AbstractEvent event, boolean isPriorityEvent) throws IOException
Description copied from interface:ResultPartitionWriter
Writes the givenAbstractEvent
to all subpartitions.- Throws:
IOException
-
createSubpartitionView
protected ResultSubpartitionView createSubpartitionView(int subpartitionId, BufferAvailabilityListener availabilityListener) throws IOException
Description copied from class:ResultPartition
Returns a reader for the subpartition with the given index.Given that the function to merge outputs from multiple subpartition views is supported uniformly in
UnionResultSubpartitionView
, subclasses ofResultPartition
only needs to take care of creating subpartition view for a single subpartition.- Specified by:
createSubpartitionView
in classResultPartition
- Throws:
IOException
-
finish
public void finish() throws IOException
Description copied from class:ResultPartition
Finishes the result partition.After this operation, it is not possible to add further data to the result partition.
For BLOCKING results, this will trigger the deployment of consuming tasks.
- Specified by:
finish
in interfaceResultPartitionWriter
- Overrides:
finish
in classResultPartition
- 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 classResultPartition
-
releaseInternal
protected void releaseInternal()
Description copied from class:ResultPartition
Releases all produced data including both those stored in memory and persisted on disk.- Specified by:
releaseInternal
in classResultPartition
-
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
-
getAvailableFuture
public CompletableFuture<?> getAvailableFuture()
- Specified by:
getAvailableFuture
in interfaceAvailabilityProvider
- Overrides:
getAvailableFuture
in classResultPartition
- Returns:
- a future that is completed if the respective provider is available.
-
alignedBarrierTimeout
public void alignedBarrierTimeout(long checkpointId) throws IOException
Description copied from interface:ResultPartitionWriter
Timeout the aligned barrier to unaligned barrier.- Throws:
IOException
-
abortCheckpoint
public void abortCheckpoint(long checkpointId, CheckpointException cause)
Description copied from interface:ResultPartitionWriter
Abort the checkpoint.
-
flushAll
public void flushAll()
Description copied from interface:ResultPartitionWriter
Manually trigger the consumption of data from all subpartitions.
-
flush
public void flush(int subpartitionIndex)
Description copied from interface:ResultPartitionWriter
Manually trigger the consumption of data from the given subpartitions.
-
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
-
getNumberOfQueuedBuffers
public int getNumberOfQueuedBuffers()
Description copied from class:ResultPartition
Returns the total number of queued buffers of all subpartitions.- Specified by:
getNumberOfQueuedBuffers
in classResultPartition
-
getSizeOfQueuedBuffersUnsafe
public long getSizeOfQueuedBuffersUnsafe()
Description copied from class:ResultPartition
Returns the total size in bytes of queued buffers of all subpartitions.- Specified by:
getSizeOfQueuedBuffersUnsafe
in classResultPartition
-
getNumberOfQueuedBuffers
public int getNumberOfQueuedBuffers(int targetSubpartition)
Description copied from class:ResultPartition
Returns the number of queued buffers of the given target subpartition.- Specified by:
getNumberOfQueuedBuffers
in classResultPartition
-
-