Class UnionInputGate
- java.lang.Object
-
- org.apache.flink.runtime.io.network.partition.consumer.InputGate
-
- org.apache.flink.runtime.io.network.partition.consumer.UnionInputGate
-
- All Implemented Interfaces:
AutoCloseable
,AvailabilityProvider
,ChannelStateHolder
,PullingAsyncDataInput<BufferOrEvent>
public class UnionInputGate extends InputGate
Input gate wrapper to union the input from multiple input gates.Each input gate has input channels attached from which it reads data. At each input gate, the input channels have unique IDs from 0 (inclusive) to the number of input channels (exclusive).
+---+---+ +---+---+---+ | 0 | 1 | | 0 | 1 | 2 | +--------------+--------------+ | Input gate 0 | Input gate 1 | +--------------+--------------+
The union input gate maps these IDs from 0 to the *total* number of input channels across all unioned input gates, e.g. the channels of input gate 0 keep their original indexes and the channel indexes of input gate 1 are set off by 2 to 2--4.
+---+---++---+---+---+ | 0 | 1 || 2 | 3 | 4 | +--------------------+ | Union input gate | +--------------------+
It is NOT possible to recursively union union input gates.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from class org.apache.flink.runtime.io.network.partition.consumer.InputGate
InputGate.InputWithData<INPUT,DATA>
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AvailabilityProvider.AvailabilityHelper
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.PullingAsyncDataInput
PullingAsyncDataInput.EndOfDataStatus
-
-
Field Summary
-
Fields inherited from class org.apache.flink.runtime.io.network.partition.consumer.InputGate
availabilityHelper, priorityAvailabilityHelper
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Constructor Summary
Constructors Constructor Description UnionInputGate(IndexedInputGate... inputGates)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
acknowledgeAllRecordsProcessed(InputChannelInfo channelInfo)
void
close()
void
finishReadRecoveredState()
InputChannel
getChannel(int channelIndex)
Returns the channel of this gate.Optional<BufferOrEvent>
getNext()
Blocking call waiting for nextBufferOrEvent
.int
getNumberOfInputChannels()
Returns the total number of input channels across all unioned input gates.CompletableFuture<Void>
getStateConsumedFuture()
PullingAsyncDataInput.EndOfDataStatus
hasReceivedEndOfData()
Tells if we consumed all available data.boolean
isFinished()
Optional<BufferOrEvent>
pollNext()
Poll theBufferOrEvent
.void
requestPartitions()
void
resumeConsumption(InputChannelInfo channelInfo)
void
sendTaskEvent(TaskEvent event)
void
setup()
Setup gate, potentially heavy-weight, blocking operation comparing to just creation.String
toString()
-
Methods inherited from class org.apache.flink.runtime.io.network.partition.consumer.InputGate
getAvailableFuture, getChannelInfos, getPriorityEventAvailableFuture, setChannelStateWriter
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
-
Methods inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
isApproximatelyAvailable, isAvailable
-
-
-
-
Constructor Detail
-
UnionInputGate
public UnionInputGate(IndexedInputGate... inputGates)
-
-
Method Detail
-
getNumberOfInputChannels
public int getNumberOfInputChannels()
Returns the total number of input channels across all unioned input gates.- Specified by:
getNumberOfInputChannels
in classInputGate
-
getChannel
public InputChannel getChannel(int channelIndex)
Description copied from class:InputGate
Returns the channel of this gate.- Specified by:
getChannel
in classInputGate
-
isFinished
public boolean isFinished()
- Specified by:
isFinished
in interfacePullingAsyncDataInput<BufferOrEvent>
- Specified by:
isFinished
in classInputGate
- Returns:
- true if is finished and for example end of input was reached, false otherwise.
-
hasReceivedEndOfData
public PullingAsyncDataInput.EndOfDataStatus hasReceivedEndOfData()
Description copied from interface:PullingAsyncDataInput
Tells if we consumed all available data.Moreover it tells us the reason why there is no more data incoming. If any of the upstream subtasks finished because of the stop-with-savepoint --no-drain, we should not drain the input. See also
StopMode
.
-
getNext
public Optional<BufferOrEvent> getNext() throws IOException, InterruptedException
Description copied from class:InputGate
Blocking call waiting for nextBufferOrEvent
.Note: It should be guaranteed that the previous returned buffer has been recycled before getting next one.
- Specified by:
getNext
in classInputGate
- Returns:
Optional.empty()
ifInputGate.isFinished()
returns true.- Throws:
IOException
InterruptedException
-
pollNext
public Optional<BufferOrEvent> pollNext() throws IOException, InterruptedException
Description copied from class:InputGate
Poll theBufferOrEvent
.Note: It should be guaranteed that the previous returned buffer has been recycled before polling next one.
- Specified by:
pollNext
in interfacePullingAsyncDataInput<BufferOrEvent>
- Specified by:
pollNext
in classInputGate
- Returns:
Optional.empty()
if there is no data to return or ifInputGate.isFinished()
returns true.- Throws:
IOException
InterruptedException
-
sendTaskEvent
public void sendTaskEvent(TaskEvent event) throws IOException
- Specified by:
sendTaskEvent
in classInputGate
- Throws:
IOException
-
resumeConsumption
public void resumeConsumption(InputChannelInfo channelInfo) throws IOException
- Specified by:
resumeConsumption
in classInputGate
- Throws:
IOException
-
acknowledgeAllRecordsProcessed
public void acknowledgeAllRecordsProcessed(InputChannelInfo channelInfo) throws IOException
- Specified by:
acknowledgeAllRecordsProcessed
in classInputGate
- Throws:
IOException
-
setup
public void setup()
Description copied from class:InputGate
Setup gate, potentially heavy-weight, blocking operation comparing to just creation.
-
getStateConsumedFuture
public CompletableFuture<Void> getStateConsumedFuture()
- Specified by:
getStateConsumedFuture
in classInputGate
-
requestPartitions
public void requestPartitions() throws IOException
- Specified by:
requestPartitions
in classInputGate
- Throws:
IOException
-
close
public void close() throws IOException
- Throws:
IOException
-
finishReadRecoveredState
public void finishReadRecoveredState() throws IOException
- Specified by:
finishReadRecoveredState
in classInputGate
- Throws:
IOException
-
-