@ExecNodeMetadata(name="stream-exec-temporal-join", version=1, producedTransformations="temporal-join", minPlanVersion=v1_15, minStateVersion=v1_15) public class StreamExecTemporalJoin extends ExecNodeBase<RowData> implements StreamExecNode<RowData>, SingleTransformationTranslator<RowData>
StreamExecNode
for temporal table join (FOR SYSTEM_TIME AS OF) and temporal TableFunction
join (LATERAL TemporalTableFunction(o.proctime)).
The legacy temporal table function join is the subset of temporal table join, the only difference is the validation, we reuse most same logic here.
Modifier and Type | Field and Description |
---|---|
static int |
FIELD_INDEX_FOR_PROC_TIME_ATTRIBUTE |
static String |
FIELD_NAME_IS_TEMPORAL_FUNCTION_JOIN |
static String |
FIELD_NAME_JOIN_SPEC |
static String |
FIELD_NAME_LEFT_TIME_ATTRIBUTE_INDEX |
static String |
FIELD_NAME_RIGHT_TIME_ATTRIBUTE_INDEX |
static String |
TEMPORAL_JOIN_TRANSFORMATION |
FIELD_NAME_CONFIGURATION, FIELD_NAME_DESCRIPTION, FIELD_NAME_ID, FIELD_NAME_INPUT_PROPERTIES, FIELD_NAME_OUTPUT_TYPE, FIELD_NAME_STATE, FIELD_NAME_TYPE
Constructor and Description |
---|
StreamExecTemporalJoin(int id,
ExecNodeContext context,
ReadableConfig persistedConfig,
JoinSpec joinSpec,
boolean isTemporalTableFunctionJoin,
int leftTimeAttributeIndex,
int rightTimeAttributeIndex,
List<InputProperty> inputProperties,
RowType outputType,
String description) |
StreamExecTemporalJoin(ReadableConfig tableConfig,
JoinSpec joinSpec,
boolean isTemporalTableFunctionJoin,
int leftTimeAttributeIndex,
int rightTimeAttributeIndex,
InputProperty leftInputProperty,
InputProperty rightInputProperty,
RowType outputType,
String description) |
Modifier and Type | Method and Description |
---|---|
protected Transformation<RowData> |
translateToPlanInternal(org.apache.flink.table.planner.delegation.PlannerBase planner,
ExecNodeConfig config)
Internal method, translates this node into a Flink operator.
|
accept, createFormattedTransformationDescription, createFormattedTransformationName, createTransformationDescription, createTransformationMeta, createTransformationMeta, createTransformationName, createTransformationUid, getContextFromAnnotation, getDescription, getId, getInputEdges, getInputProperties, getOutputType, getPersistedConfig, getSimplifiedName, getTransformation, inputsContainSingleton, replaceInputEdge, setCompiled, setInputEdges, supportFusionCodegen, translateToFusionCodegenSpec, translateToFusionCodegenSpecInternal, translateToPlan
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
accept, getDescription, getId, getInputEdges, getInputProperties, getOutputType, replaceInputEdge, setCompiled, setInputEdges
translateToPlan
supportFusionCodegen, translateToFusionCodegenSpec
public static final String TEMPORAL_JOIN_TRANSFORMATION
public static final String FIELD_NAME_JOIN_SPEC
public static final String FIELD_NAME_IS_TEMPORAL_FUNCTION_JOIN
public static final String FIELD_NAME_LEFT_TIME_ATTRIBUTE_INDEX
public static final String FIELD_NAME_RIGHT_TIME_ATTRIBUTE_INDEX
public static final int FIELD_INDEX_FOR_PROC_TIME_ATTRIBUTE
public StreamExecTemporalJoin(ReadableConfig tableConfig, JoinSpec joinSpec, boolean isTemporalTableFunctionJoin, int leftTimeAttributeIndex, int rightTimeAttributeIndex, InputProperty leftInputProperty, InputProperty rightInputProperty, RowType outputType, String description)
public StreamExecTemporalJoin(int id, ExecNodeContext context, ReadableConfig persistedConfig, JoinSpec joinSpec, boolean isTemporalTableFunctionJoin, int leftTimeAttributeIndex, int rightTimeAttributeIndex, List<InputProperty> inputProperties, RowType outputType, String description)
protected Transformation<RowData> translateToPlanInternal(org.apache.flink.table.planner.delegation.PlannerBase planner, ExecNodeConfig config)
ExecNodeBase
translateToPlanInternal
in class ExecNodeBase<RowData>
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 the planner
. For more details check ExecNodeConfig
.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.