public abstract class CommonExecLookupJoin extends ExecNodeBase<RowData>
ExecNode
for temporal table join which shares most methods.
For a lookup join query:
SELECT T.id, T.content, D.age FROM T JOIN userTable FOR SYSTEM_TIME AS OF T.proctime AS D ON T.content = concat(D.name, '!') AND D.age = 11 AND T.id = D.id WHERE D.name LIKE 'Jack%'
The LookupJoin physical node encapsulates the following RelNode tree:
Join (l.name = r.name) / \ RelNode Calc (concat(name, "!") as name, name LIKE 'Jack%') | DimTable (lookup-keys: age=11, id=l.id) (age, id, name)
The workflow of lookup join:
1) lookup records dimension table using the lookup-keys
2) project & filter on the lookup-ed records
3) join left input record and lookup-ed records
4) only outputs the rows which match to the condition
Modifier and Type | Field and Description |
---|---|
static String |
FIELD_NAME_ASYNC_OPTIONS |
static String |
FIELD_NAME_FILTER_ON_TEMPORAL_TABLE |
static String |
FIELD_NAME_INPUT_CHANGELOG_MODE |
static String |
FIELD_NAME_JOIN_TYPE |
static String |
FIELD_NAME_LOOKUP_KEYS |
static String |
FIELD_NAME_PRE_FILTER_CONDITION |
static String |
FIELD_NAME_PROJECTION_ON_TEMPORAL_TABLE |
static String |
FIELD_NAME_REMAINING_JOIN_CONDITION |
static String |
FIELD_NAME_RETRY_OPTIONS |
static String |
FIELD_NAME_TEMPORAL_TABLE |
static String |
LOOKUP_JOIN_MATERIALIZE_TRANSFORMATION |
static String |
LOOKUP_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
Modifier | Constructor and Description |
---|---|
protected |
CommonExecLookupJoin(int id,
ExecNodeContext context,
ReadableConfig persistedConfig,
FlinkJoinType joinType,
org.apache.calcite.rex.RexNode preFilterCondition,
org.apache.calcite.rex.RexNode remainingJoinCondition,
TemporalTableSourceSpec temporalTableSourceSpec,
Map<Integer,LookupJoinUtil.LookupKey> lookupKeys,
List<org.apache.calcite.rex.RexNode> projectionOnTemporalTable,
org.apache.calcite.rex.RexNode filterOnTemporalTable,
LookupJoinUtil.AsyncLookupOptions asyncLookupOptions,
LookupJoinUtil.RetryLookupOptions retryOptions,
ChangelogMode inputChangelogMode,
List<InputProperty> inputProperties,
RowType outputType,
String description) |
Modifier and Type | Method and Description |
---|---|
protected Transformation<RowData> |
createJoinTransformation(org.apache.flink.table.planner.delegation.PlannerBase planner,
ExecNodeConfig config,
boolean upsertMaterialize,
boolean lookupKeyContainsPrimaryKey) |
protected ProcessFunction<RowData,RowData> |
createSyncLookupJoinFunction(org.apache.calcite.plan.RelOptTable temporalTable,
ExecNodeConfig config,
ClassLoader classLoader,
Map<Integer,LookupJoinUtil.LookupKey> allLookupKeys,
TableFunction<?> syncLookupFunction,
RelBuilder relBuilder,
RowType inputRowType,
RowType tableSourceRowType,
RowType resultRowType,
boolean isLeftOuterJoin,
boolean isObjectReuseEnabled) |
protected abstract Transformation<RowData> |
createSyncLookupJoinWithState(Transformation<RowData> inputTransformation,
org.apache.calcite.plan.RelOptTable temporalTable,
ExecNodeConfig config,
ClassLoader classLoader,
Map<Integer,LookupJoinUtil.LookupKey> allLookupKeys,
TableFunction<?> syncLookupFunction,
RelBuilder relBuilder,
RowType inputRowType,
RowType tableSourceRowType,
RowType resultRowType,
boolean isLeftOuterJoin,
boolean isObjectReuseEnabled,
boolean lookupKeyContainsPrimaryKey) |
protected org.apache.calcite.rel.type.RelDataType |
getProjectionOutputRelDataType(RelBuilder relBuilder) |
protected RowType |
getRightOutputRowType(org.apache.calcite.rel.type.RelDataType projectionOutputRelDataType,
RowType tableSourceRowType) |
TemporalTableSourceSpec |
getTemporalTableSourceSpec() |
protected void |
validateLookupKeyType(Map<Integer,LookupJoinUtil.LookupKey> lookupKeys,
RowType inputRowType,
RowType tableSourceRowType) |
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, translateToPlanInternal
public static final String LOOKUP_JOIN_TRANSFORMATION
public static final String LOOKUP_JOIN_MATERIALIZE_TRANSFORMATION
public static final String FIELD_NAME_JOIN_TYPE
public static final String FIELD_NAME_PRE_FILTER_CONDITION
public static final String FIELD_NAME_REMAINING_JOIN_CONDITION
public static final String FIELD_NAME_TEMPORAL_TABLE
public static final String FIELD_NAME_LOOKUP_KEYS
public static final String FIELD_NAME_PROJECTION_ON_TEMPORAL_TABLE
public static final String FIELD_NAME_FILTER_ON_TEMPORAL_TABLE
public static final String FIELD_NAME_INPUT_CHANGELOG_MODE
public static final String FIELD_NAME_ASYNC_OPTIONS
public static final String FIELD_NAME_RETRY_OPTIONS
protected CommonExecLookupJoin(int id, ExecNodeContext context, ReadableConfig persistedConfig, FlinkJoinType joinType, @Nullable org.apache.calcite.rex.RexNode preFilterCondition, @Nullable org.apache.calcite.rex.RexNode remainingJoinCondition, TemporalTableSourceSpec temporalTableSourceSpec, Map<Integer,LookupJoinUtil.LookupKey> lookupKeys, @Nullable List<org.apache.calcite.rex.RexNode> projectionOnTemporalTable, @Nullable org.apache.calcite.rex.RexNode filterOnTemporalTable, @Nullable LookupJoinUtil.AsyncLookupOptions asyncLookupOptions, @Nullable LookupJoinUtil.RetryLookupOptions retryOptions, ChangelogMode inputChangelogMode, List<InputProperty> inputProperties, RowType outputType, String description)
public TemporalTableSourceSpec getTemporalTableSourceSpec()
protected Transformation<RowData> createJoinTransformation(org.apache.flink.table.planner.delegation.PlannerBase planner, ExecNodeConfig config, boolean upsertMaterialize, boolean lookupKeyContainsPrimaryKey)
protected abstract Transformation<RowData> createSyncLookupJoinWithState(Transformation<RowData> inputTransformation, org.apache.calcite.plan.RelOptTable temporalTable, ExecNodeConfig config, ClassLoader classLoader, Map<Integer,LookupJoinUtil.LookupKey> allLookupKeys, TableFunction<?> syncLookupFunction, RelBuilder relBuilder, RowType inputRowType, RowType tableSourceRowType, RowType resultRowType, boolean isLeftOuterJoin, boolean isObjectReuseEnabled, boolean lookupKeyContainsPrimaryKey)
protected void validateLookupKeyType(Map<Integer,LookupJoinUtil.LookupKey> lookupKeys, RowType inputRowType, RowType tableSourceRowType)
protected org.apache.calcite.rel.type.RelDataType getProjectionOutputRelDataType(RelBuilder relBuilder)
protected RowType getRightOutputRowType(org.apache.calcite.rel.type.RelDataType projectionOutputRelDataType, RowType tableSourceRowType)
protected ProcessFunction<RowData,RowData> createSyncLookupJoinFunction(org.apache.calcite.plan.RelOptTable temporalTable, ExecNodeConfig config, ClassLoader classLoader, Map<Integer,LookupJoinUtil.LookupKey> allLookupKeys, TableFunction<?> syncLookupFunction, RelBuilder relBuilder, RowType inputRowType, RowType tableSourceRowType, RowType resultRowType, boolean isLeftOuterJoin, boolean isObjectReuseEnabled)
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.