public abstract class InputChannel extends Object
ResultSubpartitionView
.
For each channel, the consumption life cycle is as follows:
Modifier and Type | Class and Description |
---|---|
static class |
InputChannel.BufferAndAvailability
A combination of a
Buffer and a flag indicating availability of further buffers, and
the backlog length indicating how many non-event buffers are available in the subpartition. |
Modifier and Type | Field and Description |
---|---|
protected InputChannelInfo |
channelInfo
The info of the input channel to identify it globally within a task.
|
protected int |
consumedSubpartitionIndex
The index of the subpartition consumed by this channel.
|
protected int |
initialBackoff
The initial backoff (in ms).
|
protected SingleInputGate |
inputGate |
protected int |
maxBackoff
The maximum backoff (in ms).
|
protected Counter |
numBuffersIn |
protected Counter |
numBytesIn |
protected ResultPartitionID |
partitionId
The parent partition of the subpartition consumed by this channel.
|
Modifier | Constructor and Description |
---|---|
protected |
InputChannel(SingleInputGate inputGate,
int channelIndex,
ResultPartitionID partitionId,
int consumedSubpartitionIndex,
int initialBackoff,
int maxBackoff,
Counter numBytesIn,
Counter numBuffersIn) |
Modifier and Type | Method and Description |
---|---|
abstract void |
acknowledgeAllRecordsProcessed()
When received
EndOfData from one channel, it need to acknowledge after this event get
processed. |
protected void |
checkError()
Checks for an error and rethrows it if one was reported.
|
void |
checkpointStarted(CheckpointBarrier barrier)
Called by task thread when checkpointing is started (e.g., any input channel received
barrier).
|
void |
checkpointStopped(long checkpointId)
Called by task thread on cancel/complete to clean-up temporary data.
|
void |
convertToPriorityEvent(int sequenceNumber) |
int |
getChannelIndex()
Returns the index of this channel within its
SingleInputGate . |
InputChannelInfo |
getChannelInfo()
Returns the info of this channel, which uniquely identifies the channel in respect to its
operator instance.
|
int |
getConsumedSubpartitionIndex() |
protected int |
getCurrentBackoff()
Returns the current backoff in ms.
|
abstract Optional<InputChannel.BufferAndAvailability> |
getNextBuffer()
Returns the next buffer from the consumed subpartition or
Optional.empty() if there
is no data to return. |
ResultPartitionID |
getPartitionId() |
protected boolean |
increaseBackoff()
Increases the current backoff and returns whether the operation was successful.
|
protected void |
notifyBufferAvailable(int numAvailableBuffers) |
protected void |
notifyChannelNonEmpty()
Notifies the owning
SingleInputGate that this channel became non-empty. |
void |
notifyPriorityEvent(int priorityBufferNumber) |
void |
notifyRequiredSegmentId(int segmentId)
Notify the upstream the id of required segment that should be sent to netty connection.
|
abstract void |
resumeConsumption()
After sending a
CheckpointBarrier of
exactly-once mode, the upstream will be blocked and become unavailable. |
protected void |
setError(Throwable cause)
Atomically sets an error for this channel and notifies the input gate about available data to
trigger querying this channel by the task thread.
|
int |
unsynchronizedGetNumberOfQueuedBuffers() |
long |
unsynchronizedGetSizeOfQueuedBuffers() |
protected final InputChannelInfo channelInfo
protected final ResultPartitionID partitionId
protected final int consumedSubpartitionIndex
protected final SingleInputGate inputGate
protected final int initialBackoff
protected final int maxBackoff
protected final Counter numBytesIn
protected final Counter numBuffersIn
protected InputChannel(SingleInputGate inputGate, int channelIndex, ResultPartitionID partitionId, int consumedSubpartitionIndex, int initialBackoff, int maxBackoff, Counter numBytesIn, Counter numBuffersIn)
public int getChannelIndex()
SingleInputGate
.public InputChannelInfo getChannelInfo()
public ResultPartitionID getPartitionId()
public int getConsumedSubpartitionIndex()
public abstract void resumeConsumption() throws IOException
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.IOException
public abstract void acknowledgeAllRecordsProcessed() throws IOException
EndOfData
from one channel, it need to acknowledge after this event get
processed.IOException
protected void notifyChannelNonEmpty()
SingleInputGate
that this channel became non-empty.
This is guaranteed to be called only when a Buffer was added to a previously empty input
channel. The notion of empty is atomically consistent with the flag InputChannel.BufferAndAvailability.moreAvailable()
when polling the next buffer from this channel.
Note: When the input channel observes an exception, this method is called regardless of whether the channel was empty before. That ensures that the parent InputGate will always be notified about the exception.
public void notifyPriorityEvent(int priorityBufferNumber)
protected void notifyBufferAvailable(int numAvailableBuffers) throws IOException
IOException
public abstract Optional<InputChannel.BufferAndAvailability> getNextBuffer() throws IOException, InterruptedException
Optional.empty()
if there
is no data to return.IOException
InterruptedException
public void checkpointStarted(CheckpointBarrier barrier) throws CheckpointException
CheckpointException
public void checkpointStopped(long checkpointId)
public void convertToPriorityEvent(int sequenceNumber) throws IOException
IOException
protected void checkError() throws IOException
Note: Any PartitionException
instances should not be transformed and make sure
they are always visible in task failure cause.
IOException
protected void setError(Throwable cause)
protected int getCurrentBackoff()
protected boolean increaseBackoff()
true
, iff the operation was successful. Otherwise, false
.public int unsynchronizedGetNumberOfQueuedBuffers()
public long unsynchronizedGetSizeOfQueuedBuffers()
public void notifyRequiredSegmentId(int segmentId) throws IOException
segmentId
- segment id indicates the id of segment.IOException
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.