public class RemoteInputChannel extends InputChannel
InputChannel.BufferAndAvailability
channelInfo, consumedSubpartitionIndexSet, currentBackoff, initialBackoff, inputGate, maxBackoff, numBuffersIn, numBytesIn, partitionId
Constructor and 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) |
Modifier and Type | Method and Description |
---|---|
void |
acknowledgeAllRecordsProcessed()
When received
EndOfData 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 or
Optional.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 returns
PartitionNotFoundException 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 a
CheckpointBarrier of
exactly-once mode, the upstream will be blocked and become unavailable. |
String |
toString() |
int |
unsynchronizedGetExclusiveBuffersUsed() |
int |
unsynchronizedGetFloatingBuffersAvailable() |
int |
unsynchronizedGetNumberOfQueuedBuffers() |
long |
unsynchronizedGetSizeOfQueuedBuffers() |
checkError, getChannelIndex, getChannelInfo, getConsumedSubpartitionIndexSet, getCurrentBackoff, getPartitionId, notifyChannelNonEmpty, notifyPriorityEvent, peekNextBufferSubpartitionId, setError
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)
@VisibleForTesting public void requestSubpartitions() throws IOException, InterruptedException
IOException
InterruptedException
protected boolean increaseBackoff()
PartitionNotFoundException
until the listener is timeout, so the backoff should add the
timeout milliseconds if it exists.increaseBackoff
in class InputChannel
true
, iff the operation was successful. Otherwise, false
.protected int peekNextBufferSubpartitionIdInternal() throws IOException
InputChannel
peekNextBufferSubpartitionIdInternal
in class InputChannel
IOException
public Optional<InputChannel.BufferAndAvailability> getNextBuffer() throws IOException
InputChannel
Optional.empty()
if there
is no data to return.getNextBuffer
in class InputChannel
IOException
public boolean isReleased()
@VisibleForTesting public int getNumberOfAvailableBuffers()
@VisibleForTesting public int getNumberOfRequiredBuffers()
@VisibleForTesting public int getSenderBacklog()
@VisibleForTesting public Buffer getNextReceivedBuffer()
public void notifyBufferAvailable(int numAvailableBuffers) throws IOException
notifyBufferAvailable
in class InputChannel
IOException
public void resumeConsumption() throws IOException
InputChannel
CheckpointBarrier
of
exactly-once mode, the upstream will be blocked and become unavailable. This method tries to
unblock the corresponding upstream and resume data consumption.resumeConsumption
in class InputChannel
IOException
public void acknowledgeAllRecordsProcessed() throws IOException
InputChannel
EndOfData
from one channel, it need to acknowledge after this event get
processed.acknowledgeAllRecordsProcessed
in class InputChannel
IOException
public int getUnannouncedCredit()
public int getAndResetUnannouncedCredit()
public int getNumberOfQueuedBuffers()
public int unsynchronizedGetNumberOfQueuedBuffers()
unsynchronizedGetNumberOfQueuedBuffers
in class InputChannel
public long unsynchronizedGetSizeOfQueuedBuffers()
unsynchronizedGetSizeOfQueuedBuffers
in class InputChannel
public int unsynchronizedGetExclusiveBuffersUsed()
public int unsynchronizedGetFloatingBuffersAvailable()
public InputChannelID getInputChannelId()
public int getInitialCredit()
public BufferProvider getBufferProvider() throws IOException
IOException
@Nullable public Buffer requestBuffer()
public void onSenderBacklog(int backlog) throws IOException
backlog
- The number of unsent buffers in the producer's sub partition.IOException
public void onBuffer(Buffer buffer, int sequenceNumber, int backlog, int subpartitionId) throws IOException
IOException
public void checkpointStarted(CheckpointBarrier barrier) throws CheckpointException
checkpointStarted
in class InputChannel
CheckpointException
public void checkpointStopped(long checkpointId)
InputChannel
checkpointStopped
in class InputChannel
public void convertToPriorityEvent(int sequenceNumber) throws IOException
convertToPriorityEvent
in class InputChannel
IOException
public void onEmptyBuffer(int sequenceNumber, int backlog) throws IOException
IOException
public void onFailedPartitionRequest()
public void onError(Throwable cause)
public void notifyRequiredSegmentId(int subpartitionId, int segmentId) throws IOException
InputChannel
notifyRequiredSegmentId
in class InputChannel
subpartitionId
- The id of the corresponding subpartition.segmentId
- The id of required segment.IOException
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.