Added LeftOuterUnnestMap operator.
- Added LeftOuterUnnestMap operator to represent the left-outer-join semantics properly.
Change-Id: I6760319c2d3ff90c8b7d8ddeea3d9dd8f743366b
Reviewed-on: https://asterix-gerrit.ics.uci.edu/637
Reviewed-by: Yingyi Bu <buyingyi@gmail.com>
Tested-by: Jenkins <jenkins@fulliautomatix.ics.uci.edu>
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/BTreeSearchPOperator.java b/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/BTreeSearchPOperator.java
index cadea03..21adf9f 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/BTreeSearchPOperator.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/BTreeSearchPOperator.java
@@ -34,15 +34,16 @@
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalExpressionTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalOperatorTag;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
import org.apache.hyracks.algebricks.core.algebra.base.PhysicalOperatorTag;
import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression;
import org.apache.hyracks.algebricks.core.algebra.expressions.IVariableTypeEnvironment;
import org.apache.hyracks.algebricks.core.algebra.functions.FunctionIdentifier;
import org.apache.hyracks.algebricks.core.algebra.metadata.IDataSourceIndex;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.IOperatorSchema;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.OrderOperator.IOrder.OrderKind;
-import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities;
import org.apache.hyracks.algebricks.core.algebra.properties.BroadcastPartitioningProperty;
import org.apache.hyracks.algebricks.core.algebra.properties.ILocalStructuralProperty;
@@ -94,7 +95,7 @@
public void contributeRuntimeOperator(IHyracksJobBuilder builder, JobGenContext context, ILogicalOperator op,
IOperatorSchema opSchema, IOperatorSchema[] inputSchemas, IOperatorSchema outerPlanSchema)
throws AlgebricksException {
- UnnestMapOperator unnestMap = (UnnestMapOperator) op;
+ AbstractUnnestMapOperator unnestMap = (AbstractUnnestMapOperator) op;
ILogicalExpression unnestExpr = unnestMap.getExpressionRef().getValue();
if (unnestExpr.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
throw new IllegalStateException();
@@ -120,11 +121,15 @@
outputVars = new ArrayList<LogicalVariable>();
VariableUtilities.getLiveVariables(unnestMap, outputVars);
}
+ boolean retainNull = false;
+ if (op.getOperatorTag() == LogicalOperatorTag.LEFT_OUTER_UNNEST_MAP) {
+ // By nature, LEFT_OUTER_UNNEST_MAP should generate null values for non-matching tuples.
+ retainNull = true;
+ }
Pair<IOperatorDescriptor, AlgebricksPartitionConstraint> btreeSearch = metadataProvider.buildBtreeRuntime(
- builder.getJobSpec(), outputVars, opSchema, typeEnv, context, jobGenParams.getRetainInput(),
- jobGenParams.getRetainNull(), dataset, jobGenParams.getIndexName(), lowKeyIndexes, highKeyIndexes,
- jobGenParams.isLowKeyInclusive(), jobGenParams.isHighKeyInclusive(), implConfig, minFilterFieldIndexes,
- maxFilterFieldIndexes);
+ builder.getJobSpec(), outputVars, opSchema, typeEnv, context, jobGenParams.getRetainInput(), retainNull,
+ dataset, jobGenParams.getIndexName(), lowKeyIndexes, highKeyIndexes, jobGenParams.isLowKeyInclusive(),
+ jobGenParams.isHighKeyInclusive(), implConfig, minFilterFieldIndexes, maxFilterFieldIndexes);
builder.contributeHyracksOperator(unnestMap, btreeSearch.first);
builder.contributeAlgebricksPartitionConstraint(btreeSearch.first, btreeSearch.second);
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/InvertedIndexPOperator.java b/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/InvertedIndexPOperator.java
index 0e5850b..8029166 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/InvertedIndexPOperator.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/InvertedIndexPOperator.java
@@ -53,14 +53,15 @@
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalExpressionTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalOperatorTag;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
import org.apache.hyracks.algebricks.core.algebra.base.PhysicalOperatorTag;
import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression;
import org.apache.hyracks.algebricks.core.algebra.expressions.IAlgebricksConstantValue;
import org.apache.hyracks.algebricks.core.algebra.expressions.IVariableTypeEnvironment;
import org.apache.hyracks.algebricks.core.algebra.metadata.IDataSourceIndex;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.IOperatorSchema;
-import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities;
import org.apache.hyracks.algebricks.core.jobgen.impl.JobGenContext;
import org.apache.hyracks.algebricks.core.jobgen.impl.JobGenHelper;
@@ -106,8 +107,8 @@
@Override
public void contributeRuntimeOperator(IHyracksJobBuilder builder, JobGenContext context, ILogicalOperator op,
IOperatorSchema opSchema, IOperatorSchema[] inputSchemas, IOperatorSchema outerPlanSchema)
- throws AlgebricksException {
- UnnestMapOperator unnestMapOp = (UnnestMapOperator) op;
+ throws AlgebricksException {
+ AbstractUnnestMapOperator unnestMapOp = (AbstractUnnestMapOperator) op;
ILogicalExpression unnestExpr = unnestMapOp.getExpressionRef().getValue();
if (unnestExpr.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
throw new IllegalStateException();
@@ -131,10 +132,15 @@
int[] minFilterFieldIndexes = getKeyIndexes(unnestMapOp.getMinFilterVars(), inputSchemas);
int[] maxFilterFieldIndexes = getKeyIndexes(unnestMapOp.getMaxFilterVars(), inputSchemas);
+ boolean retainNull = false;
+ if (op.getOperatorTag() == LogicalOperatorTag.LEFT_OUTER_UNNEST_MAP) {
+ // By nature, LEFT_OUTER_UNNEST_MAP should generate null values for non-matching tuples.
+ retainNull = true;
+ }
// Build runtime.
Pair<IOperatorDescriptor, AlgebricksPartitionConstraint> invIndexSearch = buildInvertedIndexRuntime(
metadataProvider, context, builder.getJobSpec(), unnestMapOp, opSchema, jobGenParams.getRetainInput(),
- jobGenParams.getRetainNull(), jobGenParams.getDatasetName(), dataset, jobGenParams.getIndexName(),
+ retainNull, jobGenParams.getDatasetName(), dataset, jobGenParams.getIndexName(),
jobGenParams.getSearchKeyType(), keyIndexes, jobGenParams.getSearchModifierType(),
jobGenParams.getSimilarityThreshold(), minFilterFieldIndexes, maxFilterFieldIndexes);
@@ -147,7 +153,7 @@
public Pair<IOperatorDescriptor, AlgebricksPartitionConstraint> buildInvertedIndexRuntime(
AqlMetadataProvider metadataProvider, JobGenContext context, JobSpecification jobSpec,
- UnnestMapOperator unnestMap, IOperatorSchema opSchema, boolean retainInput, boolean retainNull,
+ AbstractUnnestMapOperator unnestMap, IOperatorSchema opSchema, boolean retainInput, boolean retainNull,
String datasetName, Dataset dataset, String indexName, ATypeTag searchKeyType, int[] keyFields,
SearchModifierType searchModifierType, IAlgebricksConstantValue similarityThreshold,
int[] minFilterFieldIndexes, int[] maxFilterFieldIndexes) throws AlgebricksException {
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/RTreeSearchPOperator.java b/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/RTreeSearchPOperator.java
index d54d7f4..ea06619 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/RTreeSearchPOperator.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/RTreeSearchPOperator.java
@@ -33,14 +33,15 @@
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalExpressionTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalOperatorTag;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
import org.apache.hyracks.algebricks.core.algebra.base.PhysicalOperatorTag;
import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression;
import org.apache.hyracks.algebricks.core.algebra.expressions.IVariableTypeEnvironment;
import org.apache.hyracks.algebricks.core.algebra.functions.FunctionIdentifier;
import org.apache.hyracks.algebricks.core.algebra.metadata.IDataSourceIndex;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.IOperatorSchema;
-import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities;
import org.apache.hyracks.algebricks.core.jobgen.impl.JobGenContext;
import org.apache.hyracks.api.dataflow.IOperatorDescriptor;
@@ -64,7 +65,7 @@
public void contributeRuntimeOperator(IHyracksJobBuilder builder, JobGenContext context, ILogicalOperator op,
IOperatorSchema opSchema, IOperatorSchema[] inputSchemas, IOperatorSchema outerPlanSchema)
throws AlgebricksException {
- UnnestMapOperator unnestMap = (UnnestMapOperator) op;
+ AbstractUnnestMapOperator unnestMap = (AbstractUnnestMapOperator) op;
ILogicalExpression unnestExpr = unnestMap.getExpressionRef().getValue();
if (unnestExpr.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
throw new IllegalStateException();
@@ -89,10 +90,14 @@
outputVars = new ArrayList<LogicalVariable>();
VariableUtilities.getLiveVariables(unnestMap, outputVars);
}
+ boolean retainNull = false;
+ if (op.getOperatorTag() == LogicalOperatorTag.LEFT_OUTER_UNNEST_MAP) {
+ // By nature, LEFT_OUTER_UNNEST_MAP should generate null values for non-matching tuples.
+ retainNull = true;
+ }
Pair<IOperatorDescriptor, AlgebricksPartitionConstraint> rtreeSearch = mp.buildRtreeRuntime(
- builder.getJobSpec(), outputVars, opSchema, typeEnv, context, jobGenParams.getRetainInput(),
- jobGenParams.getRetainNull(), dataset, jobGenParams.getIndexName(), keyIndexes, minFilterFieldIndexes,
- maxFilterFieldIndexes);
+ builder.getJobSpec(), outputVars, opSchema, typeEnv, context, jobGenParams.getRetainInput(), retainNull,
+ dataset, jobGenParams.getIndexName(), keyIndexes, minFilterFieldIndexes, maxFilterFieldIndexes);
builder.contributeHyracksOperator(unnestMap, rtreeSearch.first);
builder.contributeAlgebricksPartitionConstraint(rtreeSearch.first, rtreeSearch.second);
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SetAsterixPhysicalOperatorsRule.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SetAsterixPhysicalOperatorsRule.java
index 1ba5702..b0cf533 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SetAsterixPhysicalOperatorsRule.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SetAsterixPhysicalOperatorsRule.java
@@ -51,11 +51,11 @@
import org.apache.hyracks.algebricks.core.algebra.metadata.IDataSourceIndex;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractOperatorWithNestedPlans;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AggregateOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.GroupByOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.InnerJoinOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterJoinOperator;
-import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.physical.ExternalGroupByPOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.physical.PreclusteredGroupByPOperator;
import org.apache.hyracks.algebricks.core.rewriter.base.IAlgebraicRewriteRule;
@@ -71,7 +71,8 @@
}
@Override
- public boolean rewritePre(Mutable<ILogicalOperator> opRef, IOptimizationContext context) throws AlgebricksException {
+ public boolean rewritePre(Mutable<ILogicalOperator> opRef, IOptimizationContext context)
+ throws AlgebricksException {
AbstractLogicalOperator op = (AbstractLogicalOperator) opRef.getValue();
if (context.checkIfInDontApplySet(this, op)) {
return false;
@@ -98,13 +99,14 @@
ILogicalPlan p0 = gby.getNestedPlans().get(0);
if (p0.getRoots().size() == 1) {
Mutable<ILogicalOperator> r0 = p0.getRoots().get(0);
- if (((AbstractLogicalOperator) (r0.getValue())).getOperatorTag().equals(
- LogicalOperatorTag.AGGREGATE)) {
+ if (((AbstractLogicalOperator) (r0.getValue())).getOperatorTag()
+ .equals(LogicalOperatorTag.AGGREGATE)) {
AggregateOperator aggOp = (AggregateOperator) r0.getValue();
boolean serializable = true;
for (Mutable<ILogicalExpression> exprRef : aggOp.getExpressions()) {
AbstractFunctionCallExpression expr = (AbstractFunctionCallExpression) exprRef.getValue();
- if (!AsterixBuiltinFunctions.isAggregateFunctionSerializable(expr.getFunctionIdentifier())) {
+ if (!AsterixBuiltinFunctions
+ .isAggregateFunctionSerializable(expr.getFunctionIdentifier())) {
serializable = false;
break;
}
@@ -170,8 +172,8 @@
op.setPhysicalOperator(new PreclusteredGroupByPOperator(columnList));
}
}
- } else if (((AbstractLogicalOperator) (r0.getValue())).getOperatorTag().equals(
- LogicalOperatorTag.RUNNINGAGGREGATE)) {
+ } else if (((AbstractLogicalOperator) (r0.getValue())).getOperatorTag()
+ .equals(LogicalOperatorTag.RUNNINGAGGREGATE)) {
List<Pair<LogicalVariable, Mutable<ILogicalExpression>>> gbyList = gby.getGroupByList();
List<LogicalVariable> columnList = new ArrayList<LogicalVariable>(gbyList.size());
for (Pair<LogicalVariable, Mutable<ILogicalExpression>> p : gbyList) {
@@ -199,9 +201,10 @@
JoinUtils.setJoinAlgorithmAndExchangeAlgo((LeftOuterJoinOperator) op, context);
break;
}
- case UNNEST_MAP: {
- UnnestMapOperator unnestMap = (UnnestMapOperator) op;
- ILogicalExpression unnestExpr = unnestMap.getExpressionRef().getValue();
+ case UNNEST_MAP:
+ case LEFT_OUTER_UNNEST_MAP: {
+ ILogicalExpression unnestExpr = null;
+ unnestExpr = ((AbstractUnnestMapOperator) op).getExpressionRef().getValue();
if (unnestExpr.getExpressionTag() == LogicalExpressionTag.FUNCTION_CALL) {
AbstractFunctionCallExpression f = (AbstractFunctionCallExpression) unnestExpr;
FunctionIdentifier fid = f.getFunctionIdentifier();
@@ -291,8 +294,8 @@
int n = aggOp.getExpressions().size();
List<Mutable<ILogicalExpression>> mergeExpressionRefs = new ArrayList<Mutable<ILogicalExpression>>();
for (int i = 0; i < n; i++) {
- ILogicalExpression mergeExpr = mergeAggregationExpressionFactory.createMergeAggregation(
- aggProducedVars.get(i), aggFuncRefs.get(i).getValue(), context);
+ ILogicalExpression mergeExpr = mergeAggregationExpressionFactory
+ .createMergeAggregation(aggProducedVars.get(i), aggFuncRefs.get(i).getValue(), context);
if (mergeExpr == null) {
throw new AlgebricksException("The aggregation function " + aggFuncRefs.get(i).getValue()
+ " does not have a registered intermediate aggregation function.");
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SweepIllegalNonfunctionalFunctions.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SweepIllegalNonfunctionalFunctions.java
index 097d192..caad270 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SweepIllegalNonfunctionalFunctions.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/SweepIllegalNonfunctionalFunctions.java
@@ -43,6 +43,7 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.InsertDeleteUpsertOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.IntersectOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterJoinOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LimitOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.MaterializeOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.NestedTupleSourceOperator;
@@ -254,6 +255,12 @@
}
@Override
+ public Void visitLeftOuterUnnestMapOperator(LeftOuterUnnestMapOperator op, Void arg)
+ throws AlgebricksException {
+ return null;
+ }
+
+ @Override
public Void visitDataScanOperator(DataSourceScanOperator op, Void arg) throws AlgebricksException {
return null;
}
@@ -287,12 +294,14 @@
}
@Override
- public Void visitInsertDeleteUpsertOperator(InsertDeleteUpsertOperator op, Void tag) throws AlgebricksException {
+ public Void visitInsertDeleteUpsertOperator(InsertDeleteUpsertOperator op, Void tag)
+ throws AlgebricksException {
return null;
}
@Override
- public Void visitIndexInsertDeleteUpsertOperator(IndexInsertDeleteUpsertOperator op, Void tag) throws AlgebricksException {
+ public Void visitIndexInsertDeleteUpsertOperator(IndexInsertDeleteUpsertOperator op, Void tag)
+ throws AlgebricksException {
return null;
}
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodJobGenParams.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodJobGenParams.java
index 1f3d869..331922d 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodJobGenParams.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodJobGenParams.java
@@ -20,12 +20,11 @@
import java.util.List;
-import org.apache.commons.lang3.mutable.Mutable;
-import org.apache.commons.lang3.mutable.MutableObject;
-
import org.apache.asterix.common.config.DatasetConfig.IndexType;
import org.apache.asterix.om.base.AInt32;
import org.apache.asterix.om.constants.AsterixConstantValue;
+import org.apache.commons.lang3.mutable.Mutable;
+import org.apache.commons.lang3.mutable.MutableObject;
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
import org.apache.hyracks.algebricks.core.algebra.expressions.ConstantExpression;
@@ -41,23 +40,21 @@
protected String dataverseName;
protected String datasetName;
protected boolean retainInput;
- protected boolean retainNull;
protected boolean requiresBroadcast;
protected boolean isPrimaryIndex;
- private final int NUM_PARAMS = 7;
+ private final int NUM_PARAMS = 6;
public AccessMethodJobGenParams() {
}
public AccessMethodJobGenParams(String indexName, IndexType indexType, String dataverseName, String datasetName,
- boolean retainInput, boolean retainNull, boolean requiresBroadcast) {
+ boolean retainInput, boolean requiresBroadcast) {
this.indexName = indexName;
this.indexType = indexType;
this.dataverseName = dataverseName;
this.datasetName = datasetName;
this.retainInput = retainInput;
- this.retainNull = retainNull;
this.requiresBroadcast = requiresBroadcast;
this.isPrimaryIndex = datasetName.equals(indexName);
}
@@ -68,7 +65,6 @@
funcArgs.add(new MutableObject<ILogicalExpression>(AccessMethodUtils.createStringConstant(dataverseName)));
funcArgs.add(new MutableObject<ILogicalExpression>(AccessMethodUtils.createStringConstant(datasetName)));
funcArgs.add(new MutableObject<ILogicalExpression>(AccessMethodUtils.createBooleanConstant(retainInput)));
- funcArgs.add(new MutableObject<ILogicalExpression>(AccessMethodUtils.createBooleanConstant(retainNull)));
funcArgs.add(new MutableObject<ILogicalExpression>(AccessMethodUtils.createBooleanConstant(requiresBroadcast)));
}
@@ -78,8 +74,7 @@
dataverseName = AccessMethodUtils.getStringConstant(funcArgs.get(2));
datasetName = AccessMethodUtils.getStringConstant(funcArgs.get(3));
retainInput = AccessMethodUtils.getBooleanConstant(funcArgs.get(4));
- retainNull = AccessMethodUtils.getBooleanConstant(funcArgs.get(5));
- requiresBroadcast = AccessMethodUtils.getBooleanConstant(funcArgs.get(6));
+ requiresBroadcast = AccessMethodUtils.getBooleanConstant(funcArgs.get(5));
isPrimaryIndex = datasetName.equals(indexName);
}
@@ -103,10 +98,6 @@
return retainInput;
}
- public boolean getRetainNull() {
- return retainNull;
- }
-
public boolean getRequiresBroadcast() {
return requiresBroadcast;
}
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java
index 294a098..2ce43c2 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java
@@ -68,7 +68,9 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractDataSourceOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator.ExecutionMode;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.GroupByOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.OrderOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.OrderOperator.IOrder;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator;
@@ -270,7 +272,10 @@
numPrimaryKeys = DatasetUtils.getPartitioningKeys(dataset).size();
}
List<LogicalVariable> primaryKeyVars = new ArrayList<LogicalVariable>();
- List<LogicalVariable> sourceVars = ((UnnestMapOperator) unnestMapOp).getVariables();
+ List<LogicalVariable> sourceVars = null;
+
+ sourceVars = ((AbstractUnnestMapOperator) unnestMapOp).getVariables();
+
// Assumes the primary keys are located at the end.
int start = sourceVars.size() - numPrimaryKeys;
int stop = sourceVars.size();
@@ -284,7 +289,12 @@
ILogicalOperator unnestMapOp) {
int numPrimaryKeys = DatasetUtils.getPartitioningKeys(dataset).size();
List<LogicalVariable> primaryKeyVars = new ArrayList<LogicalVariable>();
- List<LogicalVariable> sourceVars = ((UnnestMapOperator) unnestMapOp).getVariables();
+ List<LogicalVariable> sourceVars = null;
+
+ // For a left outer join case, LEFT_OUTER_UNNEST_MAP operator is placed
+ // instead of UNNEST_MAP operator.
+ sourceVars = ((AbstractUnnestMapOperator) unnestMapOp).getVariables();
+
// Assumes the primary keys are located at the beginning.
for (int i = 0; i < numPrimaryKeys; i++) {
primaryKeyVars.add(sourceVars.get(i));
@@ -301,7 +311,7 @@
*/
public static Pair<ILogicalExpression, Boolean> createSearchKeyExpr(IOptimizableFuncExpr optFuncExpr,
OptimizableOperatorSubTree indexSubTree, OptimizableOperatorSubTree probeSubTree)
- throws AlgebricksException {
+ throws AlgebricksException {
if (probeSubTree == null) {
// We are optimizing a selection query. Search key is a constant.
// Type Checking and type promotion is done here
@@ -386,10 +396,10 @@
return indexExprs.get(0).second;
}
- public static UnnestMapOperator createSecondaryIndexUnnestMap(Dataset dataset, ARecordType recordType,
+ public static ILogicalOperator createSecondaryIndexUnnestMap(Dataset dataset, ARecordType recordType,
ARecordType metaRecordType, Index index, ILogicalOperator inputOp, AccessMethodJobGenParams jobGenParams,
- IOptimizationContext context, boolean outputPrimaryKeysOnly, boolean retainInput)
- throws AlgebricksException {
+ IOptimizationContext context, boolean outputPrimaryKeysOnly, boolean retainInput, boolean retainNull)
+ throws AlgebricksException {
// The job gen parameters are transferred to the actual job gen via the UnnestMapOperator's function arguments.
ArrayList<Mutable<ILogicalExpression>> secondaryIndexFuncArgs = new ArrayList<Mutable<ILogicalExpression>>();
jobGenParams.writeToFuncArgs(secondaryIndexFuncArgs);
@@ -408,16 +418,35 @@
secondaryIndexSearchFunc.setReturnsUniqueValues(true);
// This is the operator that jobgen will be looking for. It contains an unnest function that has all necessary arguments to determine
// which index to use, which variables contain the index-search keys, what is the original dataset, etc.
- UnnestMapOperator secondaryIndexUnnestOp = new UnnestMapOperator(secondaryIndexUnnestVars,
- new MutableObject<ILogicalExpression>(secondaryIndexSearchFunc), secondaryIndexOutputTypes,
- retainInput);
- secondaryIndexUnnestOp.getInputs().add(new MutableObject<ILogicalOperator>(inputOp));
- context.computeAndSetTypeEnvironmentForOperator(secondaryIndexUnnestOp);
- secondaryIndexUnnestOp.setExecutionMode(ExecutionMode.PARTITIONED);
- return secondaryIndexUnnestOp;
+
+ // Left-outer-join (retainInput and retainNull) case?
+ // Then, we use the LEFT-OUTER-UNNEST-MAP operator instead of unnest-map operator.
+ if (retainNull) {
+ if (retainInput) {
+ LeftOuterUnnestMapOperator secondaryIndexLeftOuterUnnestOp = new LeftOuterUnnestMapOperator(
+ secondaryIndexUnnestVars, new MutableObject<ILogicalExpression>(secondaryIndexSearchFunc),
+ secondaryIndexOutputTypes, true);
+ secondaryIndexLeftOuterUnnestOp.getInputs().add(new MutableObject<ILogicalOperator>(inputOp));
+ context.computeAndSetTypeEnvironmentForOperator(secondaryIndexLeftOuterUnnestOp);
+ secondaryIndexLeftOuterUnnestOp.setExecutionMode(ExecutionMode.PARTITIONED);
+ return secondaryIndexLeftOuterUnnestOp;
+ } else {
+ // Left-outer-join without retainInput doesn't make sense.
+ throw new AlgebricksException("Left-outer-join should propagate all inputs from the outer branch.");
+ }
+ } else {
+ // If this is not a left-outer-join case, then we use UNNEST-MAP operator.
+ UnnestMapOperator secondaryIndexUnnestOp = new UnnestMapOperator(secondaryIndexUnnestVars,
+ new MutableObject<ILogicalExpression>(secondaryIndexSearchFunc), secondaryIndexOutputTypes,
+ retainInput);
+ secondaryIndexUnnestOp.getInputs().add(new MutableObject<ILogicalOperator>(inputOp));
+ context.computeAndSetTypeEnvironmentForOperator(secondaryIndexUnnestOp);
+ secondaryIndexUnnestOp.setExecutionMode(ExecutionMode.PARTITIONED);
+ return secondaryIndexUnnestOp;
+ }
}
- public static UnnestMapOperator createPrimaryIndexUnnestMap(AbstractDataSourceOperator dataSourceOp,
+ public static AbstractUnnestMapOperator createPrimaryIndexUnnestMap(AbstractDataSourceOperator dataSourceOp,
Dataset dataset, ARecordType recordType, ARecordType metaRecordType, ILogicalOperator inputOp,
IOptimizationContext context, boolean sortPrimaryKeys, boolean retainInput, boolean retainNull,
boolean requiresBroadcast) throws AlgebricksException {
@@ -441,7 +470,7 @@
// The job gen parameters are transferred to the actual job gen via the UnnestMapOperator's function arguments.
List<Mutable<ILogicalExpression>> primaryIndexFuncArgs = new ArrayList<Mutable<ILogicalExpression>>();
BTreeJobGenParams jobGenParams = new BTreeJobGenParams(dataset.getDatasetName(), IndexType.BTREE,
- dataset.getDataverseName(), dataset.getDatasetName(), retainInput, retainNull, requiresBroadcast);
+ dataset.getDataverseName(), dataset.getDatasetName(), retainInput, requiresBroadcast);
// Set low/high inclusive to true for a point lookup.
jobGenParams.setLowKeyInclusive(true);
jobGenParams.setHighKeyInclusive(true);
@@ -461,8 +490,21 @@
primaryIndexFuncArgs);
// This is the operator that jobgen will be looking for. It contains an unnest function that has all necessary arguments to determine
// which index to use, which variables contain the index-search keys, what is the original dataset, etc.
- UnnestMapOperator primaryIndexUnnestOp = new UnnestMapOperator(primaryIndexUnnestVars,
- new MutableObject<ILogicalExpression>(primaryIndexSearchFunc), primaryIndexOutputTypes, retainInput);
+ AbstractUnnestMapOperator primaryIndexUnnestOp = null;
+ if (retainNull) {
+ if (retainInput) {
+ primaryIndexUnnestOp = new LeftOuterUnnestMapOperator(primaryIndexUnnestVars,
+ new MutableObject<ILogicalExpression>(primaryIndexSearchFunc), primaryIndexOutputTypes,
+ retainInput);
+ } else {
+ // Left-outer-join without retainNull and retainInput doesn't make sense.
+ throw new AlgebricksException("Left-outer-join should propagate all inputs from the outer branch.");
+ }
+ } else {
+ primaryIndexUnnestOp = new UnnestMapOperator(primaryIndexUnnestVars,
+ new MutableObject<ILogicalExpression>(primaryIndexSearchFunc), primaryIndexOutputTypes,
+ retainInput);
+ }
// Fed by the order operator or the secondaryIndexUnnestOp.
if (sortPrimaryKeys) {
primaryIndexUnnestOp.getInputs().add(new MutableObject<ILogicalOperator>(order));
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java
index 15dbe95..07f3ca3 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java
@@ -33,6 +33,7 @@
import org.apache.asterix.lang.common.util.FunctionUtil;
import org.apache.asterix.metadata.entities.Dataset;
import org.apache.asterix.metadata.entities.Index;
+import org.apache.asterix.om.functions.AsterixBuiltinFunctions;
import org.apache.asterix.om.types.ARecordType;
import org.apache.asterix.optimizer.rules.util.EquivalenceClassUtils;
import org.apache.commons.lang3.mutable.Mutable;
@@ -48,6 +49,7 @@
import org.apache.hyracks.algebricks.core.algebra.expressions.IVariableTypeEnvironment;
import org.apache.hyracks.algebricks.core.algebra.expressions.IndexedNLJoinExpressionAnnotation;
import org.apache.hyracks.algebricks.core.algebra.expressions.ScalarFunctionCallExpression;
+import org.apache.hyracks.algebricks.core.algebra.expressions.UnnestingFunctionCallExpression;
import org.apache.hyracks.algebricks.core.algebra.expressions.VariableReferenceExpression;
import org.apache.hyracks.algebricks.core.algebra.functions.AlgebricksBuiltinFunctions;
import org.apache.hyracks.algebricks.core.algebra.functions.AlgebricksBuiltinFunctions.ComparisonKind;
@@ -57,7 +59,9 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractDataSourceOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator.ExecutionMode;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AssignOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.util.OperatorManipulationUtil;
@@ -458,7 +462,7 @@
assignKeyExprList, keyVarList, context, constantAtRuntimeExpressions, constAtRuntimeExprVars);
BTreeJobGenParams jobGenParams = new BTreeJobGenParams(chosenIndex.getIndexName(), IndexType.BTREE,
- dataset.getDataverseName(), dataset.getDatasetName(), retainInput, retainNull, requiresBroadcast);
+ dataset.getDataverseName(), dataset.getDatasetName(), retainInput, requiresBroadcast);
jobGenParams.setLowKeyInclusive(lowKeyInclusive[0]);
jobGenParams.setHighKeyInclusive(highKeyInclusive[0]);
jobGenParams.setIsEqCondition(isEqCondition);
@@ -479,11 +483,12 @@
inputOp = probeSubTree.root;
}
- UnnestMapOperator secondaryIndexUnnestOp = AccessMethodUtils.createSecondaryIndexUnnestMap(dataset, recordType,
- metaRecordType, chosenIndex, inputOp, jobGenParams, context, false, retainInput);
+ ILogicalOperator secondaryIndexUnnestOp = AccessMethodUtils.createSecondaryIndexUnnestMap(dataset, recordType,
+ metaRecordType, chosenIndex, inputOp, jobGenParams, context, false, retainInput, retainNull);
// Generate the rest of the upstream plan which feeds the search results into the primary index.
- UnnestMapOperator primaryIndexUnnestOp = null;
+ AbstractUnnestMapOperator primaryIndexUnnestOp = null;
+
boolean isPrimaryIndex = chosenIndex.isPrimaryIndex();
if (dataset.getDatasetType() == DatasetType.EXTERNAL) {
// External dataset
@@ -503,10 +508,12 @@
List<Object> primaryIndexOutputTypes = new ArrayList<Object>();
AccessMethodUtils.appendPrimaryIndexTypes(dataset, recordType, metaRecordType, primaryIndexOutputTypes);
List<LogicalVariable> scanVariables = dataSourceOp.getVariables();
- primaryIndexUnnestOp = new UnnestMapOperator(scanVariables, secondaryIndexUnnestOp.getExpressionRef(),
- primaryIndexOutputTypes, retainInput);
- primaryIndexUnnestOp.getInputs().add(new MutableObject<ILogicalOperator>(inputOp));
+ // Checks whether the primary index search can replace the given
+ // SELECT condition.
+ // If so, condition will be set to null and eventually the SELECT
+ // operator will be removed.
+ // If not, we create a new condition based on remaining ones.
if (!primaryIndexPostProccessingIsNeeded) {
List<Mutable<ILogicalExpression>> remainingFuncExprs = new ArrayList<Mutable<ILogicalExpression>>();
getNewConditionExprs(conditionRef, replacedFuncExprs, remainingFuncExprs);
@@ -519,6 +526,48 @@
}
}
+ // Checks whether LEFT_OUTER_UNNESTMAP operator is required.
+ boolean leftOuterUnnestMapRequired = false;
+ if (retainNull && retainInput) {
+ leftOuterUnnestMapRequired = true;
+ } else {
+ leftOuterUnnestMapRequired = false;
+ }
+
+ if (conditionRef.getValue() != null) {
+ // The job gen parameters are transferred to the actual job gen
+ // via the UnnestMapOperator's function arguments.
+ List<Mutable<ILogicalExpression>> primaryIndexFuncArgs = new ArrayList<Mutable<ILogicalExpression>>();
+ jobGenParams.writeToFuncArgs(primaryIndexFuncArgs);
+ // An index search is expressed as an unnest-map over an
+ // index-search function.
+ IFunctionInfo primaryIndexSearch = FunctionUtil.getFunctionInfo(AsterixBuiltinFunctions.INDEX_SEARCH);
+ UnnestingFunctionCallExpression primaryIndexSearchFunc = new UnnestingFunctionCallExpression(
+ primaryIndexSearch, primaryIndexFuncArgs);
+ primaryIndexSearchFunc.setReturnsUniqueValues(true);
+ if (!leftOuterUnnestMapRequired) {
+ primaryIndexUnnestOp = new UnnestMapOperator(scanVariables,
+ new MutableObject<ILogicalExpression>(primaryIndexSearchFunc), primaryIndexOutputTypes,
+ retainInput);
+ } else {
+ primaryIndexUnnestOp = new LeftOuterUnnestMapOperator(scanVariables,
+ new MutableObject<ILogicalExpression>(primaryIndexSearchFunc), primaryIndexOutputTypes,
+ true);
+ }
+ } else {
+ if (!leftOuterUnnestMapRequired) {
+ primaryIndexUnnestOp = new UnnestMapOperator(scanVariables,
+ ((UnnestMapOperator) secondaryIndexUnnestOp).getExpressionRef(), primaryIndexOutputTypes,
+ retainInput);
+ } else {
+ primaryIndexUnnestOp = new LeftOuterUnnestMapOperator(scanVariables,
+ ((LeftOuterUnnestMapOperator) secondaryIndexUnnestOp).getExpressionRef(),
+ primaryIndexOutputTypes, true);
+ }
+ }
+
+ primaryIndexUnnestOp.getInputs().add(new MutableObject<ILogicalOperator>(inputOp));
+
// Adds equivalence classes --- one equivalent class between a primary key
// variable and a record field-access expression.
EquivalenceClassUtils.addEquivalenceClassesForPrimaryIndexAccess(primaryIndexUnnestOp, scanVariables,
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeJobGenParams.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeJobGenParams.java
index 7ae9ea9..26501c2 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeJobGenParams.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeJobGenParams.java
@@ -21,10 +21,9 @@
import java.util.ArrayList;
import java.util.List;
+import org.apache.asterix.common.config.DatasetConfig.IndexType;
import org.apache.commons.lang3.mutable.Mutable;
import org.apache.commons.lang3.mutable.MutableObject;
-
-import org.apache.asterix.common.config.DatasetConfig.IndexType;
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
import org.apache.hyracks.algebricks.core.algebra.expressions.ConstantExpression;
@@ -47,8 +46,8 @@
}
public BTreeJobGenParams(String indexName, IndexType indexType, String dataverseName, String datasetName,
- boolean retainInput, boolean retainNull, boolean requiresBroadcast) {
- super(indexName, indexType, dataverseName, datasetName, retainInput, retainNull, requiresBroadcast);
+ boolean retainInput, boolean requiresBroadcast) {
+ super(indexName, indexType, dataverseName, datasetName, retainInput, requiresBroadcast);
}
public void setLowKeyVarList(List<LogicalVariable> keyVarList, int startIndex, int numKeys) {
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java
index 405e4aa..bca7e04 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java
@@ -137,7 +137,7 @@
private boolean intersectAllSecondaryIndexes(List<Pair<IAccessMethod, Index>> chosenIndexes,
Map<IAccessMethod, AccessMethodAnalysisContext> analyzedAMs, IOptimizationContext context)
- throws AlgebricksException {
+ throws AlgebricksException {
Pair<IAccessMethod, Index> chosenIndex = null;
Optional<Pair<IAccessMethod, Index>> primaryIndex = chosenIndexes.stream()
.filter(pair -> pair.second.isPrimaryIndex()).findFirst();
@@ -160,7 +160,7 @@
for (Pair<IAccessMethod, Index> pair : chosenIndexes) {
AccessMethodAnalysisContext analysisCtx = analyzedAMs.get(pair.first);
subRoots.add(pair.first.createSecondaryToPrimaryPlan(conditionRef, subTree, null, pair.second, analysisCtx,
- true, true, false, context));
+ false, false, false, context));
}
ILogicalOperator primaryUnnest = connectAll2ndarySearchPlanWithIntersect(subRoots, context);
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java
index 70d7088..25bcbca 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java
@@ -64,13 +64,13 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractBinaryJoinOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator.ExecutionMode;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AssignOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.DataSourceScanOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.InnerJoinOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.ReplicateOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnionAllOperator;
-import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.LogicalOperatorDeepCopyWithNewVariablesVisitor;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities;
@@ -142,7 +142,7 @@
public boolean analyzeGetItemFuncExpr(AbstractFunctionCallExpression funcExpr,
List<AbstractLogicalOperator> assignsAndUnnests, AccessMethodAnalysisContext analysisCtx)
- throws AlgebricksException {
+ throws AlgebricksException {
if (funcExpr.getFunctionIdentifier() != AsterixBuiltinFunctions.GET_ITEM) {
return false;
}
@@ -375,7 +375,7 @@
InvertedIndexJobGenParams jobGenParams = new InvertedIndexJobGenParams(chosenIndex.getIndexName(),
chosenIndex.getIndexType(), dataset.getDataverseName(), dataset.getDatasetName(), retainInput,
- retainNull, requiresBroadcast);
+ requiresBroadcast);
// Add function-specific args such as search modifier, and possibly a similarity threshold.
addFunctionSpecificArgs(optFuncExpr, jobGenParams);
// Add the type of search key from the optFuncExpr.
@@ -405,12 +405,13 @@
inputOp = (AbstractLogicalOperator) probeSubTree.root;
}
jobGenParams.setKeyVarList(keyVarList);
- UnnestMapOperator secondaryIndexUnnestOp = AccessMethodUtils.createSecondaryIndexUnnestMap(dataset, recordType,
- metaRecordType, chosenIndex, inputOp, jobGenParams, context, true, retainInput);
+ ILogicalOperator secondaryIndexUnnestOp = AccessMethodUtils.createSecondaryIndexUnnestMap(dataset, recordType,
+ metaRecordType, chosenIndex, inputOp, jobGenParams, context, true, retainInput, retainNull);
// Generate the rest of the upstream plan which feeds the search results into the primary index.
- UnnestMapOperator primaryIndexUnnestOp = AccessMethodUtils.createPrimaryIndexUnnestMap(dataSourceScan, dataset,
- recordType, metaRecordType, secondaryIndexUnnestOp, context, true, retainInput, retainNull, false);
+ AbstractUnnestMapOperator primaryIndexUnnestOp = AccessMethodUtils.createPrimaryIndexUnnestMap(dataSourceScan,
+ dataset, recordType, metaRecordType, secondaryIndexUnnestOp, context, true, retainInput, retainNull,
+ false);
return primaryIndexUnnestOp;
}
@@ -835,7 +836,7 @@
private void addKeyVarsAndExprs(IOptimizableFuncExpr optFuncExpr, ArrayList<LogicalVariable> keyVarList,
ArrayList<Mutable<ILogicalExpression>> keyExprList, IOptimizationContext context)
- throws AlgebricksException {
+ throws AlgebricksException {
// For now we are assuming a single secondary index key.
// Add a variable and its expr to the lists which will be passed into an assign op.
LogicalVariable keyVar = context.newVar();
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexJobGenParams.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexJobGenParams.java
index b526e6e..b6786ad 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexJobGenParams.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexJobGenParams.java
@@ -47,8 +47,8 @@
}
public InvertedIndexJobGenParams(String indexName, IndexType indexType, String dataverseName, String datasetName,
- boolean retainInput, boolean retainNull, boolean requiresBroadcast) {
- super(indexName, indexType, dataverseName, datasetName, retainInput, retainNull, requiresBroadcast);
+ boolean retainInput, boolean requiresBroadcast) {
+ super(indexName, indexType, dataverseName, datasetName, retainInput, requiresBroadcast);
}
public void setSearchModifierType(SearchModifierType searchModifierType) {
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java
index 111fcf4..a712542 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java
@@ -50,6 +50,7 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractDataSourceOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator.ExecutionMode;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.AssignOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.UnnestMapOperator;
@@ -174,7 +175,7 @@
private ILogicalOperator createSecondaryToPrimaryPlan(OptimizableOperatorSubTree indexSubTree,
OptimizableOperatorSubTree probeSubTree, Index chosenIndex, AccessMethodAnalysisContext analysisCtx,
boolean retainInput, boolean retainNull, boolean requiresBroadcast, IOptimizationContext context)
- throws AlgebricksException {
+ throws AlgebricksException {
IOptimizableFuncExpr optFuncExpr = AccessMethodUtils.chooseFirstOptFuncExpr(chosenIndex, analysisCtx);
Dataset dataset = indexSubTree.dataset;
@@ -195,7 +196,7 @@
// we made sure indexSubTree has datasource scan
AbstractDataSourceOperator dataSourceOp = (AbstractDataSourceOperator) indexSubTree.dataSourceRef.getValue();
RTreeJobGenParams jobGenParams = new RTreeJobGenParams(chosenIndex.getIndexName(), IndexType.RTREE,
- dataset.getDataverseName(), dataset.getDatasetName(), retainInput, retainNull, requiresBroadcast);
+ dataset.getDataverseName(), dataset.getDatasetName(), retainInput, requiresBroadcast);
// A spatial object is serialized in the constant of the func expr we are optimizing.
// The R-Tree expects as input an MBR represented with 1 field per dimension.
// Here we generate vars and funcs for extracting MBR fields from the constant into fields of a tuple (as the R-Tree expects them).
@@ -239,8 +240,8 @@
assignSearchKeys.getInputs().add(probeSubTree.rootRef);
}
- UnnestMapOperator secondaryIndexUnnestOp = AccessMethodUtils.createSecondaryIndexUnnestMap(dataset, recordType,
- metaRecordType, chosenIndex, assignSearchKeys, jobGenParams, context, false, retainInput);
+ ILogicalOperator secondaryIndexUnnestOp = AccessMethodUtils.createSecondaryIndexUnnestMap(dataset, recordType,
+ metaRecordType, chosenIndex, assignSearchKeys, jobGenParams, context, false, retainInput, retainNull);
// Generate the rest of the upstream plan which feeds the search results into the primary index.
if (dataset.getDatasetType() == DatasetType.EXTERNAL) {
@@ -248,7 +249,7 @@
dataset, recordType, secondaryIndexUnnestOp, context, chosenIndex, retainInput, retainNull);
return externalDataAccessOp;
} else {
- UnnestMapOperator primaryIndexUnnestOp = AccessMethodUtils.createPrimaryIndexUnnestMap(dataSourceOp,
+ AbstractUnnestMapOperator primaryIndexUnnestOp = AccessMethodUtils.createPrimaryIndexUnnestMap(dataSourceOp,
dataset, recordType, metaRecordType, secondaryIndexUnnestOp, context, true, retainInput, false,
false);
return primaryIndexUnnestOp;
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeJobGenParams.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeJobGenParams.java
index ed74f97..cccb6ef 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeJobGenParams.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeJobGenParams.java
@@ -21,9 +21,8 @@
import java.util.ArrayList;
import java.util.List;
-import org.apache.commons.lang3.mutable.Mutable;
-
import org.apache.asterix.common.config.DatasetConfig.IndexType;
+import org.apache.commons.lang3.mutable.Mutable;
import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
@@ -39,8 +38,8 @@
}
public RTreeJobGenParams(String indexName, IndexType indexType, String dataverseName, String datasetName,
- boolean retainInput, boolean retainNull, boolean requiresBroadcast) {
- super(indexName, indexType, dataverseName, datasetName, retainInput, retainNull, requiresBroadcast);
+ boolean retainInput, boolean requiresBroadcast) {
+ super(indexName, indexType, dataverseName, datasetName, retainInput, requiresBroadcast);
}
public void writeToFuncArgs(List<Mutable<ILogicalExpression>> funcArgs) {
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineAllNtsInSubplanVisitor.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineAllNtsInSubplanVisitor.java
index beebe0f..10ef1f6 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineAllNtsInSubplanVisitor.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineAllNtsInSubplanVisitor.java
@@ -58,6 +58,7 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.InnerJoinOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.IntersectOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterJoinOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LimitOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.MaterializeOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.NestedTupleSourceOperator;
@@ -129,10 +130,12 @@
// The input operator to the subplan.
private final ILogicalOperator subplanInputOperator;
- // Maps live variables at <code>subplanInputOperator</code> to variables in the flattened nested plan.
+ // Maps live variables at <code>subplanInputOperator</code> to variables in
+ // the flattened nested plan.
private final Map<LogicalVariable, LogicalVariable> subplanInputVarToCurrentVarMap = new HashMap<>();
- // Maps variables in the flattened nested plan to live variables at <code>subplannputOperator</code>.
+ // Maps variables in the flattened nested plan to live variables at
+ // <code>subplannputOperator</code>.
private final Map<LogicalVariable, LogicalVariable> currentVarToSubplanInputVarMap = new HashMap<>();
// The set of key variables at the current operator that is being visited.
@@ -141,14 +144,16 @@
// The list of variables determining the ordering.
private final List<Pair<IOrder, Mutable<ILogicalExpression>>> orderingExprs = new ArrayList<>();
- // Maps variables in the flattened nested plan to live variables at <code>subplannputOperator</code>.
+ // Maps variables in the flattened nested plan to live variables at
+ // <code>subplannputOperator</code>.
private final List<Pair<LogicalVariable, LogicalVariable>> varMapIntroducedByRewriting = new ArrayList<>();
/**
* @param context
* the optimization context
* @param subplanInputOperator
- * the input operator to the target subplan operator, which is to be inlined.
+ * the input operator to the target subplan operator, which is to
+ * be inlined.
* @throws AlgebricksException
*/
public InlineAllNtsInSubplanVisitor(IOptimizationContext context, ILogicalOperator subplanOperator)
@@ -191,7 +196,8 @@
public ILogicalOperator visitGroupByOperator(GroupByOperator op, Void arg) throws AlgebricksException {
visitSingleInputOperator(op);
Set<LogicalVariable> groupKeyVars = new HashSet<>();
- // Maps group by key variables if the corresponding expressions are VariableReferenceExpressions.
+ // Maps group by key variables if the corresponding expressions are
+ // VariableReferenceExpressions.
for (Pair<LogicalVariable, Mutable<ILogicalExpression>> keyVarExprRef : op.getGroupByList()) {
ILogicalExpression expr = keyVarExprRef.second.getValue();
if (expr.getExpressionTag() == LogicalExpressionTag.VARIABLE) {
@@ -297,15 +303,18 @@
GroupByOperator gbyOp = new GroupByOperator();
List<Pair<LogicalVariable, LogicalVariable>> keyVarNewVarPairs = new ArrayList<>();
for (LogicalVariable keyVar : correlatedKeyVars) {
- // This limits the visitor can only be applied to a nested logical plan inside a Subplan operator,
- // where the keyVarsToEnforce forms a candidate key which can uniquely identify a tuple out of the nested-tuple-source.
+ // This limits the visitor can only be applied to a nested logical
+ // plan inside a Subplan operator,
+ // where the keyVarsToEnforce forms a candidate key which can
+ // uniquely identify a tuple out of the nested-tuple-source.
LogicalVariable newVar = context.newVar();
gbyOp.getGroupByList().add(new Pair<LogicalVariable, Mutable<ILogicalExpression>>(newVar,
new MutableObject<ILogicalExpression>(new VariableReferenceExpression(keyVar))));
keyVarNewVarPairs.add(new Pair<LogicalVariable, LogicalVariable>(keyVar, newVar));
}
- // Creates an aggregate operator doing LISTIFY, as the root of the nested plan of the added group-by operator.
+ // Creates an aggregate operator doing LISTIFY, as the root of the
+ // nested plan of the added group-by operator.
List<LogicalVariable> aggVarList = new ArrayList<LogicalVariable>();
List<Mutable<ILogicalExpression>> aggExprList = new ArrayList<Mutable<ILogicalExpression>>();
LogicalVariable aggVar = context.newVar();
@@ -318,7 +327,8 @@
aggExprList.add(new MutableObject<ILogicalExpression>(aggExpr));
AggregateOperator aggOp = new AggregateOperator(aggVarList, aggExprList);
- // Adds the original limit operator as the input operator to the added aggregate operator.
+ // Adds the original limit operator as the input operator to the added
+ // aggregate operator.
aggOp.getInputs().add(new MutableObject<ILogicalOperator>(op));
op.getInputs().clear();
ILogicalOperator currentOp = op;
@@ -328,7 +338,8 @@
currentOp = orderOp;
}
- // Adds a nested tuple source operator as the input operator to the limit operator.
+ // Adds a nested tuple source operator as the input operator to the
+ // limit operator.
NestedTupleSourceOperator nts = new NestedTupleSourceOperator(new MutableObject<ILogicalOperator>(gbyOp));
currentOp.getInputs().add(new MutableObject<ILogicalOperator>(nts));
@@ -364,7 +375,7 @@
Set<LogicalVariable> inputLiveVars) {
List<LogicalVariable> fieldAccessVars = new ArrayList<>();
List<Mutable<ILogicalExpression>> fieldAccessExprs = new ArrayList<>();
- //Adds field access by name.
+ // Adds field access by name.
for (LogicalVariable inputLiveVar : inputLiveVars) {
if (!correlatedKeyVars.contains(inputLiveVar)) {
// field Var
@@ -454,7 +465,8 @@
visitSingleInputOperator(op);
List<Mutable<ILogicalExpression>> assignedExprRefs = op.getExpressions();
List<LogicalVariable> assignedVars = op.getVariables();
- // Maps assigning variables if assignment expressions are VariableReferenceExpressions.
+ // Maps assigning variables if assignment expressions are
+ // VariableReferenceExpressions.
for (int index = 0; index < assignedVars.size(); ++index) {
ILogicalExpression expr = assignedExprRefs.get(index).getValue();
if (expr.getExpressionTag() == LogicalExpressionTag.VARIABLE) {
@@ -565,6 +577,13 @@
}
@Override
+ public ILogicalOperator visitLeftOuterUnnestMapOperator(LeftOuterUnnestMapOperator op, Void arg)
+ throws AlgebricksException {
+ throw new AlgebricksException(
+ "The subquery de-correlation rule should always be applied before index-access-method related rules.");
+ }
+
+ @Override
public ILogicalOperator visitDataScanOperator(DataSourceScanOperator op, Void arg) throws AlgebricksException {
return visitSingleInputOperator(op);
}
@@ -593,16 +612,18 @@
}
/**
- * Wraps an AggregateOperator or RunningAggregateOperator with a group-by operator where
- * the group-by keys are variables in keyVarsToEnforce.
- * Note that the function here prevents this visitor being used to rewrite arbitrary query plans.
- * Instead, it could only be used for rewriting a nested plan within a subplan operator.
+ * Wraps an AggregateOperator or RunningAggregateOperator with a group-by
+ * operator where the group-by keys are variables in keyVarsToEnforce. Note
+ * that the function here prevents this visitor being used to rewrite
+ * arbitrary query plans. Instead, it could only be used for rewriting a
+ * nested plan within a subplan operator.
*
* @param op
* the logical operator for aggregate or running aggregate.
* @param keyVarsToEnforce
* the set of variables that needs to preserve.
- * @return the wrapped group-by operator if {@code keyVarsToEnforce} is not empty, and {@code op} otherwise.
+ * @return the wrapped group-by operator if {@code keyVarsToEnforce} is not
+ * empty, and {@code op} otherwise.
* @throws AlgebricksException
*/
private ILogicalOperator visitAggregateOperator(ILogicalOperator op) throws AlgebricksException {
@@ -612,8 +633,10 @@
}
GroupByOperator gbyOp = new GroupByOperator();
for (LogicalVariable keyVar : correlatedKeyVars) {
- // This limits the visitor can only be applied to a nested logical plan inside a Subplan operator,
- // where the keyVarsToEnforce forms a candidate key which can uniquely identify a tuple out of the nested-tuple-source.
+ // This limits the visitor can only be applied to a nested logical
+ // plan inside a Subplan operator,
+ // where the keyVarsToEnforce forms a candidate key which can
+ // uniquely identify a tuple out of the nested-tuple-source.
LogicalVariable newVar = context.newVar();
gbyOp.getGroupByList().add(new Pair<LogicalVariable, Mutable<ILogicalExpression>>(newVar,
new MutableObject<ILogicalExpression>(new VariableReferenceExpression(keyVar))));
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineLeftNtsInSubplanJoinFlatteningVisitor.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineLeftNtsInSubplanJoinFlatteningVisitor.java
index 33ed9f9..65b9eba 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineLeftNtsInSubplanJoinFlatteningVisitor.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/InlineLeftNtsInSubplanJoinFlatteningVisitor.java
@@ -46,6 +46,7 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.InnerJoinOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.IntersectOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterJoinOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LimitOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.MaterializeOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.NestedTupleSourceOperator;
@@ -337,6 +338,13 @@
}
@Override
+ public ILogicalOperator visitLeftOuterUnnestMapOperator(LeftOuterUnnestMapOperator op, Void arg)
+ throws AlgebricksException {
+ throw new AlgebricksException(
+ "The subquery de-correlation rule should always be applied before index-access-method related rules.");
+ }
+
+ @Override
public ILogicalOperator visitDataScanOperator(DataSourceScanOperator op, Void arg) throws AlgebricksException {
return visitSingleInputOperator(op);
}
diff --git a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/SubplanSpecialFlatteningCheckVisitor.java b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/SubplanSpecialFlatteningCheckVisitor.java
index d2fe52f..22cdb7b 100644
--- a/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/SubplanSpecialFlatteningCheckVisitor.java
+++ b/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/subplan/SubplanSpecialFlatteningCheckVisitor.java
@@ -32,6 +32,7 @@
import org.apache.hyracks.algebricks.core.algebra.operators.logical.InnerJoinOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.IntersectOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterJoinOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterUnnestMapOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.LimitOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.MaterializeOperator;
import org.apache.hyracks.algebricks.core.algebra.operators.logical.NestedTupleSourceOperator;
@@ -204,6 +205,11 @@
}
@Override
+ public Boolean visitLeftOuterUnnestMapOperator(LeftOuterUnnestMapOperator op, Void arg) throws AlgebricksException {
+ return false;
+ }
+
+ @Override
public Boolean visitDataScanOperator(DataSourceScanOperator op, Void arg) throws AlgebricksException {
return false;
}
@@ -238,7 +244,8 @@
* If an operator reduces its input cardinality, the operator should not be a descendant
* of a join operator.
*
- * @param op,
+ * @param op
+ * ,
* the operator to consider
* @return FALSE if it is certainly disqualified; TRUE otherwise.
* @throws AlgebricksException
@@ -251,7 +258,8 @@
* If an operator discard tuples variables before the join, the query's
* semantics cannot be preserved after applying the <code>FlatternSubplanJoinRule</code> rule.
*
- * @param op,
+ * @param op
+ * ,
* the operator to consider
* @return FALSE if it is certainly disqualified; TRUE otherwise.
* @throws AlgebricksException
diff --git a/asterix-app/src/test/resources/optimizerts/results/multi-indexes/btree-rtree-ngram-intersect.plan b/asterix-app/src/test/resources/optimizerts/results/multi-indexes/btree-rtree-ngram-intersect.plan
index 2d74649..ecb9eff 100644
--- a/asterix-app/src/test/resources/optimizerts/results/multi-indexes/btree-rtree-ngram-intersect.plan
+++ b/asterix-app/src/test/resources/optimizerts/results/multi-indexes/btree-rtree-ngram-intersect.plan
@@ -20,12 +20,10 @@
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
-- STABLE_SORT [$$31(ASC)] |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- STREAM_PROJECT |PARTITIONED|
+ -- LENGTH_PARTITIONED_INVERTED_INDEX_SEARCH |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- LENGTH_PARTITIONED_INVERTED_INDEX_SEARCH |PARTITIONED|
- -- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- ASSIGN |PARTITIONED|
- -- EMPTY_TUPLE_SOURCE |PARTITIONED|
+ -- ASSIGN |PARTITIONED|
+ -- EMPTY_TUPLE_SOURCE |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
-- STABLE_SORT [$$40(ASC)] |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
diff --git a/asterix-app/src/test/resources/optimizerts/results/multi-indexes/two-inverted-index-intersect.plan b/asterix-app/src/test/resources/optimizerts/results/multi-indexes/two-inverted-index-intersect.plan
index 7d99635..6b762e7 100644
--- a/asterix-app/src/test/resources/optimizerts/results/multi-indexes/two-inverted-index-intersect.plan
+++ b/asterix-app/src/test/resources/optimizerts/results/multi-indexes/two-inverted-index-intersect.plan
@@ -9,18 +9,14 @@
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
-- STABLE_SORT [$$18(ASC)] |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- STREAM_PROJECT |PARTITIONED|
+ -- LENGTH_PARTITIONED_INVERTED_INDEX_SEARCH |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- LENGTH_PARTITIONED_INVERTED_INDEX_SEARCH |PARTITIONED|
- -- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- ASSIGN |PARTITIONED|
- -- EMPTY_TUPLE_SOURCE |PARTITIONED|
+ -- ASSIGN |PARTITIONED|
+ -- EMPTY_TUPLE_SOURCE |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
-- STABLE_SORT [$$20(ASC)] |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- STREAM_PROJECT |PARTITIONED|
+ -- LENGTH_PARTITIONED_INVERTED_INDEX_SEARCH |PARTITIONED|
-- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- LENGTH_PARTITIONED_INVERTED_INDEX_SEARCH |PARTITIONED|
- -- ONE_TO_ONE_EXCHANGE |PARTITIONED|
- -- ASSIGN |PARTITIONED|
- -- EMPTY_TUPLE_SOURCE |PARTITIONED|
+ -- ASSIGN |PARTITIONED|
+ -- EMPTY_TUPLE_SOURCE |PARTITIONED|