Class AbstractStreamTaskNetworkInput<T,R extends RecordDeserializer<DeserializationDelegate<StreamElement>>>
- java.lang.Object
-
- org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput<T,R>
-
- All Implemented Interfaces:
Closeable
,AutoCloseable
,AvailabilityProvider
,PushingAsyncDataInput<T>
,StreamTaskInput<T>
- Direct Known Subclasses:
RescalingStreamTaskNetworkInput
,StreamTaskNetworkInput
public abstract class AbstractStreamTaskNetworkInput<T,R extends RecordDeserializer<DeserializationDelegate<StreamElement>>> extends Object implements StreamTaskInput<T>
Base class for network-based StreamTaskInput where each channel has a designatedRecordDeserializer
for spanning records. Specific implementation bind it to a specificRecordDeserializer
.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AvailabilityProvider.AvailabilityHelper
-
Nested classes/interfaces inherited from interface org.apache.flink.streaming.runtime.io.PushingAsyncDataInput
PushingAsyncDataInput.DataOutput<T>
-
-
Field Summary
Fields Modifier and Type Field Description protected StreamTask.CanEmitBatchOfRecordsChecker
canEmitBatchOfRecords
protected CheckpointedInputGate
checkpointedInputGate
protected DeserializationDelegate<StreamElement>
deserializationDelegate
protected Map<InputChannelInfo,Integer>
flattenedChannelIndices
protected int
inputIndex
protected TypeSerializer<T>
inputSerializer
protected Map<InputChannelInfo,R>
recordDeserializers
protected StatusWatermarkValve
statusWatermarkValve
Valve that controls how watermarks and watermark statuses are forwarded.-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
Fields inherited from interface org.apache.flink.streaming.runtime.io.StreamTaskInput
UNSPECIFIED
-
-
Constructor Summary
Constructors Constructor Description AbstractStreamTaskNetworkInput(CheckpointedInputGate checkpointedInputGate, TypeSerializer<T> inputSerializer, StatusWatermarkValve statusWatermarkValve, int inputIndex, Map<InputChannelInfo,R> recordDeserializers, StreamTask.CanEmitBatchOfRecordsChecker canEmitBatchOfRecords)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
close()
DataInputStatus
emitNext(PushingAsyncDataInput.DataOutput<T> output)
Pushes elements to the output from current data input, and returns the input status to indicate whether there are more available data in current input.protected R
getActiveSerializer(InputChannelInfo channelInfo)
CompletableFuture<?>
getAvailableFuture()
int
getInputIndex()
Returns the input index of this input.protected void
processBuffer(BufferOrEvent bufferOrEvent)
protected DataInputStatus
processEvent(BufferOrEvent bufferOrEvent)
protected void
releaseDeserializer(InputChannelInfo channelInfo)
-
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
-
Methods inherited from interface org.apache.flink.streaming.runtime.io.StreamTaskInput
prepareSnapshot
-
-
-
-
Field Detail
-
checkpointedInputGate
protected final CheckpointedInputGate checkpointedInputGate
-
deserializationDelegate
protected final DeserializationDelegate<StreamElement> deserializationDelegate
-
inputSerializer
protected final TypeSerializer<T> inputSerializer
-
recordDeserializers
protected final Map<InputChannelInfo,R extends RecordDeserializer<DeserializationDelegate<StreamElement>>> recordDeserializers
-
flattenedChannelIndices
protected final Map<InputChannelInfo,Integer> flattenedChannelIndices
-
statusWatermarkValve
protected final StatusWatermarkValve statusWatermarkValve
Valve that controls how watermarks and watermark statuses are forwarded.
-
inputIndex
protected final int inputIndex
-
canEmitBatchOfRecords
protected final StreamTask.CanEmitBatchOfRecordsChecker canEmitBatchOfRecords
-
-
Constructor Detail
-
AbstractStreamTaskNetworkInput
public AbstractStreamTaskNetworkInput(CheckpointedInputGate checkpointedInputGate, TypeSerializer<T> inputSerializer, StatusWatermarkValve statusWatermarkValve, int inputIndex, Map<InputChannelInfo,R> recordDeserializers, StreamTask.CanEmitBatchOfRecordsChecker canEmitBatchOfRecords)
-
-
Method Detail
-
emitNext
public DataInputStatus emitNext(PushingAsyncDataInput.DataOutput<T> output) throws Exception
Description copied from interface:PushingAsyncDataInput
Pushes elements to the output from current data input, and returns the input status to indicate whether there are more available data in current input.This method should be non blocking.
- Specified by:
emitNext
in interfacePushingAsyncDataInput<T>
- Throws:
Exception
-
processEvent
protected DataInputStatus processEvent(BufferOrEvent bufferOrEvent)
-
processBuffer
protected void processBuffer(BufferOrEvent bufferOrEvent) throws IOException
- Throws:
IOException
-
getActiveSerializer
protected R getActiveSerializer(InputChannelInfo channelInfo)
-
getInputIndex
public int getInputIndex()
Description copied from interface:StreamTaskInput
Returns the input index of this input.- Specified by:
getInputIndex
in interfaceStreamTaskInput<T>
-
getAvailableFuture
public CompletableFuture<?> getAvailableFuture()
- Specified by:
getAvailableFuture
in interfaceAvailabilityProvider
- Returns:
- a future that is completed if the respective provider is available.
-
close
public void close() throws IOException
- Specified by:
close
in interfaceAutoCloseable
- Specified by:
close
in interfaceCloseable
- Throws:
IOException
-
releaseDeserializer
protected void releaseDeserializer(InputChannelInfo channelInfo)
-
-