@Internal public class StreamGraph extends StreamingPlan
Modifier and Type | Field and Description |
---|---|
protected Map<Integer,String> |
vertexIDtoBrokerID |
protected Map<Integer,Long> |
vertexIDtoLoopTimeout |
Constructor and Description |
---|
StreamGraph(StreamExecutionEnvironment environment) |
Modifier and Type | Method and Description |
---|---|
<IN1,IN2,OUT> |
addCoOperator(Integer vertexID,
String slotSharingGroup,
TwoInputStreamOperator<IN1,IN2,OUT> taskOperatorObject,
TypeInformation<IN1> in1TypeInfo,
TypeInformation<IN2> in2TypeInfo,
TypeInformation<OUT> outTypeInfo,
String operatorName) |
void |
addEdge(Integer upStreamVertexID,
Integer downStreamVertexID,
int typeNumber) |
protected StreamNode |
addNode(Integer vertexID,
String slotSharingGroup,
Class<? extends AbstractInvokable> vertexClass,
StreamOperator<?> operatorObject,
String operatorName) |
<IN,OUT> void |
addOperator(Integer vertexID,
String slotSharingGroup,
StreamOperator<OUT> operatorObject,
TypeInformation<IN> inTypeInfo,
TypeInformation<OUT> outTypeInfo,
String operatorName) |
<T> void |
addOutputSelector(Integer vertexID,
OutputSelector<T> outputSelector) |
<IN,OUT> void |
addSink(Integer vertexID,
String slotSharingGroup,
StreamOperator<OUT> operatorObject,
TypeInformation<IN> inTypeInfo,
TypeInformation<OUT> outTypeInfo,
String operatorName) |
<IN,OUT> void |
addSource(Integer vertexID,
String slotSharingGroup,
StreamOperator<OUT> operatorObject,
TypeInformation<IN> inTypeInfo,
TypeInformation<OUT> outTypeInfo,
String operatorName) |
void |
addVirtualPartitionNode(Integer originalId,
Integer virtualId,
StreamPartitioner<?> partitioner)
Adds a new virtual node that is used to connect a downstream vertex to an input with a
certain partitioning.
|
void |
addVirtualSelectNode(Integer originalId,
Integer virtualId,
List<String> selectedNames)
Adds a new virtual node that is used to connect a downstream vertex to only the outputs
with the selected names.
|
void |
addVirtualSideOutputNode(Integer originalId,
Integer virtualId,
OutputTag outputTag)
Adds a new virtual node that is used to connect a downstream vertex to only the outputs with
the selected side-output
OutputTag . |
void |
clear()
Remove all registered nodes etc.
|
Tuple2<StreamNode,StreamNode> |
createIterationSourceAndSink(int loopId,
int sourceId,
int sinkId,
long timeout,
int parallelism,
int maxParallelism,
ResourceSpec minResources,
ResourceSpec preferredResources) |
void |
dumpStreamingPlanAsJSON(File file) |
String |
getBrokerID(Integer vertexID) |
CheckpointConfig |
getCheckpointConfig() |
StreamExecutionEnvironment |
getEnvironment() |
ExecutionConfig |
getExecutionConfig() |
Set<Tuple2<StreamNode,StreamNode>> |
getIterationSourceSinkPairs() |
JobGraph |
getJobGraph()
Gets the assembled
JobGraph . |
String |
getJobName() |
long |
getLoopTimeout(Integer vertexID) |
Set<Tuple2<Integer,StreamOperator<?>>> |
getOperators() |
Collection<Integer> |
getSinkIDs() |
String |
getSlotSharingGroup(Integer id)
Determines the slot sharing group of an operation across virtual nodes.
|
Collection<Integer> |
getSourceIDs() |
StateBackend |
getStateBackend() |
List<StreamEdge> |
getStreamEdges(int sourceId,
int targetId) |
String |
getStreamingPlanAsJSON() |
StreamNode |
getStreamNode(Integer vertexID) |
Collection<StreamNode> |
getStreamNodes() |
protected Collection<? extends Integer> |
getVertexIDs() |
boolean |
isChainingEnabled() |
boolean |
isIterative() |
void |
setBufferTimeout(Integer vertexID,
long bufferTimeout) |
void |
setChaining(boolean chaining) |
void |
setInputFormat(Integer vertexID,
InputFormat<?,?> inputFormat) |
void |
setJobName(String jobName) |
void |
setMaxParallelism(int vertexID,
int maxParallelism) |
void |
setOneInputStateKey(Integer vertexID,
KeySelector<?,?> keySelector,
TypeSerializer<?> keySerializer) |
<IN,OUT> void |
setOperator(Integer vertexID,
StreamOperator<OUT> operatorObject) |
<OUT> void |
setOutType(Integer vertexID,
TypeInformation<OUT> outType) |
void |
setParallelism(Integer vertexID,
int parallelism) |
void |
setResources(int vertexID,
ResourceSpec minResources,
ResourceSpec preferredResources) |
void |
setSerializers(Integer vertexID,
TypeSerializer<?> in1,
TypeSerializer<?> in2,
TypeSerializer<?> out) |
void |
setSerializersFrom(Integer from,
Integer to) |
void |
setStateBackend(StateBackend backend) |
void |
setTwoInputStateKey(Integer vertexID,
KeySelector<?,?> keySelector1,
KeySelector<?,?> keySelector2,
TypeSerializer<?> keySerializer) |
public StreamGraph(StreamExecutionEnvironment environment)
public void clear()
public StreamExecutionEnvironment getEnvironment()
public ExecutionConfig getExecutionConfig()
public CheckpointConfig getCheckpointConfig()
public String getJobName()
public void setJobName(String jobName)
public void setChaining(boolean chaining)
public void setStateBackend(StateBackend backend)
public StateBackend getStateBackend()
public boolean isChainingEnabled()
public boolean isIterative()
public <IN,OUT> void addSource(Integer vertexID, String slotSharingGroup, StreamOperator<OUT> operatorObject, TypeInformation<IN> inTypeInfo, TypeInformation<OUT> outTypeInfo, String operatorName)
public <IN,OUT> void addSink(Integer vertexID, String slotSharingGroup, StreamOperator<OUT> operatorObject, TypeInformation<IN> inTypeInfo, TypeInformation<OUT> outTypeInfo, String operatorName)
public <IN,OUT> void addOperator(Integer vertexID, String slotSharingGroup, StreamOperator<OUT> operatorObject, TypeInformation<IN> inTypeInfo, TypeInformation<OUT> outTypeInfo, String operatorName)
public <IN1,IN2,OUT> void addCoOperator(Integer vertexID, String slotSharingGroup, TwoInputStreamOperator<IN1,IN2,OUT> taskOperatorObject, TypeInformation<IN1> in1TypeInfo, TypeInformation<IN2> in2TypeInfo, TypeInformation<OUT> outTypeInfo, String operatorName)
protected StreamNode addNode(Integer vertexID, String slotSharingGroup, Class<? extends AbstractInvokable> vertexClass, StreamOperator<?> operatorObject, String operatorName)
public void addVirtualSelectNode(Integer originalId, Integer virtualId, List<String> selectedNames)
When adding an edge from the virtual node to a downstream node the connection will be made to the original node, only with the selected names given here.
originalId
- ID of the node that should be connected to.virtualId
- ID of the virtual node.selectedNames
- The selected names.public void addVirtualSideOutputNode(Integer originalId, Integer virtualId, OutputTag outputTag)
OutputTag
.originalId
- ID of the node that should be connected to.virtualId
- ID of the virtual node.outputTag
- The selected side-output OutputTag
.public void addVirtualPartitionNode(Integer originalId, Integer virtualId, StreamPartitioner<?> partitioner)
When adding an edge from the virtual node to a downstream node the connection will be made to the original node, but with the partitioning given here.
originalId
- ID of the node that should be connected to.virtualId
- ID of the virtual node.partitioner
- The partitionerpublic String getSlotSharingGroup(Integer id)
public <T> void addOutputSelector(Integer vertexID, OutputSelector<T> outputSelector)
public void setParallelism(Integer vertexID, int parallelism)
public void setMaxParallelism(int vertexID, int maxParallelism)
public void setResources(int vertexID, ResourceSpec minResources, ResourceSpec preferredResources)
public void setOneInputStateKey(Integer vertexID, KeySelector<?,?> keySelector, TypeSerializer<?> keySerializer)
public void setTwoInputStateKey(Integer vertexID, KeySelector<?,?> keySelector1, KeySelector<?,?> keySelector2, TypeSerializer<?> keySerializer)
public void setBufferTimeout(Integer vertexID, long bufferTimeout)
public void setSerializers(Integer vertexID, TypeSerializer<?> in1, TypeSerializer<?> in2, TypeSerializer<?> out)
public <OUT> void setOutType(Integer vertexID, TypeInformation<OUT> outType)
public <IN,OUT> void setOperator(Integer vertexID, StreamOperator<OUT> operatorObject)
public void setInputFormat(Integer vertexID, InputFormat<?,?> inputFormat)
public StreamNode getStreamNode(Integer vertexID)
protected Collection<? extends Integer> getVertexIDs()
public List<StreamEdge> getStreamEdges(int sourceId, int targetId)
public Collection<Integer> getSourceIDs()
public Collection<Integer> getSinkIDs()
public Collection<StreamNode> getStreamNodes()
public Set<Tuple2<Integer,StreamOperator<?>>> getOperators()
public long getLoopTimeout(Integer vertexID)
public Tuple2<StreamNode,StreamNode> createIterationSourceAndSink(int loopId, int sourceId, int sinkId, long timeout, int parallelism, int maxParallelism, ResourceSpec minResources, ResourceSpec preferredResources)
public Set<Tuple2<StreamNode,StreamNode>> getIterationSourceSinkPairs()
public JobGraph getJobGraph()
JobGraph
.getJobGraph
in class StreamingPlan
public String getStreamingPlanAsJSON()
getStreamingPlanAsJSON
in class StreamingPlan
public void dumpStreamingPlanAsJSON(File file) throws IOException
dumpStreamingPlanAsJSON
in class StreamingPlan
IOException
Copyright © 2014–2019 The Apache Software Foundation. All rights reserved.