Class ExecNodeBase<T>
- java.lang.Object
-
- org.apache.flink.table.planner.plan.nodes.exec.ExecNodeBase<T>
-
- Type Parameters:
T
- The type of the elements that result from this node.
- All Implemented Interfaces:
ExecNode<T>
,ExecNodeTranslator<T>
,FusionCodegenExecNode
- Direct Known Subclasses:
BatchExecBoundedStreamScan
,BatchExecDynamicFilteringDataCollector
,BatchExecExecutionOrderEnforcer
,BatchExecGlobalRuntimeFilterBuilder
,BatchExecHashAggregate
,BatchExecHashJoin
,BatchExecHashWindowAggregate
,BatchExecInputAdapter
,BatchExecLimit
,BatchExecLocalRuntimeFilterBuilder
,BatchExecMultipleInput
,BatchExecNestedLoopJoin
,BatchExecOverAggregateBase
,BatchExecPythonGroupAggregate
,BatchExecPythonGroupWindowAggregate
,BatchExecRank
,BatchExecRuntimeFilter
,BatchExecScriptTransform
,BatchExecSort
,BatchExecSortAggregate
,BatchExecSortLimit
,BatchExecSortMergeJoin
,BatchExecSortWindowAggregate
,CommonExecAsyncCalc
,CommonExecCalc
,CommonExecCorrelate
,CommonExecExchange
,CommonExecExpand
,CommonExecLegacySink
,CommonExecLegacyTableSourceScan
,CommonExecLookupJoin
,CommonExecMatch
,CommonExecPythonCalc
,CommonExecPythonCorrelate
,CommonExecSink
,CommonExecTableSourceScan
,CommonExecUnion
,CommonExecValues
,CommonExecWindowTableFunction
,StreamExecAggregateBase
,StreamExecChangelogNormalize
,StreamExecDataStreamScan
,StreamExecDeduplicate
,StreamExecDropUpdateBefore
,StreamExecGroupTableAggregate
,StreamExecIntervalJoin
,StreamExecJoin
,StreamExecMiniBatchAssigner
,StreamExecMultipleInput
,StreamExecOverAggregate
,StreamExecPythonGroupTableAggregate
,StreamExecPythonOverAggregate
,StreamExecRank
,StreamExecSort
,StreamExecTemporalJoin
,StreamExecTemporalSort
,StreamExecWatermarkAssigner
,StreamExecWindowDeduplicate
,StreamExecWindowJoin
,StreamExecWindowRank
public abstract class ExecNodeBase<T> extends Object implements ExecNode<T>
Base class forExecNode
.
-
-
Field Summary
-
Fields inherited from interface org.apache.flink.table.planner.plan.nodes.exec.ExecNode
FIELD_NAME_CONFIGURATION, FIELD_NAME_DESCRIPTION, FIELD_NAME_ID, FIELD_NAME_INPUT_PROPERTIES, FIELD_NAME_OUTPUT_TYPE, FIELD_NAME_STATE, FIELD_NAME_TYPE
-
-
Constructor Summary
Constructors Modifier Constructor Description protected
ExecNodeBase(int id, ExecNodeContext context, ReadableConfig persistedConfig, List<InputProperty> inputProperties, LogicalType outputType, String description)
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
accept(ExecNodeVisitor visitor)
Accepts a visit from aExecNodeVisitor
.protected String
createFormattedTransformationDescription(String description, ReadableConfig config)
protected String
createFormattedTransformationName(String detailName, String simplifiedName, ReadableConfig config)
protected String
createTransformationDescription(ReadableConfig config)
protected TransformationMetadata
createTransformationMeta(String operatorName, String detailName, String simplifiedName, ExecNodeConfig config)
protected TransformationMetadata
createTransformationMeta(String operatorName, ExecNodeConfig config)
protected String
createTransformationName(ReadableConfig config)
protected String
createTransformationUid(String operatorName, ExecNodeConfig config)
protected ExecNodeContext
getContextFromAnnotation()
Retrieves the default context from theExecNodeMetadata
annotation to be serialized into the JSON plan.String
getDescription()
Returns a string which describes this node.int
getId()
The unique ID of the node.List<ExecEdge>
getInputEdges()
Returns a list of this node's inputExecEdge
s.List<InputProperty>
getInputProperties()
Returns a list of this node's input properties.LogicalType
getOutputType()
Returns the outputLogicalType
of this node, this type should be consistent with the type parameterExecNode
.ReadableConfig
getPersistedConfig()
protected String
getSimplifiedName()
Transformation<T>
getTransformation()
protected boolean
inputsContainSingleton()
Whether singleton distribution is required.void
replaceInputEdge(int index, ExecEdge newInputEdge)
Replaces theordinalInParent
th input edge.void
setCompiled(boolean compiled)
Declares whether the node has been created as part of a plan compilation.void
setInputEdges(List<ExecEdge> inputEdges)
Sets the inputExecEdge
s which connect this nodes and its input nodes.boolean
supportFusionCodegen()
Whether this ExecNode supports OFCG or not.OpFusionCodegenSpecGenerator
translateToFusionCodegenSpec(Planner planner, org.apache.flink.table.planner.codegen.CodeGeneratorContext parentCtx)
Translates this node into aOpFusionCodegenSpecGenerator
.protected OpFusionCodegenSpecGenerator
translateToFusionCodegenSpecInternal(org.apache.flink.table.planner.delegation.PlannerBase planner, ExecNodeConfig config, org.apache.flink.table.planner.codegen.CodeGeneratorContext parentCtx)
Internal method, translates this node into a operator codegen spec generator.Transformation<T>
translateToPlan(Planner planner)
Translates this node into aTransformation
.protected abstract Transformation<T>
translateToPlanInternal(org.apache.flink.table.planner.delegation.PlannerBase planner, ExecNodeConfig config)
Internal method, translates this node into a Flink operator.
-
-
-
Constructor Detail
-
ExecNodeBase
protected ExecNodeBase(int id, ExecNodeContext context, ReadableConfig persistedConfig, List<InputProperty> inputProperties, LogicalType outputType, String description)
-
-
Method Detail
-
getContextFromAnnotation
protected final ExecNodeContext getContextFromAnnotation()
Retrieves the default context from theExecNodeMetadata
annotation to be serialized into the JSON plan.
-
getPersistedConfig
public ReadableConfig getPersistedConfig()
-
getId
public final int getId()
Description copied from interface:ExecNode
The unique ID of the node.
-
getDescription
public String getDescription()
Description copied from interface:ExecNode
Returns a string which describes this node.- Specified by:
getDescription
in interfaceExecNode<T>
-
getOutputType
public LogicalType getOutputType()
Description copied from interface:ExecNode
Returns the outputLogicalType
of this node, this type should be consistent with the type parameterExecNode
.Such as, if T is
RowData
, the output type should beRowType
. please refer to the JavaDoc ofRowData
for more info about mapping of logical types to internal data structures.- Specified by:
getOutputType
in interfaceExecNode<T>
-
getInputProperties
public List<InputProperty> getInputProperties()
Description copied from interface:ExecNode
Returns a list of this node's input properties.NOTE: If there are no inputs, returns an empty list, not null.
- Specified by:
getInputProperties
in interfaceExecNode<T>
- Returns:
- List of this node's input properties.
-
getInputEdges
public List<ExecEdge> getInputEdges()
Description copied from interface:ExecNode
Returns a list of this node's inputExecEdge
s.NOTE: If there are no inputs, returns an empty list, not null.
- Specified by:
getInputEdges
in interfaceExecNode<T>
-
setInputEdges
public void setInputEdges(List<ExecEdge> inputEdges)
Description copied from interface:ExecNode
Sets the inputExecEdge
s which connect this nodes and its input nodes.NOTE: If there are no inputs, the given inputEdges should be empty, not null.
- Specified by:
setInputEdges
in interfaceExecNode<T>
- Parameters:
inputEdges
- the inputExecEdge
s.
-
replaceInputEdge
public void replaceInputEdge(int index, ExecEdge newInputEdge)
Description copied from interface:ExecNode
Replaces theordinalInParent
th input edge.- Specified by:
replaceInputEdge
in interfaceExecNode<T>
- Parameters:
index
- Position of the child input edge, 0 is the first.newInputEdge
- New edge that should be put at position `index`.
-
translateToPlan
public final Transformation<T> translateToPlan(Planner planner)
Description copied from interface:ExecNodeTranslator
Translates this node into aTransformation
.NOTE: This method should return same translate result if called multiple times.
- Specified by:
translateToPlan
in interfaceExecNodeTranslator<T>
- Parameters:
planner
- ThePlanner
of the translated graph.
-
accept
public void accept(ExecNodeVisitor visitor)
Description copied from interface:ExecNode
Accepts a visit from aExecNodeVisitor
.
-
setCompiled
public void setCompiled(boolean compiled)
Description copied from interface:ExecNode
Declares whether the node has been created as part of a plan compilation. Some translation properties might be impacted by this (e.g. UID generation for transformations).- Specified by:
setCompiled
in interfaceExecNode<T>
-
translateToPlanInternal
protected abstract Transformation<T> translateToPlanInternal(org.apache.flink.table.planner.delegation.PlannerBase planner, ExecNodeConfig config)
Internal method, translates this node into a Flink operator.- Parameters:
planner
- The planner.config
- per-ExecNode
configuration that contains the merged configuration from various layers which all the nodes implementing this method should use, instead of retrieving configuration from theplanner
. For more details checkExecNodeConfig
.
-
inputsContainSingleton
protected boolean inputsContainSingleton()
Whether singleton distribution is required.
-
getSimplifiedName
protected String getSimplifiedName()
-
createTransformationUid
protected String createTransformationUid(String operatorName, ExecNodeConfig config)
-
createTransformationName
protected String createTransformationName(ReadableConfig config)
-
createTransformationDescription
protected String createTransformationDescription(ReadableConfig config)
-
createTransformationMeta
protected TransformationMetadata createTransformationMeta(String operatorName, ExecNodeConfig config)
-
createTransformationMeta
protected TransformationMetadata createTransformationMeta(String operatorName, String detailName, String simplifiedName, ExecNodeConfig config)
-
createFormattedTransformationDescription
protected String createFormattedTransformationDescription(String description, ReadableConfig config)
-
createFormattedTransformationName
protected String createFormattedTransformationName(String detailName, String simplifiedName, ReadableConfig config)
-
getTransformation
@VisibleForTesting public Transformation<T> getTransformation()
-
supportFusionCodegen
public boolean supportFusionCodegen()
Description copied from interface:FusionCodegenExecNode
Whether this ExecNode supports OFCG or not.- Specified by:
supportFusionCodegen
in interfaceFusionCodegenExecNode
-
translateToFusionCodegenSpec
public OpFusionCodegenSpecGenerator translateToFusionCodegenSpec(Planner planner, org.apache.flink.table.planner.codegen.CodeGeneratorContext parentCtx)
Description copied from interface:FusionCodegenExecNode
Translates this node into aOpFusionCodegenSpecGenerator
.NOTE: This method should return same spec generator result if called multiple times.
- Specified by:
translateToFusionCodegenSpec
in interfaceFusionCodegenExecNode
- Parameters:
planner
- ThePlanner
of the translated graph.parentCtx
- Parent CodeGeneratorContext.
-
translateToFusionCodegenSpecInternal
protected OpFusionCodegenSpecGenerator translateToFusionCodegenSpecInternal(org.apache.flink.table.planner.delegation.PlannerBase planner, ExecNodeConfig config, org.apache.flink.table.planner.codegen.CodeGeneratorContext parentCtx)
Internal method, translates this node into a operator codegen spec generator.- Parameters:
planner
- The planner.config
- per-ExecNode
configuration that contains the merged configuration from various layers which all the nodes implementing this method should use, instead of retrieving configuration from theplanner
. For more details checkExecNodeConfig
.
-
-