Class MultiInputSortingDataInput<IN,K>
- java.lang.Object
-
- org.apache.flink.streaming.api.operators.sort.MultiInputSortingDataInput<IN,K>
-
- All Implemented Interfaces:
Closeable
,AutoCloseable
,AvailabilityProvider
,PushingAsyncDataInput<IN>
,StreamTaskInput<IN>
public final class MultiInputSortingDataInput<IN,K> extends Object implements StreamTaskInput<IN>
An input that wraps an underlying input and sorts the incoming records. It starts emitting records downstream only when all the other inputs coupled with thisMultiInputSortingDataInput
have finished sorting as well.Moreover it will report it is
available
orapproximately available
if it has some records pending only if the head of theMultiInputSortingDataInput.CommonContext.getQueueOfHeads()
belongs to the input. That way there is only ever one input that reports it is available.The sorter uses binary comparison of keys, which are extracted and serialized when received from the chained input. Moreover the timestamps of incoming records are used for secondary ordering. For the comparison it uses either
FixedLengthByteKeyComparator
if the length of the serialized key is constant, orVariableLengthByteKeyComparator
otherwise.Watermarks, watermark statuses, nor latency markers are propagated downstream as they do not make sense with buffered records. The input emits the largest watermark seen after all records.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static class
MultiInputSortingDataInput.SelectableSortingInputs
A wrapper that combines sortinginputs
with aInputSelectable
that should be used to choose which input to consume next from.-
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 inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
Fields inherited from interface org.apache.flink.streaming.runtime.io.StreamTaskInput
UNSPECIFIED
-
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description void
close()
DataInputStatus
emitNext(PushingAsyncDataInput.DataOutput<IN> 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.CompletableFuture<?>
getAvailableFuture()
int
getInputIndex()
Returns the input index of this input.CompletableFuture<Void>
prepareSnapshot(ChannelStateWriter channelStateWriter, long checkpointId)
Prepares to spill the in-flight input buffers as checkpoint snapshot.static <K> MultiInputSortingDataInput.SelectableSortingInputs
wrapInputs(TaskInvokable containingTask, StreamTaskInput<Object>[] sortingInputs, KeySelector<Object,K>[] keySelectors, TypeSerializer<Object>[] inputSerializers, TypeSerializer<K> keySerializer, StreamTaskInput<Object>[] passThroughInputs, MemoryManager memoryManager, IOManager ioManager, boolean objectReuse, double managedMemoryFraction, Configuration taskManagerConfiguration, ExecutionConfig executionConfig)
-
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
-
-
-
-
Method Detail
-
wrapInputs
public static <K> MultiInputSortingDataInput.SelectableSortingInputs wrapInputs(TaskInvokable containingTask, StreamTaskInput<Object>[] sortingInputs, KeySelector<Object,K>[] keySelectors, TypeSerializer<Object>[] inputSerializers, TypeSerializer<K> keySerializer, StreamTaskInput<Object>[] passThroughInputs, MemoryManager memoryManager, IOManager ioManager, boolean objectReuse, double managedMemoryFraction, Configuration taskManagerConfiguration, ExecutionConfig executionConfig)
-
getInputIndex
public int getInputIndex()
Description copied from interface:StreamTaskInput
Returns the input index of this input.- Specified by:
getInputIndex
in interfaceStreamTaskInput<IN>
-
prepareSnapshot
public CompletableFuture<Void> prepareSnapshot(ChannelStateWriter channelStateWriter, long checkpointId)
Description copied from interface:StreamTaskInput
Prepares to spill the in-flight input buffers as checkpoint snapshot.- Specified by:
prepareSnapshot
in interfaceStreamTaskInput<IN>
-
close
public void close() throws IOException
- Specified by:
close
in interfaceAutoCloseable
- Specified by:
close
in interfaceCloseable
- Throws:
IOException
-
emitNext
public DataInputStatus emitNext(PushingAsyncDataInput.DataOutput<IN> 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<IN>
- Throws:
Exception
-
getAvailableFuture
public CompletableFuture<?> getAvailableFuture()
- Specified by:
getAvailableFuture
in interfaceAvailabilityProvider
- Returns:
- a future that is completed if the respective provider is available.
-
-