Class RemoteInputChannel
- java.lang.Object
-
- org.apache.flink.runtime.io.network.partition.consumer.InputChannel
-
- org.apache.flink.runtime.io.network.partition.consumer.RemoteInputChannel
-
public class RemoteInputChannel extends InputChannel
An input channel, which requests a remote partition queue.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from class org.apache.flink.runtime.io.network.partition.consumer.InputChannel
InputChannel.BufferAndAvailability
-
-
Field Summary
-
Fields inherited from class org.apache.flink.runtime.io.network.partition.consumer.InputChannel
channelInfo, consumedSubpartitionIndexSet, currentBackoff, initialBackoff, inputGate, maxBackoff, numBuffersIn, numBytesIn, partitionId
-
-
Constructor Summary
Constructors Constructor Description RemoteInputChannel(SingleInputGate inputGate, int channelIndex, ResultPartitionID partitionId, ResultSubpartitionIndexSet consumedSubpartitionIndexSet, ConnectionID connectionId, ConnectionManager connectionManager, int initialBackOff, int maxBackoff, int partitionRequestListenerTimeout, int networkBuffersPerChannel, Counter numBytesIn, Counter numBuffersIn, ChannelStateWriter stateWriter)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
acknowledgeAllRecordsProcessed()
When receivedEndOfData
from one channel, it need to acknowledge after this event get processed.void
checkpointStarted(CheckpointBarrier barrier)
Spills all queued buffers on checkpoint start.void
checkpointStopped(long checkpointId)
Called by task thread on cancel/complete to clean-up temporary data.void
convertToPriorityEvent(int sequenceNumber)
int
getAndResetUnannouncedCredit()
Gets the unannounced credit and resets it to 0 atomically.BufferProvider
getBufferProvider()
int
getInitialCredit()
InputChannelID
getInputChannelId()
Optional<InputChannel.BufferAndAvailability>
getNextBuffer()
Returns the next buffer from the consumed subpartitions orOptional.empty()
if there is no data to return.Buffer
getNextReceivedBuffer()
int
getNumberOfAvailableBuffers()
int
getNumberOfQueuedBuffers()
Gets the current number of received buffers which have not been processed yet.int
getNumberOfRequiredBuffers()
int
getSenderBacklog()
int
getUnannouncedCredit()
Gets the currently unannounced credit.protected boolean
increaseBackoff()
The remote task manager creates partition request listener and returnsPartitionNotFoundException
until the listener is timeout, so the backoff should add the timeout milliseconds if it exists.boolean
isReleased()
void
notifyBufferAvailable(int numAvailableBuffers)
The unannounced credit is increased by the given amount and might notify increased credit to the producer.void
notifyRequiredSegmentId(int subpartitionId, int segmentId)
Notify the upstream the id of required segment that should be sent to netty connection.void
onBuffer(Buffer buffer, int sequenceNumber, int backlog, int subpartitionId)
Handles the input buffer.void
onEmptyBuffer(int sequenceNumber, int backlog)
void
onError(Throwable cause)
void
onFailedPartitionRequest()
void
onSenderBacklog(int backlog)
Receives the backlog from the producer's buffer response.protected int
peekNextBufferSubpartitionIdInternal()
Returns the index of the subpartition where the next buffer locates, or -1 if there is no buffer available and the subpartition to be consumed is not determined.Buffer
requestBuffer()
Requests buffer from input channel directly for receiving network data.void
requestSubpartitions()
Requests a remote subpartition.void
resumeConsumption()
After sending aCheckpointBarrier
of exactly-once mode, the upstream will be blocked and become unavailable.String
toString()
int
unsynchronizedGetExclusiveBuffersUsed()
int
unsynchronizedGetFloatingBuffersAvailable()
int
unsynchronizedGetNumberOfQueuedBuffers()
long
unsynchronizedGetSizeOfQueuedBuffers()
-
Methods inherited from class org.apache.flink.runtime.io.network.partition.consumer.InputChannel
checkError, getChannelIndex, getChannelInfo, getConsumedSubpartitionIndexSet, getCurrentBackoff, getPartitionId, notifyChannelNonEmpty, notifyPriorityEvent, peekNextBufferSubpartitionId, setError
-
-
-
-
Constructor Detail
-
RemoteInputChannel
public RemoteInputChannel(SingleInputGate inputGate, int channelIndex, ResultPartitionID partitionId, ResultSubpartitionIndexSet consumedSubpartitionIndexSet, ConnectionID connectionId, ConnectionManager connectionManager, int initialBackOff, int maxBackoff, int partitionRequestListenerTimeout, int networkBuffersPerChannel, Counter numBytesIn, Counter numBuffersIn, ChannelStateWriter stateWriter)
-
-
Method Detail
-
requestSubpartitions
@VisibleForTesting public void requestSubpartitions() throws IOException, InterruptedException
Requests a remote subpartition.- Throws:
IOException
InterruptedException
-
increaseBackoff
protected boolean increaseBackoff()
The remote task manager creates partition request listener and returnsPartitionNotFoundException
until the listener is timeout, so the backoff should add the timeout milliseconds if it exists.- Overrides:
increaseBackoff
in classInputChannel
- Returns:
true
, iff the operation was successful. Otherwise,false
.
-
peekNextBufferSubpartitionIdInternal
protected int peekNextBufferSubpartitionIdInternal() throws IOException
Description copied from class:InputChannel
Returns the index of the subpartition where the next buffer locates, or -1 if there is no buffer available and the subpartition to be consumed is not determined.- Specified by:
peekNextBufferSubpartitionIdInternal
in classInputChannel
- Throws:
IOException
-
getNextBuffer
public Optional<InputChannel.BufferAndAvailability> getNextBuffer() throws IOException
Description copied from class:InputChannel
Returns the next buffer from the consumed subpartitions orOptional.empty()
if there is no data to return.- Specified by:
getNextBuffer
in classInputChannel
- Throws:
IOException
-
isReleased
public boolean isReleased()
-
getNumberOfAvailableBuffers
@VisibleForTesting public int getNumberOfAvailableBuffers()
-
getNumberOfRequiredBuffers
@VisibleForTesting public int getNumberOfRequiredBuffers()
-
getSenderBacklog
@VisibleForTesting public int getSenderBacklog()
-
getNextReceivedBuffer
@VisibleForTesting public Buffer getNextReceivedBuffer()
-
notifyBufferAvailable
public void notifyBufferAvailable(int numAvailableBuffers) throws IOException
The unannounced credit is increased by the given amount and might notify increased credit to the producer.- Overrides:
notifyBufferAvailable
in classInputChannel
- Throws:
IOException
-
resumeConsumption
public void resumeConsumption() throws IOException
Description copied from class:InputChannel
After sending aCheckpointBarrier
of exactly-once mode, the upstream will be blocked and become unavailable. This method tries to unblock the corresponding upstream and resume data consumption.- Specified by:
resumeConsumption
in classInputChannel
- Throws:
IOException
-
acknowledgeAllRecordsProcessed
public void acknowledgeAllRecordsProcessed() throws IOException
Description copied from class:InputChannel
When receivedEndOfData
from one channel, it need to acknowledge after this event get processed.- Specified by:
acknowledgeAllRecordsProcessed
in classInputChannel
- Throws:
IOException
-
getUnannouncedCredit
public int getUnannouncedCredit()
Gets the currently unannounced credit.- Returns:
- Credit which was not announced to the sender yet.
-
getAndResetUnannouncedCredit
public int getAndResetUnannouncedCredit()
Gets the unannounced credit and resets it to 0 atomically.- Returns:
- Credit which was not announced to the sender yet.
-
getNumberOfQueuedBuffers
public int getNumberOfQueuedBuffers()
Gets the current number of received buffers which have not been processed yet.- Returns:
- Buffers queued for processing.
-
unsynchronizedGetNumberOfQueuedBuffers
public int unsynchronizedGetNumberOfQueuedBuffers()
- Overrides:
unsynchronizedGetNumberOfQueuedBuffers
in classInputChannel
-
unsynchronizedGetSizeOfQueuedBuffers
public long unsynchronizedGetSizeOfQueuedBuffers()
- Overrides:
unsynchronizedGetSizeOfQueuedBuffers
in classInputChannel
-
unsynchronizedGetExclusiveBuffersUsed
public int unsynchronizedGetExclusiveBuffersUsed()
-
unsynchronizedGetFloatingBuffersAvailable
public int unsynchronizedGetFloatingBuffersAvailable()
-
getInputChannelId
public InputChannelID getInputChannelId()
-
getInitialCredit
public int getInitialCredit()
-
getBufferProvider
public BufferProvider getBufferProvider() throws IOException
- Throws:
IOException
-
requestBuffer
@Nullable public Buffer requestBuffer()
Requests buffer from input channel directly for receiving network data. It should always return an available buffer in credit-based mode unless the channel has been released.- Returns:
- The available buffer.
-
onSenderBacklog
public void onSenderBacklog(int backlog) throws IOException
Receives the backlog from the producer's buffer response. If the number of available buffers is less than backlog + initialCredit, it will request floating buffers from the buffer manager, and then notify unannounced credits to the producer.- Parameters:
backlog
- The number of unsent buffers in the producer's sub partition.- Throws:
IOException
-
onBuffer
public void onBuffer(Buffer buffer, int sequenceNumber, int backlog, int subpartitionId) throws IOException
Handles the input buffer. This method is taking over the ownership of the buffer and is fully responsible for cleaning it up both on the happy path and in case of an error.- Throws:
IOException
-
checkpointStarted
public void checkpointStarted(CheckpointBarrier barrier) throws CheckpointException
Spills all queued buffers on checkpoint start. If barrier has already been received (and reordered), spill only the overtaken buffers.- Overrides:
checkpointStarted
in classInputChannel
- Throws:
CheckpointException
-
checkpointStopped
public void checkpointStopped(long checkpointId)
Description copied from class:InputChannel
Called by task thread on cancel/complete to clean-up temporary data.- Overrides:
checkpointStopped
in classInputChannel
-
convertToPriorityEvent
public void convertToPriorityEvent(int sequenceNumber) throws IOException
- Overrides:
convertToPriorityEvent
in classInputChannel
- Throws:
IOException
-
onEmptyBuffer
public void onEmptyBuffer(int sequenceNumber, int backlog) throws IOException
- Throws:
IOException
-
onFailedPartitionRequest
public void onFailedPartitionRequest()
-
onError
public void onError(Throwable cause)
-
notifyRequiredSegmentId
public void notifyRequiredSegmentId(int subpartitionId, int segmentId) throws IOException
Description copied from class:InputChannel
Notify the upstream the id of required segment that should be sent to netty connection.- Overrides:
notifyRequiredSegmentId
in classInputChannel
- Parameters:
subpartitionId
- The id of the corresponding subpartition.segmentId
- The id of required segment.- Throws:
IOException
-
-