Class InputGate
- java.lang.Object
-
- org.apache.flink.runtime.io.network.partition.consumer.InputGate
-
- All Implemented Interfaces:
AutoCloseable
,AvailabilityProvider
,ChannelStateHolder
,PullingAsyncDataInput<BufferOrEvent>
- Direct Known Subclasses:
IndexedInputGate
,UnionInputGate
public abstract class InputGate extends Object implements PullingAsyncDataInput<BufferOrEvent>, AutoCloseable, ChannelStateHolder
An input gate consumes one or more partitions of a single produced intermediate result.Each intermediate result is partitioned over its producing parallel subtasks; each of these partitions is furthermore partitioned into one or more subpartitions.
As an example, consider a map-reduce program, where the map operator produces data and the reduce operator consumes the produced data.
+-----+ +---------------------+ +--------+ | Map | = produce => | Intermediate Result | <= consume = | Reduce | +-----+ +---------------------+ +--------+
When deploying such a program in parallel, the intermediate result will be partitioned over its producing parallel subtasks; each of these partitions is furthermore partitioned into one or more subpartitions.
Intermediate result +-----------------------------------------+ | +----------------+ | +-----------------------+ +-------+ | +-------------+ +=> | Subpartition 1 | | <=======+=== | Input Gate | Reduce 1 | | Map 1 | ==> | | Partition 1 | =| +----------------+ | | +-----------------------+ +-------+ | +-------------+ +=> | Subpartition 2 | | <==+ | | +----------------+ | | | Subpartition request | | | | | +----------------+ | | | +-------+ | +-------------+ +=> | Subpartition 1 | | <==+====+ | Map 2 | ==> | | Partition 2 | =| +----------------+ | | +-----------------------+ +-------+ | +-------------+ +=> | Subpartition 2 | | <==+======== | Input Gate | Reduce 2 | | +----------------+ | +-----------------------+ +-----------------------------------------+
In the above example, two map subtasks produce the intermediate result in parallel, resulting in two partitions (Partition 1 and 2). Each of these partitions is further partitioned into two subpartitions -- one for each parallel reduce subtask. As shown in the Figure, each reduce task will have an input gate attached to it. This will provide its input, which will consist of one subpartition from each partition of the intermediate result.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description protected static class
InputGate.InputWithData<INPUT,DATA>
Simple pojo for INPUT, DATA and moreAvailable.-
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 Modifier and Type Field Description protected AvailabilityProvider.AvailabilityHelper
availabilityHelper
protected AvailabilityProvider.AvailabilityHelper
priorityAvailabilityHelper
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Constructor Summary
Constructors Constructor Description InputGate()
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description abstract void
acknowledgeAllRecordsProcessed(InputChannelInfo channelInfo)
abstract void
finishReadRecoveredState()
CompletableFuture<?>
getAvailableFuture()
abstract InputChannel
getChannel(int channelIndex)
Returns the channel of this gate.List<InputChannelInfo>
getChannelInfos()
Returns the channel infos of this gate.abstract Optional<BufferOrEvent>
getNext()
Blocking call waiting for nextBufferOrEvent
.abstract int
getNumberOfInputChannels()
CompletableFuture<?>
getPriorityEventAvailableFuture()
Notifies when a priority event has been enqueued.abstract CompletableFuture<Void>
getStateConsumedFuture()
abstract boolean
isFinished()
abstract Optional<BufferOrEvent>
pollNext()
Poll theBufferOrEvent
.abstract void
requestPartitions()
abstract void
resumeConsumption(InputChannelInfo channelInfo)
abstract void
sendTaskEvent(TaskEvent event)
void
setChannelStateWriter(ChannelStateWriter channelStateWriter)
Injects theChannelStateWriter
.abstract void
setup()
Setup gate, potentially heavy-weight, blocking operation comparing to just creation.-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface java.lang.AutoCloseable
close
-
Methods inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
isApproximatelyAvailable, isAvailable
-
Methods inherited from interface org.apache.flink.runtime.io.PullingAsyncDataInput
hasReceivedEndOfData
-
-
-
-
Field Detail
-
availabilityHelper
protected final AvailabilityProvider.AvailabilityHelper availabilityHelper
-
priorityAvailabilityHelper
protected final AvailabilityProvider.AvailabilityHelper priorityAvailabilityHelper
-
-
Method Detail
-
setChannelStateWriter
public void setChannelStateWriter(ChannelStateWriter channelStateWriter)
Description copied from interface:ChannelStateHolder
Injects theChannelStateWriter
. Must only be called once.- Specified by:
setChannelStateWriter
in interfaceChannelStateHolder
-
getNumberOfInputChannels
public abstract int getNumberOfInputChannels()
-
isFinished
public abstract boolean isFinished()
- Specified by:
isFinished
in interfacePullingAsyncDataInput<BufferOrEvent>
- Returns:
- true if is finished and for example end of input was reached, false otherwise.
-
getNext
public abstract Optional<BufferOrEvent> getNext() throws IOException, InterruptedException
Blocking call waiting for nextBufferOrEvent
.Note: It should be guaranteed that the previous returned buffer has been recycled before getting next one.
- Returns:
Optional.empty()
ifisFinished()
returns true.- Throws:
IOException
InterruptedException
-
pollNext
public abstract Optional<BufferOrEvent> pollNext() throws IOException, InterruptedException
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>
- Returns:
Optional.empty()
if there is no data to return or ifisFinished()
returns true.- Throws:
IOException
InterruptedException
-
sendTaskEvent
public abstract void sendTaskEvent(TaskEvent event) throws IOException
- Throws:
IOException
-
getAvailableFuture
public CompletableFuture<?> getAvailableFuture()
- Specified by:
getAvailableFuture
in interfaceAvailabilityProvider
- Returns:
- a future that is completed if there are more records available. If there are more
records available immediately,
AvailabilityProvider.AVAILABLE
should be returned. Previously returned not completed futures should become completed once there are more records available.
-
resumeConsumption
public abstract void resumeConsumption(InputChannelInfo channelInfo) throws IOException
- Throws:
IOException
-
acknowledgeAllRecordsProcessed
public abstract void acknowledgeAllRecordsProcessed(InputChannelInfo channelInfo) throws IOException
- Throws:
IOException
-
getChannel
public abstract InputChannel getChannel(int channelIndex)
Returns the channel of this gate.
-
getChannelInfos
public List<InputChannelInfo> getChannelInfos()
Returns the channel infos of this gate.
-
getPriorityEventAvailableFuture
public CompletableFuture<?> getPriorityEventAvailableFuture()
Notifies when a priority event has been enqueued. If this future is queried from task thread, it is guaranteed that a priority event is available and retrieved throughgetNext()
.
-
setup
public abstract void setup() throws IOException
Setup gate, potentially heavy-weight, blocking operation comparing to just creation.- Throws:
IOException
-
requestPartitions
public abstract void requestPartitions() throws IOException
- Throws:
IOException
-
getStateConsumedFuture
public abstract CompletableFuture<Void> getStateConsumedFuture()
-
finishReadRecoveredState
public abstract void finishReadRecoveredState() throws IOException
- Throws:
IOException
-
-