Class NetworkBufferPool
- java.lang.Object
-
- org.apache.flink.runtime.io.network.buffer.NetworkBufferPool
-
- All Implemented Interfaces:
MemorySegmentProvider
,AvailabilityProvider
,BufferPoolFactory
public class NetworkBufferPool extends Object implements BufferPoolFactory, MemorySegmentProvider, AvailabilityProvider
The NetworkBufferPool is a fixed size pool ofMemorySegment
instances for the network stack.The NetworkBufferPool creates
LocalBufferPool
s from which the individual tasks draw the buffers for the network data transfer. When new local buffer pools are created, the NetworkBufferPool dynamically redistributes the buffers between the pools.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AvailabilityProvider.AvailabilityHelper
-
-
Field Summary
Fields Modifier and Type Field Description static int
UNBOUNDED_POOL_SIZE
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Constructor Summary
Constructors Constructor Description NetworkBufferPool(int numberOfSegmentsToAllocate, int segmentSize)
NetworkBufferPool(int numberOfSegmentsToAllocate, int segmentSize, Duration requestSegmentsTimeout)
Allocates allMemorySegment
instances managed by this pool.
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description int
countBuffers()
BufferPool
createBufferPool(int numRequiredBuffers, int maxUsedBuffers)
Tries to create a buffer pool, which is guaranteed to provide at least the number of required buffers.BufferPool
createBufferPool(int numRequiredBuffers, int maxUsedBuffers, int numSubpartitions, int maxBuffersPerChannel, int maxOverdraftBuffersPerGate)
Tries to create a buffer pool with an owner, which is guaranteed to provide at least the number of required buffers.void
destroy()
void
destroyAllBufferPools()
Destroys all buffer pools that allocate their buffers from this buffer pool (created viacreateBufferPool(int, int)
).void
destroyBufferPool(BufferPool bufferPool)
Destroy callback for updating factory book keeping.CompletableFuture<?>
getAvailableFuture()
Returns a future that is completed when there are free segments in this pool.long
getAvailableMemory()
long
getEstimatedNumberOfRequestedMemorySegments()
long
getEstimatedRequestedMemory()
int
getEstimatedRequestedSegmentsUsage()
int
getNumberOfAvailableMemorySegments()
int
getNumberOfRegisteredBufferPools()
int
getNumberOfUsedMemorySegments()
long
getTotalMemory()
int
getTotalNumberOfMemorySegments()
long
getUsedMemory()
boolean
isDestroyed()
void
maybeLogUsageWarning()
void
recyclePooledMemorySegment(MemorySegment segment)
Corresponding torequestPooledMemorySegmentsBlocking(int)
andrequestPooledMemorySegment()
, this method is for pooled memory segments recycling.void
recycleUnpooledMemorySegments(Collection<MemorySegment> segments)
Corresponding torequestUnpooledMemorySegments(int)
, this method is for unpooled memory segments recycling.MemorySegment
requestPooledMemorySegment()
Different fromrequestUnpooledMemorySegments(int)
for unpooled segments allocation.List<MemorySegment>
requestPooledMemorySegmentsBlocking(int numberOfSegmentsToRequest)
List<MemorySegment>
requestUnpooledMemorySegments(int numberOfSegmentsToRequest)
Unpooled memory segments are requested directly fromNetworkBufferPool
, as opposed to pooled segments, that are requested throughBufferPool
that was created from thisNetworkBufferPool
(seecreateBufferPool(int, int)
).-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
isApproximatelyAvailable, isAvailable
-
-
-
-
Field Detail
-
UNBOUNDED_POOL_SIZE
public static final int UNBOUNDED_POOL_SIZE
- See Also:
- Constant Field Values
-
-
Constructor Detail
-
NetworkBufferPool
@VisibleForTesting public NetworkBufferPool(int numberOfSegmentsToAllocate, int segmentSize)
-
NetworkBufferPool
public NetworkBufferPool(int numberOfSegmentsToAllocate, int segmentSize, Duration requestSegmentsTimeout)
Allocates allMemorySegment
instances managed by this pool.
-
-
Method Detail
-
requestPooledMemorySegment
@Nullable public MemorySegment requestPooledMemorySegment()
Different fromrequestUnpooledMemorySegments(int)
for unpooled segments allocation. This method and the belowrequestPooledMemorySegmentsBlocking(int)
method are designed to be used fromLocalBufferPool
for pooled memory segments allocation. Note that these methods for pooled memory segments requesting and recycling are prohibited from acquiring the factoryLock to avoid deadlock.
-
requestPooledMemorySegmentsBlocking
public List<MemorySegment> requestPooledMemorySegmentsBlocking(int numberOfSegmentsToRequest) throws IOException
- Throws:
IOException
-
recyclePooledMemorySegment
public void recyclePooledMemorySegment(MemorySegment segment)
Corresponding torequestPooledMemorySegmentsBlocking(int)
andrequestPooledMemorySegment()
, this method is for pooled memory segments recycling.
-
requestUnpooledMemorySegments
public List<MemorySegment> requestUnpooledMemorySegments(int numberOfSegmentsToRequest) throws IOException
Unpooled memory segments are requested directly fromNetworkBufferPool
, as opposed to pooled segments, that are requested throughBufferPool
that was created from thisNetworkBufferPool
(seecreateBufferPool(int, int)
). They are used for example for exclusiveRemoteInputChannel
credits, that are permanently assigned to that channel, and never returned to anyBufferPool
. As opposed to pooled segments, when requested, unpooled segments needs to be accounted againstnumTotalRequiredBuffers
, which might require redistribution of the segments.- Specified by:
requestUnpooledMemorySegments
in interfaceMemorySegmentProvider
- Throws:
IOException
-
recycleUnpooledMemorySegments
public void recycleUnpooledMemorySegments(Collection<MemorySegment> segments)
Corresponding torequestUnpooledMemorySegments(int)
, this method is for unpooled memory segments recycling.- Specified by:
recycleUnpooledMemorySegments
in interfaceMemorySegmentProvider
-
destroy
public void destroy()
-
isDestroyed
public boolean isDestroyed()
-
getTotalNumberOfMemorySegments
public int getTotalNumberOfMemorySegments()
-
getTotalMemory
public long getTotalMemory()
-
getNumberOfAvailableMemorySegments
public int getNumberOfAvailableMemorySegments()
-
getAvailableMemory
public long getAvailableMemory()
-
getNumberOfUsedMemorySegments
public int getNumberOfUsedMemorySegments()
-
getUsedMemory
public long getUsedMemory()
-
getNumberOfRegisteredBufferPools
public int getNumberOfRegisteredBufferPools()
-
getEstimatedNumberOfRequestedMemorySegments
public long getEstimatedNumberOfRequestedMemorySegments()
-
getEstimatedRequestedMemory
public long getEstimatedRequestedMemory()
-
getEstimatedRequestedSegmentsUsage
public int getEstimatedRequestedSegmentsUsage()
-
maybeLogUsageWarning
public void maybeLogUsageWarning()
-
countBuffers
public int countBuffers()
-
getAvailableFuture
public CompletableFuture<?> getAvailableFuture()
Returns a future that is completed when there are free segments in this pool.- Specified by:
getAvailableFuture
in interfaceAvailabilityProvider
- Returns:
- a future that is completed if the respective provider is available.
-
createBufferPool
public BufferPool createBufferPool(int numRequiredBuffers, int maxUsedBuffers) throws IOException
Description copied from interface:BufferPoolFactory
Tries to create a buffer pool, which is guaranteed to provide at least the number of required buffers.The buffer pool is of dynamic size with at least numRequiredBuffers buffers.
- Specified by:
createBufferPool
in interfaceBufferPoolFactory
- Parameters:
numRequiredBuffers
- minimum number of network buffers in this poolmaxUsedBuffers
- maximum number of network buffers this pool offers- Throws:
IOException
-
createBufferPool
public BufferPool createBufferPool(int numRequiredBuffers, int maxUsedBuffers, int numSubpartitions, int maxBuffersPerChannel, int maxOverdraftBuffersPerGate) throws IOException
Description copied from interface:BufferPoolFactory
Tries to create a buffer pool with an owner, which is guaranteed to provide at least the number of required buffers.The buffer pool is of dynamic size with at least numRequiredBuffers buffers.
- Specified by:
createBufferPool
in interfaceBufferPoolFactory
- Parameters:
numRequiredBuffers
- minimum number of network buffers in this poolmaxUsedBuffers
- maximum number of network buffers this pool offersnumSubpartitions
- number of subpartitions in this poolmaxBuffersPerChannel
- maximum number of buffers to use for each channelmaxOverdraftBuffersPerGate
- maximum number of overdraft buffers to use for each gate- Throws:
IOException
-
destroyBufferPool
public void destroyBufferPool(BufferPool bufferPool)
Description copied from interface:BufferPoolFactory
Destroy callback for updating factory book keeping.- Specified by:
destroyBufferPool
in interfaceBufferPoolFactory
-
destroyAllBufferPools
public void destroyAllBufferPools()
Destroys all buffer pools that allocate their buffers from this buffer pool (created viacreateBufferPool(int, int)
).
-
-