Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id 74D1D200BCC for ; Tue, 29 Nov 2016 19:06:04 +0100 (CET) Received: by cust-asf.ponee.io (Postfix) id 73E67160B15; Tue, 29 Nov 2016 18:06:04 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id 07732160AFC for ; Tue, 29 Nov 2016 19:06:01 +0100 (CET) Received: (qmail 90234 invoked by uid 500); 29 Nov 2016 18:06:01 -0000 Mailing-List: contact notifications-help@asterixdb.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@asterixdb.apache.org Delivered-To: mailing list notifications@asterixdb.apache.org Received: (qmail 90219 invoked by uid 99); 29 Nov 2016 18:06:01 -0000 Received: from pnap-us-west-generic-nat.apache.org (HELO spamd2-us-west.apache.org) (209.188.14.142) by apache.org (qpsmtpd/0.29) with ESMTP; Tue, 29 Nov 2016 18:06:00 +0000 Received: from localhost (localhost [127.0.0.1]) by spamd2-us-west.apache.org (ASF Mail Server at spamd2-us-west.apache.org) with ESMTP id 1A2391A7B3A for ; Tue, 29 Nov 2016 18:06:00 +0000 (UTC) X-Virus-Scanned: Debian amavisd-new at spamd2-us-west.apache.org X-Spam-Flag: NO X-Spam-Score: 0.919 X-Spam-Level: X-Spam-Status: No, score=0.919 tagged_above=-999 required=6.31 tests=[SPF_FAIL=0.919] autolearn=disabled Received: from mx1-lw-us.apache.org ([10.40.0.8]) by localhost (spamd2-us-west.apache.org [10.40.0.9]) (amavisd-new, port 10024) with ESMTP id gbQkrGdWx4bY for ; Tue, 29 Nov 2016 18:05:49 +0000 (UTC) Received: from unhygienix.ics.uci.edu (unhygienix.ics.uci.edu [128.195.14.130]) by mx1-lw-us.apache.org (ASF Mail Server at mx1-lw-us.apache.org) with ESMTP id A03DF5FC2A for ; Tue, 29 Nov 2016 18:05:48 +0000 (UTC) Received: from localhost (localhost [127.0.0.1]) by unhygienix.ics.uci.edu (Postfix) with ESMTP id 4D9B8241E23; Tue, 29 Nov 2016 10:05:48 -0800 (PST) Date: Tue, 29 Nov 2016 10:05:47 -0800 From: "Till Westmann (Code Review)" To: Steven Jacobs CC: Michael Blow , Jenkins , Yingyi Bu , Preston Carman , Taewoo Kim , Ian Maxon Reply-To: tillw@apache.org X-Gerrit-MessageType: merged Subject: Change in asterixdb[master]: ASTERIXDB-1608, ASTERIXDB-1617 Match user query for nonpure ... X-Gerrit-Change-Id: I2dec322b30835625430c06acd7626d902bada137 X-Gerrit-ChangeURL: X-Gerrit-Commit: 6b8a42f3df9cebb9bd5d56e5986215bc0d98e45c In-Reply-To: References: MIME-Version: 1.0 Content-Type: text/plain; charset=UTF-8 Content-Transfer-Encoding: 8bit Content-Disposition: inline User-Agent: Gerrit/2.8.4 Message-Id: <20161129180548.4D9B8241E23@unhygienix.ics.uci.edu> archived-at: Tue, 29 Nov 2016 18:06:04 -0000 Till Westmann has submitted this change and it was merged. Change subject: ASTERIXDB-1608, ASTERIXDB-1617 Match user query for nonpure function calls ...................................................................... ASTERIXDB-1608, ASTERIXDB-1617 Match user query for nonpure function calls This fix makes it so that nonpure functions are called in the same place and with the same number of executions as specified by the user in the query. This also means that indexes cannot be used for queries that compare with a nonpure call that is made on a per-record basis. Added optimizer tests Change-Id: I2dec322b30835625430c06acd7626d902bada137 Reviewed-on: https://asterix-gerrit.ics.uci.edu/1057 Tested-by: Jenkins Reviewed-by: Till Westmann --- M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/CommitPOperator.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/PushFieldAccessRule.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AbstractIntroduceAccessMethodRule.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IAccessMethod.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IOptimizableFuncExpr.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceLSMComponentFilterRule.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableFuncExpr.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableOperatorSubTree.java M asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-no-index.aql A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index-return-time.aql A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index.aql A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/keep-datetime-local.aql A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-ignore-index.aql A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-no-index.aql A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/maintain-nonpure-location-in-join-cannot-index.aql A asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/query-ASTERIXDB-1608.aql M asterixdb/asterix-app/src/test/resources/optimizerts/results/btree-index/btree-datetime-02.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-no-index.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index-return-time.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/keep-datetime-local.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-ignore-index.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-no-index.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/maintain-nonpure-location-in-join-cannot-index.plan A asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/query-ASTERIXDB-1608.plan A asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.1.ddl.aql A asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.2.update.aql A asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.3.query.aql A asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.ddl.aql A asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.2.update.aql A asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.3.query.aql A asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/global-datetime-use-index/global-datetime-use-index.1.adm A asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.adm M asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml M hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/base/PhysicalOperatorTag.java M hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/util/OperatorPropertiesUtil.java M hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateAssignsRule.java M hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/EnforceStructuralPropertiesRule.java M hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ExtractCommonExpressionsRule.java M hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/InlineVariablesRule.java M hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/PushMapOperatorDownThroughProductRule.java M hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/SetExecutionModeRule.java 47 files changed, 1,023 insertions(+), 171 deletions(-) Approvals: Till Westmann: Looks good to me, approved Jenkins: Verified Objections: Jenkins: Violations found diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/CommitPOperator.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/CommitPOperator.java index 28c883a..1c01c40 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/CommitPOperator.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/algebra/operators/physical/CommitPOperator.java @@ -62,7 +62,7 @@ @Override public PhysicalOperatorTag getOperatorTag() { - return PhysicalOperatorTag.EXTENSION_OPERATOR; + return PhysicalOperatorTag.DELEGATE_OPERATOR; } @Override diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/PushFieldAccessRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/PushFieldAccessRule.java index e68ad02..3fe5e30 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/PushFieldAccessRule.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/PushFieldAccessRule.java @@ -62,10 +62,11 @@ import org.apache.hyracks.algebricks.core.algebra.operators.logical.GroupByOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.NestedTupleSourceOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities; -import org.apache.hyracks.algebricks.core.algebra.util.OperatorPropertiesUtil; import org.apache.hyracks.algebricks.core.rewriter.base.IAlgebraicRewriteRule; public class PushFieldAccessRule implements IAlgebraicRewriteRule { + + private static final String IS_MOVABLE = "isMovable"; @Override public boolean rewritePre(Mutable opRef, IOptimizationContext context) { @@ -184,7 +185,8 @@ && !(op2.getOperatorTag() == LogicalOperatorTag.SELECT && isAccessToIndexedField(access, context))) { return false; } - if (!OperatorPropertiesUtil.isMovable(op2)) { + Object annotation = op2.getAnnotations().get(IS_MOVABLE); + if (annotation != null && !((Boolean) annotation)) { return false; } if (tryingToPushThroughSelectionWithSameDataSource(access, op2)) { diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AbstractIntroduceAccessMethodRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AbstractIntroduceAccessMethodRule.java index 3c79009..c44cebc 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AbstractIntroduceAccessMethodRule.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AbstractIntroduceAccessMethodRule.java @@ -243,11 +243,13 @@ if (j != exprAndVarIdx.second) { matchedTypes.add(optFuncExpr.getFieldType(j)); } + } if (matchedTypes.size() < 2 && optFuncExpr.getNumLogicalVars() == 1) { - matchedTypes.add((IAType) ExpressionTypeComputer.INSTANCE.getType( - optFuncExpr.getConstantAtRuntimeExpr(0), context.getMetadataProvider(), + matchedTypes + .add((IAType) ExpressionTypeComputer.INSTANCE.getType(optFuncExpr.getConstantExpr(0), + context.getMetadataProvider(), typeEnvironment)); } @@ -583,9 +585,7 @@ subTree.getRecordType(), optVarIndex, optFuncExpr.getFuncExpr().getArguments().get(optVarIndex).getValue(), datasetRecordVar, subTree.getMetaRecordType(), datasetMetaVar); - if (fieldName == null) { - continue; - } + IAType fieldType = (IAType) context.getOutputTypeEnvironment(assignOp).getVarType(var); // Set the fieldName in the corresponding matched // function expression. diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java index 2d46a0b..cee77ed 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/AccessMethodUtils.java @@ -74,6 +74,7 @@ import org.apache.hyracks.algebricks.core.algebra.operators.logical.OrderOperator.IOrder; 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.operators.logical.visitors.VariableUtilities; import org.apache.hyracks.algebricks.core.algebra.plan.ALogicalPlanImpl; import org.apache.hyracks.algebricks.core.algebra.util.OperatorPropertiesUtil; @@ -308,11 +309,18 @@ // Type Checking and type promotion is done here IAType fieldType = optFuncExpr.getFieldType(0); + if (optFuncExpr.getNumConstantExpr() == 0) { + //We are looking at a selection case, but using two variables + //This means that the second variable comes from a nonPure function call + //TODO: Right now we miss on type promotion for nonpure functions + return new Pair<>(new VariableReferenceExpression(optFuncExpr.getLogicalVar(1)), false); + } + ILogicalExpression constantAtRuntimeExpression = null; AsterixConstantValue constantValue = null; ATypeTag constantValueTag = null; - constantAtRuntimeExpression = optFuncExpr.getConstantAtRuntimeExpr(0); + constantAtRuntimeExpression = optFuncExpr.getConstantExpr(0); if (constantAtRuntimeExpression.getExpressionTag() == LogicalExpressionTag.CONSTANT) { constantValue = (AsterixConstantValue) ((ConstantExpression) constantAtRuntimeExpression).getValue(); @@ -355,19 +363,16 @@ } if (typeCastingApplied) { - return new Pair(new ConstantExpression(replacedConstantValue), - realTypeConvertedToIntegerType); + return new Pair<>(new ConstantExpression(replacedConstantValue), realTypeConvertedToIntegerType); } else { - return new Pair(optFuncExpr.getConstantAtRuntimeExpr(0), false); + return new Pair<>(optFuncExpr.getConstantExpr(0), false); } } else { // We are optimizing a join query. Determine which variable feeds the secondary index. if (optFuncExpr.getOperatorSubTree(0) == null || optFuncExpr.getOperatorSubTree(0) == probeSubTree) { - return new Pair( - new VariableReferenceExpression(optFuncExpr.getLogicalVar(0)), false); + return new Pair<>(new VariableReferenceExpression(optFuncExpr.getLogicalVar(0)), false); } else { - return new Pair( - new VariableReferenceExpression(optFuncExpr.getLogicalVar(1)), false); + return new Pair<>(new VariableReferenceExpression(optFuncExpr.getLogicalVar(1)), false); } } } @@ -645,7 +650,7 @@ return unnestOp; } - //If the expression is constant at runtime, runturn the type + //If the expression is constant at runtime, return the type public static IAType constantRuntimeResultType(ILogicalExpression expr, IOptimizationContext context, IVariableTypeEnvironment typeEnvironment) throws AlgebricksException { Set usedVariables = new HashSet(); @@ -656,4 +661,24 @@ return (IAType) context.getExpressionTypeComputer().getType(expr, context.getMetadataProvider(), typeEnvironment); } + + //Get Variables used by afterSelectRefs that were created before the datasource + //If there are any, we should retain inputs + public static boolean retainInputs(List dataSourceVariables, ILogicalOperator sourceOp, + List> afterSelectRefs) throws AlgebricksException { + List usedVars = new ArrayList<>(); + List producedVars = new ArrayList<>(); + List liveVars = new ArrayList<>(); + VariableUtilities.getLiveVariables(sourceOp, liveVars); + for (Mutable opMutable : afterSelectRefs) { + ILogicalOperator op = opMutable.getValue(); + VariableUtilities.getUsedVariables(op, usedVars); + VariableUtilities.getProducedVariables(op, producedVars); + } + usedVars.removeAll(producedVars); + usedVars.removeAll(dataSourceVariables); + usedVars.retainAll(liveVars); + return usedVars.isEmpty() ? false : true; + } + } diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java index eb7d3a4..3035c76 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/BTreeAccessMethod.java @@ -120,13 +120,20 @@ } @Override - public boolean applySelectPlanTransformation(Mutable selectRef, - OptimizableOperatorSubTree subTree, Index chosenIndex, AccessMethodAnalysisContext analysisCtx, - IOptimizationContext context) throws AlgebricksException { + public boolean applySelectPlanTransformation(List> afterSelectRefs, + Mutable selectRef, OptimizableOperatorSubTree subTree, Index chosenIndex, + AccessMethodAnalysisContext analysisCtx, IOptimizationContext context) throws AlgebricksException { SelectOperator select = (SelectOperator) selectRef.getValue(); Mutable conditionRef = select.getCondition(); + ILogicalOperator primaryIndexUnnestOp = createSecondaryToPrimaryPlan(conditionRef, subTree, null, chosenIndex, - analysisCtx, false, false, false, context); + analysisCtx, + AccessMethodUtils.retainInputs(subTree.getDataSourceVariables(), subTree.getDataSourceRef().getValue(), + afterSelectRefs), + false, subTree.getDataSourceRef().getValue().getInputs().get(0).getValue() + .getExecutionMode() == ExecutionMode.UNPARTITIONED, + context); + if (primaryIndexUnnestOp == null) { return false; } @@ -484,6 +491,18 @@ OperatorManipulationUtil.deepCopy(dataSourceOp.getInputs().get(0).getValue()))); assignConstantSearchKeys.setExecutionMode(dataSourceOp.getExecutionMode()); inputOp = assignConstantSearchKeys; + } else if (probeSubTree == null) { + //nonpure case + //Make sure that the nonpure function is unpartitioned + ILogicalOperator checkOp = dataSourceOp.getInputs().get(0).getValue(); + while (checkOp.getExecutionMode() != ExecutionMode.UNPARTITIONED) { + if (checkOp.getInputs().size() == 1) { + checkOp = checkOp.getInputs().get(0).getValue(); + } else { + return null; + } + } + inputOp = dataSourceOp.getInputs().get(0).getValue(); } else { // All index search keys are variables. inputOp = probeSubTree.getRoot(); @@ -694,8 +713,11 @@ private boolean probeIsOnLhs(IOptimizableFuncExpr optFuncExpr, OptimizableOperatorSubTree probeSubTree) { if (probeSubTree == null) { + if (optFuncExpr.getConstantExpressions().length == 0) { + return optFuncExpr.getLogicalExpr(0) == null; + } // We are optimizing a selection query. Search key is a constant. Return true if constant is on lhs. - return optFuncExpr.getFuncExpr().getArguments().get(0) == optFuncExpr.getConstantAtRuntimeExpr(0); + return optFuncExpr.getFuncExpr().getArguments().get(0) == optFuncExpr.getConstantExpr(0); } else { // We are optimizing a join query. Determine whether the feeding variable is on the lhs. return (optFuncExpr.getOperatorSubTree(0) == null || optFuncExpr.getOperatorSubTree(0) == probeSubTree); @@ -711,10 +733,21 @@ } @Override - public boolean exprIsOptimizable(Index index, IOptimizableFuncExpr optFuncExpr) { + public boolean exprIsOptimizable(Index index, IOptimizableFuncExpr optFuncExpr) throws AlgebricksException { // If we are optimizing a join, check for the indexed nested-loop join hint. if (optFuncExpr.getNumLogicalVars() == 2) { - if (!optFuncExpr.getFuncExpr().getAnnotations().containsKey(IndexedNLJoinExpressionAnnotation.INSTANCE)) { + if (optFuncExpr.getOperatorSubTree(0) == optFuncExpr.getOperatorSubTree(1)) { + if ((optFuncExpr.getSourceVar(0) == null && optFuncExpr.getFieldType(0) != null) + || (optFuncExpr.getSourceVar(1) == null && optFuncExpr.getFieldType(1) != null)) { + //We are in the select case (trees are the same, and one field comes from non-scan) + //We can do the index search + } else { + //One of the vars was from an assign rather than a scan + //And we were unable to determine its type + return false; + } + } else if (!optFuncExpr.getFuncExpr().getAnnotations() + .containsKey(IndexedNLJoinExpressionAnnotation.INSTANCE)) { return false; } } diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IAccessMethod.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IAccessMethod.java index 5691d57..d249b96 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IAccessMethod.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IAccessMethod.java @@ -30,7 +30,6 @@ 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.operators.logical.AbstractLogicalOperator; -import org.apache.hyracks.data.std.api.IDataOutputProvider; /** * Interface that an access method should implement to work with the rewrite @@ -38,7 +37,7 @@ * methods for analyzing a select/join condition, and for rewriting the plan * with a given index. */ -public interface IAccessMethod extends Comparable{ +public interface IAccessMethod extends Comparable { /** * @return A list of function identifiers that are optimizable by this @@ -80,19 +79,17 @@ /** * Applies the plan transformation to use chosenIndex to optimize a selection query. + * + * @param afterSelectRefs */ - public boolean applySelectPlanTransformation(Mutable selectRef, - OptimizableOperatorSubTree subTree, Index chosenIndex, AccessMethodAnalysisContext analysisCtx, - IOptimizationContext context) throws AlgebricksException; + public boolean applySelectPlanTransformation(List> afterSelectRefs, + Mutable selectRef, OptimizableOperatorSubTree subTree, Index chosenIndex, + AccessMethodAnalysisContext analysisCtx, IOptimizationContext context) throws AlgebricksException; public ILogicalOperator createSecondaryToPrimaryPlan(Mutable conditionRef, - OptimizableOperatorSubTree indexSubTree, - OptimizableOperatorSubTree probeSubTree, - Index chosenIndex, - AccessMethodAnalysisContext analysisCtx, - boolean retainInput, boolean retainNull, boolean requiresBroadcast, - IOptimizationContext context) - throws AlgebricksException; + OptimizableOperatorSubTree indexSubTree, OptimizableOperatorSubTree probeSubTree, Index chosenIndex, + AccessMethodAnalysisContext analysisCtx, boolean retainInput, boolean retainNull, boolean requiresBroadcast, + IOptimizationContext context) throws AlgebricksException; /** * Applies the plan transformation to use chosenIndex to optimize a join query. diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IOptimizableFuncExpr.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IOptimizableFuncExpr.java index b4f8c9f..05dc4a6 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IOptimizableFuncExpr.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IOptimizableFuncExpr.java @@ -35,7 +35,7 @@ public int getNumLogicalVars(); - public int getNumConstantAtRuntimeExpr(); + public int getNumConstantExpr(); public LogicalVariable getLogicalVar(int index); @@ -55,7 +55,7 @@ public OptimizableOperatorSubTree getOperatorSubTree(int index); - public ILogicalExpression getConstantAtRuntimeExpr(int index); + public ILogicalExpression getConstantExpr(int index); public int findLogicalVar(LogicalVariable var); @@ -75,5 +75,7 @@ IAType getConstantType(int index); - void setConstantAtRuntimeExpr(int index, ILogicalExpression expr); + void setConstantExpr(int index, ILogicalExpression expr); + + ILogicalExpression[] getConstantExpressions(); } diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceLSMComponentFilterRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceLSMComponentFilterRule.java index 02fb2a5..53f7a72 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceLSMComponentFilterRule.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceLSMComponentFilterRule.java @@ -131,7 +131,7 @@ for (IOptimizableFuncExpr optFuncExpr : optFuncExprs) { ComparisonKind ck = AlgebricksBuiltinFunctions .getComparisonType(optFuncExpr.getFuncExpr().getFunctionIdentifier()); - ILogicalExpression searchKeyExpr = optFuncExpr.getConstantAtRuntimeExpr(0); + ILogicalExpression searchKeyExpr = optFuncExpr.getConstantExpr(0); LogicalVariable var = context.newVar(); assignKeyExprList.add(new MutableObject(searchKeyExpr)); assignKeyVarList.add(var); diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java index 434b961..1d332b6 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/IntroduceSelectAccessMethodRule.java @@ -25,6 +25,7 @@ import java.util.Optional; import java.util.TreeMap; +import org.apache.asterix.algebra.operators.CommitOperator; import org.apache.asterix.metadata.declared.MetadataProvider; import org.apache.asterix.metadata.entities.Index; import org.apache.commons.lang3.mutable.Mutable; @@ -42,6 +43,8 @@ import org.apache.hyracks.algebricks.core.algebra.expressions.VariableReferenceExpression; import org.apache.hyracks.algebricks.core.algebra.functions.FunctionIdentifier; 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.DelegateOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.IntersectOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.OrderOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator; @@ -79,10 +82,11 @@ // Operators representing the patterns to be matched: // These ops are set in matchesPattern() protected Mutable selectRef = null; - protected SelectOperator select = null; + protected SelectOperator selectOp = null; protected AbstractFunctionCallExpression selectCond = null; protected IVariableTypeEnvironment typeEnvironment = null; protected final OptimizableOperatorSubTree subTree = new OptimizableOperatorSubTree(); + protected List> afterSelectRefs = null; // Register access methods. protected static Map> accessMethods = new HashMap>(); @@ -93,46 +97,111 @@ registerAccessMethod(InvertedIndexAccessMethod.INSTANCE, accessMethods); } + /** + * Recursively check the given plan from the root operator to transform a plan + * with SELECT operator into an index-utilized plan. + */ @Override public boolean rewritePost(Mutable opRef, IOptimizationContext context) throws AlgebricksException { clear(); setMetadataDeclarations(context); - // Match operator pattern and initialize operator members. - if (!matchesOperatorPattern(opRef, context)) { + AbstractLogicalOperator op = (AbstractLogicalOperator) opRef.getValue(); + if (context.checkIfInDontApplySet(this, op)) { return false; } - // Analyze select condition. - Map analyzedAMs = new TreeMap(); - if (!analyzeCondition(selectCond, subTree.getAssignsAndUnnests(), analyzedAMs, context, typeEnvironment)) { + //We start at the top of the plan + if (op.getOperatorTag() != LogicalOperatorTag.DISTRIBUTE_RESULT + && op.getOperatorTag() != LogicalOperatorTag.SINK + && op.getOperatorTag() != LogicalOperatorTag.DELEGATE_OPERATOR) { + return false; + } + if (op.getOperatorTag() == LogicalOperatorTag.DELEGATE_OPERATOR + && !(((DelegateOperator) op).getDelegate() instanceof CommitOperator)) { return false; } - // Set dataset and type metadata. - if (!subTree.setDatasetAndTypeMetadata((MetadataProvider) context.getMetadataProvider())) { - return false; + afterSelectRefs = new ArrayList<>(); + + // Recursively check the given plan whether the desired pattern exists in it. + // If so, try to optimize the plan. + boolean planTransformed = checkAndApplyTheSelectTransformationRule(opRef, context); + + if (selectOp != null) { + // We found an optimization here. Don't need to optimize this operator again. + context.addToDontApplySet(this, selectOp); } - fillSubTreeIndexExprs(subTree, analyzedAMs, context); - pruneIndexCandidates(analyzedAMs, context, typeEnvironment); - - // Choose index to be applied. - List> chosenIndexes = chooseAllIndex(analyzedAMs); - if (chosenIndexes == null || chosenIndexes.size() == 0) { - context.addToDontApplySet(this, select); + if (!planTransformed) { + // We found an optimization here. Don't need to optimize this operator again. return false; - } - - // Apply plan transformation using chosen index. - boolean res = intersectAllSecondaryIndexes(chosenIndexes, analyzedAMs, context); - - if (res) { + } else { OperatorPropertiesUtil.typeOpRec(opRef, context); + } - context.addToDontApplySet(this, select); - return res; + + return planTransformed; + } + + protected boolean checkAndApplyTheSelectTransformationRule(Mutable opRef, + IOptimizationContext context) throws AlgebricksException { + AbstractLogicalOperator op = (AbstractLogicalOperator) opRef.getValue(); + + // Match operator pattern and initialize operator members. + if (matchesOperatorPattern(opRef, context)) { + // Analyze select condition. + Map analyzedAMs = new TreeMap<>(); + if (!analyzeCondition(selectCond, subTree.getAssignsAndUnnests(), analyzedAMs, context, typeEnvironment)) { + return false; + } + + // Set dataset and type metadata. + if (!subTree.setDatasetAndTypeMetadata((MetadataProvider) context.getMetadataProvider())) { + return false; + } + + fillSubTreeIndexExprs(subTree, analyzedAMs, context); + pruneIndexCandidates(analyzedAMs, context, typeEnvironment); + + // Choose index to be applied. + List> chosenIndexes = chooseAllIndex(analyzedAMs); + if (chosenIndexes == null || chosenIndexes.isEmpty()) { + context.addToDontApplySet(this, selectOp); + return false; + } + + // Apply plan transformation using chosen index. + boolean res = intersectAllSecondaryIndexes(chosenIndexes, analyzedAMs, context); + + context.addToDontApplySet(this, selectOp); + if (res) { + OperatorPropertiesUtil.typeOpRec(opRef, context); + return res; + } + selectRef = null; + selectOp = null; + afterSelectRefs.add(opRef); + + } else { + // This is not a SELECT operator. Remember operators + afterSelectRefs.add(opRef); + + } + // Recursively check the plan and try to optimize it. + boolean selectFoundAndOptimizationApplied = false; + for (Mutable inputOpRef : op.getInputs()) { + boolean foundHere = checkAndApplyTheSelectTransformationRule(inputOpRef, context); + if (foundHere) { + selectFoundAndOptimizationApplied = true; + } + } + + // Clean the path after SELECT operator by removing the current operator in the list. + afterSelectRefs.remove(opRef); + return selectFoundAndOptimizationApplied; + } private boolean intersectAllSecondaryIndexes(List> chosenIndexes, @@ -149,18 +218,22 @@ } if (chosenIndex != null) { AccessMethodAnalysisContext analysisCtx = analyzedAMs.get(chosenIndex.first); - return chosenIndex.first.applySelectPlanTransformation(selectRef, subTree, chosenIndex.second, analysisCtx, - context); + return chosenIndex.first.applySelectPlanTransformation(afterSelectRefs, selectRef, subTree, + chosenIndex.second, analysisCtx, context); } // Intersect all secondary indexes, and postpone the primary index search. - Mutable conditionRef = select.getCondition(); + Mutable conditionRef = selectOp.getCondition(); List subRoots = new ArrayList<>(); for (Pair pair : chosenIndexes) { AccessMethodAnalysisContext analysisCtx = analyzedAMs.get(pair.first); subRoots.add(pair.first.createSecondaryToPrimaryPlan(conditionRef, subTree, null, pair.second, analysisCtx, - false, false, false, context)); + AccessMethodUtils.retainInputs(subTree.getDataSourceVariables(), + subTree.getDataSourceRef().getValue(), afterSelectRefs), + false, subTree.getDataSourceRef().getValue().getInputs().get(0).getValue() + .getExecutionMode() == ExecutionMode.UNPARTITIONED, + context)); } ILogicalOperator primaryUnnest = connectAll2ndarySearchPlanWithIntersect(subRoots, context); @@ -217,11 +290,11 @@ } // Set and analyze select. selectRef = opRef; - select = (SelectOperator) op1; + selectOp = (SelectOperator) op1; typeEnvironment = context.getOutputTypeEnvironment(op1); // Check that the select's condition is a function call. - ILogicalExpression condExpr = select.getCondition().getValue(); + ILogicalExpression condExpr = selectOp.getCondition().getValue(); if (condExpr.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) { return false; } @@ -236,8 +309,9 @@ } private void clear() { + afterSelectRefs = null; selectRef = null; - select = null; + selectOp = null; selectCond = null; } } diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java index e43af61..066757d 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/InvertedIndexAccessMethod.java @@ -144,7 +144,7 @@ public boolean analyzeGetItemFuncExpr(AbstractFunctionCallExpression funcExpr, List assignsAndUnnests, AccessMethodAnalysisContext analysisCtx) - throws AlgebricksException { + throws AlgebricksException { if (funcExpr.getFunctionIdentifier() != AsterixBuiltinFunctions.GET_ITEM) { return false; } @@ -436,11 +436,15 @@ } @Override - public boolean applySelectPlanTransformation(Mutable selectRef, - OptimizableOperatorSubTree subTree, Index chosenIndex, AccessMethodAnalysisContext analysisCtx, - IOptimizationContext context) throws AlgebricksException { + public boolean applySelectPlanTransformation(List> afterSelectRefs, + Mutable selectRef, OptimizableOperatorSubTree subTree, Index chosenIndex, + AccessMethodAnalysisContext analysisCtx, IOptimizationContext context) throws AlgebricksException { ILogicalOperator indexPlanRootOp = createSecondaryToPrimaryPlan(null, subTree, null, chosenIndex, analysisCtx, - false, false, false, context); + AccessMethodUtils.retainInputs(subTree.getDataSourceVariables(), subTree.getDataSourceRef().getValue(), + afterSelectRefs), + false, subTree.getDataSourceRef().getValue().getInputs().get(0).getValue() + .getExecutionMode() == ExecutionMode.UNPARTITIONED, + context); // Replace the datasource scan with the new plan rooted at primaryIndexUnnestMap. subTree.getDataSourceRef().setValue(indexPlanRootOp); return true; @@ -737,7 +741,7 @@ isFilterableArgs .add(new MutableObject(new VariableReferenceExpression(inputSearchVar))); // Since we are optimizing a join, the similarity threshold should be the only constant in the optimizable function expression. - isFilterableArgs.add(new MutableObject(optFuncExpr.getConstantAtRuntimeExpr(0))); + isFilterableArgs.add(new MutableObject(optFuncExpr.getConstantExpr(0))); isFilterableArgs.add(new MutableObject( AccessMethodUtils.createInt32Constant(chosenIndex.getGramLength()))); boolean usePrePost = optFuncExpr.containsPartialField() ? false : true; @@ -754,7 +758,7 @@ isFilterableArgs .add(new MutableObject(new VariableReferenceExpression(inputSearchVar))); // Since we are optimizing a join, the similarity threshold should be the only constant in the optimizable function expression. - isFilterableArgs.add(new MutableObject(optFuncExpr.getConstantAtRuntimeExpr(0))); + isFilterableArgs.add(new MutableObject(optFuncExpr.getConstantExpr(0))); isFilterableExpr = new ScalarFunctionCallExpression( FunctionUtil.getFunctionInfo(AsterixBuiltinFunctions.EDIT_DISTANCE_LIST_IS_FILTERABLE), isFilterableArgs); @@ -828,8 +832,9 @@ if (optFuncExpr.getFuncExpr().getFunctionIdentifier() == AsterixBuiltinFunctions.SIMILARITY_JACCARD_CHECK) { jobGenParams.setSearchModifierType(SearchModifierType.JACCARD); // Add the similarity threshold which, by convention, is the last constant value. - jobGenParams.setSimilarityThreshold(((ConstantExpression) optFuncExpr - .getConstantAtRuntimeExpr(optFuncExpr.getNumConstantAtRuntimeExpr() - 1)).getValue()); + jobGenParams.setSimilarityThreshold( + ((ConstantExpression) optFuncExpr.getConstantExpr(optFuncExpr.getNumConstantExpr() - 1)) + .getValue()); } if (optFuncExpr.getFuncExpr().getFunctionIdentifier() == AsterixBuiltinFunctions.EDIT_DISTANCE_CHECK || optFuncExpr.getFuncExpr() @@ -840,19 +845,20 @@ jobGenParams.setSearchModifierType(SearchModifierType.EDIT_DISTANCE); } // Add the similarity threshold which, by convention, is the last constant value. - jobGenParams.setSimilarityThreshold(((ConstantExpression) optFuncExpr - .getConstantAtRuntimeExpr(optFuncExpr.getNumConstantAtRuntimeExpr() - 1)).getValue()); + jobGenParams.setSimilarityThreshold( + ((ConstantExpression) optFuncExpr.getConstantExpr(optFuncExpr.getNumConstantExpr() - 1)) + .getValue()); } } private void addKeyVarsAndExprs(IOptimizableFuncExpr optFuncExpr, ArrayList keyVarList, ArrayList> 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(); keyVarList.add(keyVar); - keyExprList.add(new MutableObject(optFuncExpr.getConstantAtRuntimeExpr(0))); + keyExprList.add(new MutableObject(optFuncExpr.getConstantExpr(0))); return; } @@ -882,7 +888,7 @@ private boolean isEditDistanceFuncOptimizable(Index index, IOptimizableFuncExpr optFuncExpr) throws AlgebricksException { - if (optFuncExpr.getNumConstantAtRuntimeExpr() == 1) { + if (optFuncExpr.getNumConstantExpr() == 1) { return isEditDistanceFuncJoinOptimizable(index, optFuncExpr); } else { return isEditDistanceFuncSelectOptimizable(index, optFuncExpr); @@ -917,7 +923,7 @@ // Check for panic in selection query. // TODO: Panic also depends on prePost which is currently hardcoded to be true. AsterixConstantValue listOrStrConstVal = (AsterixConstantValue) ((ConstantExpression) optFuncExpr - .getConstantAtRuntimeExpr(0)).getValue(); + .getConstantExpr(0)).getValue(); IAObject listOrStrObj = listOrStrConstVal.getObject(); ATypeTag typeTag = listOrStrObj.getType().getTypeTag(); @@ -925,8 +931,8 @@ return false; } - AsterixConstantValue intConstVal = (AsterixConstantValue) ((ConstantExpression) optFuncExpr - .getConstantAtRuntimeExpr(1)).getValue(); + AsterixConstantValue intConstVal = (AsterixConstantValue) ((ConstantExpression) optFuncExpr.getConstantExpr(1)) + .getValue(); IAObject intObj = intConstVal.getObject(); AInt32 edThresh = null; @@ -1084,8 +1090,8 @@ } private boolean isContainsFuncSelectOptimizable(Index index, IOptimizableFuncExpr optFuncExpr) { - AsterixConstantValue strConstVal = (AsterixConstantValue) ((ConstantExpression) optFuncExpr - .getConstantAtRuntimeExpr(0)).getValue(); + AsterixConstantValue strConstVal = (AsterixConstantValue) ((ConstantExpression) optFuncExpr.getConstantExpr(0)) + .getValue(); IAObject strObj = strConstVal.getObject(); ATypeTag typeTag = strObj.getType().getTypeTag(); diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableFuncExpr.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableFuncExpr.java index 2ecd504..d792e3d 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableFuncExpr.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableFuncExpr.java @@ -39,8 +39,8 @@ protected final List> fieldNames; protected final IAType[] fieldTypes; protected final OptimizableOperatorSubTree[] subTrees; - protected final ILogicalExpression[] constantAtRuntimeExpressions; - protected final IAType[] constantAtRuntimeExpressionTypes; + protected final ILogicalExpression[] constantExpressions; + protected final IAType[] constantExpressionTypes; protected boolean partialField; public OptimizableFuncExpr(AbstractFunctionCallExpression funcExpr, LogicalVariable[] logicalVars, @@ -49,8 +49,8 @@ this.logicalVars = logicalVars; this.sourceVars = new LogicalVariable[logicalVars.length]; this.logicalExprs = new ILogicalExpression[logicalVars.length]; - this.constantAtRuntimeExpressionTypes = constantExpressionTypes; - this.constantAtRuntimeExpressions = constantExpressions; + this.constantExpressionTypes = constantExpressionTypes; + this.constantExpressions = constantExpressions; this.fieldNames = new ArrayList>(); for (int i = 0; i < logicalVars.length; i++) { fieldNames.add(new ArrayList()); @@ -72,8 +72,8 @@ this.logicalVars = new LogicalVariable[] { logicalVar }; this.sourceVars = new LogicalVariable[1]; this.logicalExprs = new ILogicalExpression[1]; - this.constantAtRuntimeExpressions = new ILogicalExpression[] { constantExpression }; - this.constantAtRuntimeExpressionTypes = new IAType[] { constantExpressionType }; + this.constantExpressions = new ILogicalExpression[] { constantExpression }; + this.constantExpressionTypes = new IAType[] { constantExpressionType }; this.fieldNames = new ArrayList>(); for (int i = 0; i < logicalVars.length; i++) { fieldNames.add(new ArrayList()); @@ -98,8 +98,8 @@ } @Override - public int getNumConstantAtRuntimeExpr() { - return constantAtRuntimeExpressions.length; + public int getNumConstantExpr() { + return constantExpressions.length; } @Override @@ -138,23 +138,28 @@ } @Override - public ILogicalExpression getConstantAtRuntimeExpr(int index) { - return constantAtRuntimeExpressions[index]; + public ILogicalExpression getConstantExpr(int index) { + return constantExpressions[index]; + } + + @Override + public ILogicalExpression[] getConstantExpressions() { + return constantExpressions; } @Override public void setConstType(int index, IAType fieldType) { - constantAtRuntimeExpressionTypes[index] = fieldType; + constantExpressionTypes[index] = fieldType; } @Override public IAType getConstantType(int index) { - return constantAtRuntimeExpressionTypes[index]; + return constantExpressionTypes[index]; } @Override - public void setConstantAtRuntimeExpr(int index, ILogicalExpression expr) { - constantAtRuntimeExpressions[index] = expr; + public void setConstantExpr(int index, ILogicalExpression expr) { + constantExpressions[index] = expr; } @Override diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableOperatorSubTree.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableOperatorSubTree.java index 75ee46b..1869d60 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableOperatorSubTree.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/OptimizableOperatorSubTree.java @@ -42,6 +42,7 @@ import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression; import org.apache.hyracks.algebricks.core.algebra.metadata.IDataSource; 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.AbstractScanOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractUnnestOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.DataSourceScanOperator; @@ -85,36 +86,50 @@ reset(); rootRef = subTreeOpRef; root = subTreeOpRef.getValue(); + boolean passedSource = false; + boolean result = false; + Mutable searchOpRef = subTreeOpRef; // Examine the op's children to match the expected patterns. - AbstractLogicalOperator subTreeOp = (AbstractLogicalOperator) subTreeOpRef.getValue(); + AbstractLogicalOperator subTreeOp = (AbstractLogicalOperator) searchOpRef.getValue(); do { // Skip select operator. if (subTreeOp.getOperatorTag() == LogicalOperatorTag.SELECT) { - subTreeOpRef = subTreeOp.getInputs().get(0); - subTreeOp = (AbstractLogicalOperator) subTreeOpRef.getValue(); + searchOpRef = subTreeOp.getInputs().get(0); + subTreeOp = (AbstractLogicalOperator) searchOpRef.getValue(); } // Check primary-index pattern. if (subTreeOp.getOperatorTag() != LogicalOperatorTag.ASSIGN && subTreeOp.getOperatorTag() != LogicalOperatorTag.UNNEST) { // Pattern may still match if we are looking for primary index matches as well. - return initializeDataSource(subTreeOpRef); + result = initializeDataSource(searchOpRef); + passedSource = true; + if (!subTreeOp.getInputs().isEmpty()) { + searchOpRef = subTreeOp.getInputs().get(0); + subTreeOp = (AbstractLogicalOperator) searchOpRef.getValue(); + } } // Match (assign | unnest)+. - while ((subTreeOp.getOperatorTag() == LogicalOperatorTag.ASSIGN - || subTreeOp.getOperatorTag() == LogicalOperatorTag.UNNEST)) { - if (!OperatorPropertiesUtil.isMovable(subTreeOp)) { + while (subTreeOp.getOperatorTag() == LogicalOperatorTag.ASSIGN + || subTreeOp.getOperatorTag() == LogicalOperatorTag.UNNEST) { + if (!passedSource && !OperatorPropertiesUtil.isMovable(subTreeOp)) { return false; - } else { - getAssignsAndUnnestsRefs().add(subTreeOpRef); - getAssignsAndUnnests().add(subTreeOp); } - subTreeOpRef = subTreeOp.getInputs().get(0); - subTreeOp = (AbstractLogicalOperator) subTreeOpRef.getValue(); + if (subTreeOp.getExecutionMode() != ExecutionMode.UNPARTITIONED) { + //The unpartitioned ops should stay below the search + assignsAndUnnestsRefs.add(searchOpRef); + } + assignsAndUnnests.add(subTreeOp); + + searchOpRef = subTreeOp.getInputs().get(0); + subTreeOp = (AbstractLogicalOperator) searchOpRef.getValue(); + } + if (passedSource) { + return result; } } while (subTreeOp.getOperatorTag() == LogicalOperatorTag.SELECT); // Match data source (datasource scan or primary index search). - return initializeDataSource(subTreeOpRef); + return initializeDataSource(searchOpRef); } private boolean initializeDataSource(Mutable subTreeOpRef) { @@ -395,8 +410,8 @@ case DATASOURCE_SCAN: case EXTERNAL_SCAN: case PRIMARY_INDEX_LOOKUP: - AbstractScanOperator scanOp = - (AbstractScanOperator) getIxJoinOuterAdditionalDataSourceRefs().get(idx).getValue(); + AbstractScanOperator scanOp = (AbstractScanOperator) getIxJoinOuterAdditionalDataSourceRefs() + .get(idx).getValue(); return scanOp.getVariables(); case COLLECTION_SCAN: return new ArrayList<>(); diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java index c3c162e..eeaaa8d 100644 --- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java +++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/am/RTreeAccessMethod.java @@ -95,12 +95,14 @@ } @Override - public boolean applySelectPlanTransformation(Mutable selectRef, - OptimizableOperatorSubTree subTree, Index chosenIndex, AccessMethodAnalysisContext analysisCtx, - IOptimizationContext context) throws AlgebricksException { + public boolean applySelectPlanTransformation(List> afterSelectRefs, + Mutable selectRef, OptimizableOperatorSubTree subTree, Index chosenIndex, + AccessMethodAnalysisContext analysisCtx, IOptimizationContext context) throws AlgebricksException { // TODO: We can probably do something smarter here based on selectivity or MBR area. ILogicalOperator primaryIndexUnnestOp = createSecondaryToPrimaryPlan(subTree, null, chosenIndex, analysisCtx, - false, false, false, context); + AccessMethodUtils.retainInputs(subTree.getDataSourceVariables(), subTree.getDataSourceRef().getValue(), + afterSelectRefs), + false, false, context); if (primaryIndexUnnestOp == null) { return false; } diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-no-index.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-no-index.aql new file mode 100644 index 0000000..bd3f019 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-no-index.aql @@ -0,0 +1,41 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + + /* + * Description : Time should be unpartitioned and Broadcast to nodes + * Expected Result : Success + * Date : 20th Oct 2016 + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + userId: int, + stamp: datetime +} + +create dataset Users(userLocation) +primary key userId; + +let $time := current-datetime() +for $result in dataset Users +where $result.stamp = $time +return $time \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index-return-time.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index-return-time.aql new file mode 100644 index 0000000..34c4e30 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index-return-time.aql @@ -0,0 +1,45 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +/* + * Description : Time should be unpartitioned + * : but used by index and returned + * Expected Result : Success + * Date : 20th Oct 2016 + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + userId: int, + stamp: datetime +} + +create dataset Users(userLocation) +primary key stamp; + +let $time := current-datetime() +for $result in dataset Users +where $result.stamp = $time +return { + "date":$time, + "result":$result +} \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index.aql new file mode 100644 index 0000000..010ad6f --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/global-datetime-use-index.aql @@ -0,0 +1,42 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + + /* + * Description : Time should be unpartitioned + * : but used by index + * Expected Result : Success + * Date : 20th Oct 2016 + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + userId: int, + stamp: datetime +} + +create dataset Users(userLocation) +primary key stamp; + +let $time := current-datetime() +for $result in dataset Users +where $result.stamp = $time +return $result \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/keep-datetime-local.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/keep-datetime-local.aql new file mode 100644 index 0000000..bb0ffc1 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/keep-datetime-local.aql @@ -0,0 +1,74 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +/* + * Description : Time should remain partitoned + * : and be returned + * Expected Result : Success + * Date : 20th Oct 2016 + */ + +drop dataverse test if exists; +create dataverse test; +use dataverse test; + +create type TwitterUserType as closed { + screen-name: string, + lang: string, + friends-count: int32, + statuses-count: int32, + name: string, + followers-count: int32 +} + +create type TweetMessageType as closed { + tweetid: int64, + user: TwitterUserType, + sender-location: point, + send-time: datetime, + referred-topics: {{ string }}, + message-text: string, + countA: int32, + countB: int32 +} + +create dataset TweetMessages(TweetMessageType) +primary key tweetid; + +create index twmSndLocIx on TweetMessages(sender-location) type rtree; +create index msgCountAIx on TweetMessages(countA) type btree; +create index msgCountBIx on TweetMessages(countB) type btree; +create index msgTextIx on TweetMessages(message-text) type keyword; + +write output to asterix_nc1:"rttest/btree-index-join_leftouterjoin-probe-pidx-with-join-btree-sidx_01.adm"; + +for $t1 in dataset('TweetMessages') +let $time := current-datetime() +where $t1.tweetid < int64("10") +order by $t1.tweetid +return { +"time": $time, +"tweetid1": $t1.tweetid, +"count1":$t1.countA, +"t2info": for $t2 in dataset('TweetMessages') + where $t1.countA /* +indexnl */= $t2.countB + order by $t2.tweetid + return {"tweetid2": $t2.tweetid, + "count2":$t2.countB} +}; \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-ignore-index.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-ignore-index.aql new file mode 100644 index 0000000..1077257 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-ignore-index.aql @@ -0,0 +1,42 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + + /* + * Description : Time should remain partitoned + * : and therefore unusable by index + * Expected Result : Success + * Date : 20th Oct 2016 + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + userId: int, + stamp: datetime +} + +create dataset Users(userLocation) +primary key stamp; + +for $result in dataset Users +let $time := current-datetime() +where $result.stamp = $time +return $result \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-no-index.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-no-index.aql new file mode 100644 index 0000000..5083aaa --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/local-datetime-no-index.aql @@ -0,0 +1,42 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +/* + * Description : Time should remain partitoned + * : and be returned + * Expected Result : Success + * Date : 20th Oct 2016 + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + userId: int, + stamp: datetime +} + +create dataset Users(userLocation) +primary key userId; + +for $result in dataset Users +let $time := current-datetime() +where $result.stamp = $time +return $time \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/maintain-nonpure-location-in-join-cannot-index.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/maintain-nonpure-location-in-join-cannot-index.aql new file mode 100644 index 0000000..4a7b695 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/maintain-nonpure-location-in-join-cannot-index.aql @@ -0,0 +1,48 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + + /* + * Description : Time should remain partitoned + * Expected Result : Success + * Date : 20th Oct 2016 + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + id: int, + stamp: time +} + +create dataset Users1(userLocation) +primary key stamp; + +create dataset Users2(userLocation) +primary key stamp; + +for $x in dataset Users1 +let $time := current-time() +for $y in dataset Users2 +where $y.stamp > $time-time("123045678+0800") +return { + "x_id": $x.id, + "y_id": $y.id +} \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/query-ASTERIXDB-1608.aql b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/query-ASTERIXDB-1608.aql new file mode 100644 index 0000000..04620bd --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/nonpure/query-ASTERIXDB-1608.aql @@ -0,0 +1,33 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +/* + * Description : Check fix for ASTERIXDB-1608 + * Expected Result : Success + * Date : 20th Oct 2016 + */ + + +drop dataverse test if exists; +create dataverse test; + +for $x in range(1, 3) +for $y in range(1, 3) +return +{"id": create-uuid(), "x": $x} \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/btree-index/btree-datetime-02.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/btree-index/btree-datetime-02.plan index a9e223a..a461461 100644 --- a/asterixdb/asterix-app/src/test/resources/optimizerts/results/btree-index/btree-datetime-02.plan +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/btree-index/btree-datetime-02.plan @@ -1,8 +1,9 @@ -- DISTRIBUTE_RESULT |PARTITIONED| -- ONE_TO_ONE_EXCHANGE |PARTITIONED| -- STREAM_PROJECT |PARTITIONED| - -- ONE_TO_ONE_EXCHANGE |PARTITIONED| - -- BTREE_SEARCH |PARTITIONED| + -- STREAM_SELECT |PARTITIONED| + -- ASSIGN |PARTITIONED| -- ONE_TO_ONE_EXCHANGE |PARTITIONED| - -- ASSIGN |PARTITIONED| - -- EMPTY_TUPLE_SOURCE |PARTITIONED| \ No newline at end of file + -- DATASOURCE_SCAN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- EMPTY_TUPLE_SOURCE |PARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-no-index.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-no-index.plan new file mode 100644 index 0000000..82d3768 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-no-index.plan @@ -0,0 +1,10 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- STREAM_SELECT |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- DATASOURCE_SCAN |PARTITIONED| + -- BROADCAST_EXCHANGE |PARTITIONED| + -- ASSIGN |UNPARTITIONED| + -- EMPTY_TUPLE_SOURCE |UNPARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index-return-time.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index-return-time.plan new file mode 100644 index 0000000..c6fd83e --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index-return-time.plan @@ -0,0 +1,12 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- BTREE_SEARCH |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STABLE_SORT [$$0(ASC)] |PARTITIONED| + -- HASH_PARTITION_EXCHANGE [$$0] |PARTITIONED| + -- ASSIGN |UNPARTITIONED| + -- EMPTY_TUPLE_SOURCE |UNPARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index.plan new file mode 100644 index 0000000..972e56a --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/global-datetime-use-index.plan @@ -0,0 +1,10 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- BTREE_SEARCH |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STABLE_SORT [$$0(ASC)] |PARTITIONED| + -- HASH_PARTITION_EXCHANGE [$$0] |PARTITIONED| + -- ASSIGN |UNPARTITIONED| + -- EMPTY_TUPLE_SOURCE |UNPARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/keep-datetime-local.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/keep-datetime-local.plan new file mode 100644 index 0000000..5246d83 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/keep-datetime-local.plan @@ -0,0 +1,36 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- SORT_MERGE_EXCHANGE [$$22(ASC) ] |PARTITIONED| + -- STABLE_SORT [$$22(ASC)] |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- PRE_CLUSTERED_GROUP_BY[$$30] |PARTITIONED| + { + -- AGGREGATE |LOCAL| + -- STREAM_SELECT |LOCAL| + -- NESTED_TUPLE_SOURCE |LOCAL| + } + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STABLE_SORT [$$30(ASC), $$23(ASC)] |PARTITIONED| + -- HASH_PARTITION_EXCHANGE [$$30] |PARTITIONED| + -- HYBRID_HASH_JOIN [$$25][$$24] |PARTITIONED| + -- HASH_PARTITION_EXCHANGE [$$25] |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- STREAM_SELECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- DATASOURCE_SCAN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- EMPTY_TUPLE_SOURCE |PARTITIONED| + -- HASH_PARTITION_EXCHANGE [$$24] |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- DATASOURCE_SCAN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- EMPTY_TUPLE_SOURCE |PARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-ignore-index.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-ignore-index.plan new file mode 100644 index 0000000..a461461 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-ignore-index.plan @@ -0,0 +1,9 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- STREAM_SELECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- DATASOURCE_SCAN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- EMPTY_TUPLE_SOURCE |PARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-no-index.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-no-index.plan new file mode 100644 index 0000000..2d604a9 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/local-datetime-no-index.plan @@ -0,0 +1,10 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- STREAM_SELECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- DATASOURCE_SCAN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- EMPTY_TUPLE_SOURCE |PARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/maintain-nonpure-location-in-join-cannot-index.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/maintain-nonpure-location-in-join-cannot-index.plan new file mode 100644 index 0000000..fe2675b --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/maintain-nonpure-location-in-join-cannot-index.plan @@ -0,0 +1,25 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- NESTED_LOOP |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- DATASOURCE_SCAN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- EMPTY_TUPLE_SOURCE |PARTITIONED| + -- BROADCAST_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- DATASOURCE_SCAN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- EMPTY_TUPLE_SOURCE |PARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/query-ASTERIXDB-1608.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/query-ASTERIXDB-1608.plan new file mode 100644 index 0000000..1864e29 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/nonpure/query-ASTERIXDB-1608.plan @@ -0,0 +1,13 @@ +-- DISTRIBUTE_RESULT |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |PARTITIONED| + -- ASSIGN |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |PARTITIONED| + -- NESTED_LOOP |PARTITIONED| + -- ONE_TO_ONE_EXCHANGE |UNPARTITIONED| + -- UNNEST |UNPARTITIONED| + -- EMPTY_TUPLE_SOURCE |UNPARTITIONED| + -- BROADCAST_EXCHANGE |PARTITIONED| + -- STREAM_PROJECT |UNPARTITIONED| + -- UNNEST |UNPARTITIONED| + -- EMPTY_TUPLE_SOURCE |UNPARTITIONED| \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.1.ddl.aql b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.1.ddl.aql new file mode 100644 index 0000000..99ad6a1 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.1.ddl.aql @@ -0,0 +1,30 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + userId: int, + stamp: datetime +} + +create dataset Users(userLocation) +primary key stamp; \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.2.update.aql b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.2.update.aql new file mode 100644 index 0000000..6c18409 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.2.update.aql @@ -0,0 +1,26 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +use dataverse channels; + +insert into dataset Users( +[{"userId":1,"stamp":current-datetime() - day-time-duration("PT10M")}, +{"userId":2,"stamp":current-datetime() - day-time-duration("PT8M")}, +{"userId":3,"stamp":current-datetime() + day-time-duration("PT10M")}] +); diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.3.query.aql b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.3.query.aql new file mode 100644 index 0000000..3885444 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/global-datetime-use-index/global-datetime-use-index.3.query.aql @@ -0,0 +1,26 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +use dataverse channels; + +let $time := current-datetime() +for $result in dataset Users +where $result.stamp < $time +order by $result.userId +return $result.userId; \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.ddl.aql b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.ddl.aql new file mode 100644 index 0000000..99ad6a1 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.ddl.aql @@ -0,0 +1,30 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +drop dataverse channels if exists; +create dataverse channels; +use dataverse channels; + +create type userLocation as { + userId: int, + stamp: datetime +} + +create dataset Users(userLocation) +primary key stamp; \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.2.update.aql b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.2.update.aql new file mode 100644 index 0000000..541a203 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.2.update.aql @@ -0,0 +1,26 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +use dataverse channels; + +insert into dataset Users( +[{"userId":1,"stamp":current-datetime() - day-time-duration("PT10M")}, +{"userId":2,"stamp":current-datetime() + day-time-duration("PT8M")}, +{"userId":3,"stamp":current-datetime() + day-time-duration("PT10M")}] +); diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.3.query.aql b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.3.query.aql new file mode 100644 index 0000000..ab2ac11 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.3.query.aql @@ -0,0 +1,26 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +use dataverse channels; + +for $result in dataset Users +let $time := current-datetime() +where $result.stamp > $time +order by $result.userId +return $result.userId; \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/global-datetime-use-index/global-datetime-use-index.1.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/global-datetime-use-index/global-datetime-use-index.1.adm new file mode 100644 index 0000000..7a754f4 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/global-datetime-use-index/global-datetime-use-index.1.adm @@ -0,0 +1,2 @@ +1 +2 \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.adm new file mode 100644 index 0000000..1234e84 --- /dev/null +++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/nonpure/local-datetime-ignore-index/local-datetime-ignore-index.1.adm @@ -0,0 +1,2 @@ +2 +3 \ No newline at end of file diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml b/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml index 5664c72..cbf715d 100644 --- a/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml +++ b/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml @@ -4139,8 +4139,20 @@ - - query-ASTERIXDB-1025 + + global-datetime-use-index + + + + + + + local-datetime-ignore-index + + + + + global-datetime-use-index diff --git a/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/base/PhysicalOperatorTag.java b/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/base/PhysicalOperatorTag.java index 1d20e08..4ceeb5c 100644 --- a/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/base/PhysicalOperatorTag.java +++ b/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/base/PhysicalOperatorTag.java @@ -27,7 +27,7 @@ DATASOURCE_SCAN, DISTRIBUTE_RESULT, EMPTY_TUPLE_SOURCE, - EXTENSION_OPERATOR, + DELEGATE_OPERATOR, EXTERNAL_GROUP_BY, EXTERNAL_LOOKUP, HASH_GROUP_BY, diff --git a/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/util/OperatorPropertiesUtil.java b/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/util/OperatorPropertiesUtil.java index e2643dd..8117921 100644 --- a/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/util/OperatorPropertiesUtil.java +++ b/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/util/OperatorPropertiesUtil.java @@ -37,6 +37,7 @@ import org.apache.hyracks.algebricks.core.algebra.functions.AlgebricksBuiltinFunctions; 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.AssignOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.CardinalityInferenceVisitor; import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities; @@ -287,13 +288,37 @@ public static boolean isMovable(ILogicalOperator op) { Object annotation = op.getAnnotations().get(MOVABLE); if (annotation == null) { - // By default, it is movable. + // Can't move nonPures! + if (op.getOperatorTag() == LogicalOperatorTag.ASSIGN) { + AssignOperator assign = (AssignOperator) op; + for (Mutable expr : assign.getExpressions()) { + if (containsNonpureCall(expr.getValue())) { + return false; + } + } + } return true; } Boolean movable = (Boolean) annotation; return movable; } + private static boolean containsNonpureCall(ILogicalExpression expr) { + if (expr.getExpressionTag() == LogicalExpressionTag.FUNCTION_CALL) { + AbstractFunctionCallExpression fExpr = (AbstractFunctionCallExpression) expr; + if (!fExpr.getFunctionInfo().isFunctional()) { + return true; + } + for (Mutable subExpr : fExpr.getArguments()) { + if (containsNonpureCall(subExpr.getValue())) { + return true; + } + } + + } + return false; + } + /** * Mark an operator to be either movable or not. * diff --git a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateAssignsRule.java b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateAssignsRule.java index 8f6e9a1..bbb6cbd 100644 --- a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateAssignsRule.java +++ b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateAssignsRule.java @@ -22,7 +22,6 @@ import java.util.List; import org.apache.commons.lang3.mutable.Mutable; - import org.apache.hyracks.algebricks.common.exceptions.AlgebricksException; import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator; import org.apache.hyracks.algebricks.core.algebra.base.IOptimizationContext; @@ -31,6 +30,7 @@ import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.AssignOperator; import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities; +import org.apache.hyracks.algebricks.core.algebra.util.OperatorPropertiesUtil; import org.apache.hyracks.algebricks.core.rewriter.base.IAlgebraicRewriteRule; public class ConsolidateAssignsRule implements IAlgebraicRewriteRule { @@ -53,6 +53,11 @@ if (op2.getOperatorTag() != LogicalOperatorTag.ASSIGN) { return false; } + + if (!OperatorPropertiesUtil.isMovable(op) || !OperatorPropertiesUtil.isMovable(op2)) { + return false; + } + AssignOperator assign2 = (AssignOperator) op2; HashSet used1 = new HashSet(); diff --git a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/EnforceStructuralPropertiesRule.java b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/EnforceStructuralPropertiesRule.java index 0b3a57e..a1730ac 100644 --- a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/EnforceStructuralPropertiesRule.java +++ b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/EnforceStructuralPropertiesRule.java @@ -68,8 +68,10 @@ import org.apache.hyracks.algebricks.core.algebra.prettyprint.PlanPrettyPrinter; import org.apache.hyracks.algebricks.core.algebra.properties.FunctionalDependency; import org.apache.hyracks.algebricks.core.algebra.properties.ILocalStructuralProperty; +import org.apache.hyracks.algebricks.core.algebra.properties.ILocalStructuralProperty.PropertyType; import org.apache.hyracks.algebricks.core.algebra.properties.INodeDomain; import org.apache.hyracks.algebricks.core.algebra.properties.IPartitioningProperty; +import org.apache.hyracks.algebricks.core.algebra.properties.IPartitioningProperty.PartitioningType; import org.apache.hyracks.algebricks.core.algebra.properties.IPartitioningRequirementsCoordinator; import org.apache.hyracks.algebricks.core.algebra.properties.IPhysicalPropertiesVector; import org.apache.hyracks.algebricks.core.algebra.properties.LocalGroupingProperty; @@ -81,8 +83,6 @@ import org.apache.hyracks.algebricks.core.algebra.properties.RandomPartitioningProperty; import org.apache.hyracks.algebricks.core.algebra.properties.StructuralPropertiesVector; import org.apache.hyracks.algebricks.core.algebra.properties.UnorderedPartitionedProperty; -import org.apache.hyracks.algebricks.core.algebra.properties.ILocalStructuralProperty.PropertyType; -import org.apache.hyracks.algebricks.core.algebra.properties.IPartitioningProperty.PartitioningType; import org.apache.hyracks.algebricks.core.algebra.util.OperatorManipulationUtil; import org.apache.hyracks.algebricks.core.algebra.util.OperatorPropertiesUtil; import org.apache.hyracks.algebricks.core.config.AlgebricksConfig; diff --git a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ExtractCommonExpressionsRule.java b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ExtractCommonExpressionsRule.java index 11ff4be..60275dd 100644 --- a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ExtractCommonExpressionsRule.java +++ b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ExtractCommonExpressionsRule.java @@ -241,19 +241,22 @@ if (exprEqClass != null) { // Replace common subexpression with existing variable. if (exprEqClass.variableIsSet()) { - Set liveVars = new HashSet(); - List usedVars = new ArrayList(); - VariableUtilities.getLiveVariables(op, liveVars); - VariableUtilities.getUsedVariables(op, usedVars); - // Check if the replacing variable is live at this op. - // However, if the op is already using variables that are not live, then a replacement may enable fixing the plan. - // This behavior is necessary to, e.g., properly deal with distinct by. - // Also just replace the expr if we are replacing common exprs from within the same operator. - if (liveVars.contains(exprEqClass.getVariable()) || !liveVars.containsAll(usedVars) - || op == exprEqClass.getFirstOperator()) { - exprRef.setValue(new VariableReferenceExpression(exprEqClass.getVariable())); - // Do not descend into children since this expr has been completely replaced. - return true; + if (expr.isFunctional()) { + Set liveVars = new HashSet<>(); + List usedVars = new ArrayList<>(); + VariableUtilities.getLiveVariables(op, liveVars); + VariableUtilities.getUsedVariables(op, usedVars); + // Check if the replacing variable is live at this op. + // However, if the op is already using variables that are not live, + // then a replacement may enable fixing the plan. + // This behavior is necessary to, e.g., properly deal with distinct by. + // Also just replace the expr if we are replacing common exprs from within the same operator. + if (liveVars.contains(exprEqClass.getVariable()) || !liveVars.containsAll(usedVars) + || op == exprEqClass.getFirstOperator()) { + exprRef.setValue(new VariableReferenceExpression(exprEqClass.getVariable())); + // Do not descend into children since this expr has been completely replaced. + return true; + } } } else { if (expr.isFunctional() && assignCommonExpression(exprEqClass, expr)) { diff --git a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/InlineVariablesRule.java b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/InlineVariablesRule.java index b4ade4c..cd42407 100644 --- a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/InlineVariablesRule.java +++ b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/InlineVariablesRule.java @@ -118,25 +118,6 @@ return false; } - /** - * An expression will be constant at runtime if it has: - * 1. A type - * 2. No free variables - * - * @param op - * @param funcExpr - * @param context - * @return whether a function is constant - * @throws AlgebricksException - */ - public static boolean functionIsConstantAtRuntime(AbstractFunctionCallExpression funcExpr) - throws AlgebricksException { - //make sure that there are no variables in the expression - Set usedVariables = new HashSet<>(); - funcExpr.getUsedVariables(usedVariables); - return usedVariables.isEmpty(); - } - protected boolean inlineVariables(Mutable opRef, IOptimizationContext context) throws AlgebricksException { AbstractLogicalOperator op = (AbstractLogicalOperator) opRef.getValue(); @@ -151,8 +132,7 @@ // Ignore functions that are either in the doNotInline set or are non-functional if (expr.getExpressionTag() == LogicalExpressionTag.FUNCTION_CALL) { AbstractFunctionCallExpression funcExpr = (AbstractFunctionCallExpression) expr; - if (doNotInlineFuncs.contains(funcExpr.getFunctionIdentifier()) || (!funcExpr.isFunctional() - && !InlineVariablesRule.functionIsConstantAtRuntime(funcExpr))) { + if (doNotInlineFuncs.contains(funcExpr.getFunctionIdentifier()) || !funcExpr.isFunctional()) { continue; } } diff --git a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/PushMapOperatorDownThroughProductRule.java b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/PushMapOperatorDownThroughProductRule.java index f956d73..f71af5a 100644 --- a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/PushMapOperatorDownThroughProductRule.java +++ b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/PushMapOperatorDownThroughProductRule.java @@ -50,6 +50,11 @@ if (!op1.isMap() || op1.getOperatorTag() == LogicalOperatorTag.LIMIT) { return false; } + + if (!OperatorPropertiesUtil.isMovable(op1)) { + return false; + }; + Mutable op2Ref = op1.getInputs().get(0); AbstractLogicalOperator op2 = (AbstractLogicalOperator) op2Ref.getValue(); if (op2.getOperatorTag() != LogicalOperatorTag.INNERJOIN) { diff --git a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/SetExecutionModeRule.java b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/SetExecutionModeRule.java index ed85001..32a3bac 100644 --- a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/SetExecutionModeRule.java +++ b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/SetExecutionModeRule.java @@ -19,8 +19,7 @@ package org.apache.hyracks.algebricks.rewriter.rules; import org.apache.commons.lang3.mutable.Mutable; - -import org.apache.hyracks.algebricks.common.exceptions.NotImplementedException; +import org.apache.hyracks.algebricks.common.exceptions.AlgebricksException; import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator; import org.apache.hyracks.algebricks.core.algebra.base.IOptimizationContext; import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractLogicalOperator; @@ -38,7 +37,8 @@ public class SetExecutionModeRule implements IAlgebraicRewriteRule { @Override - public boolean rewritePost(Mutable opRef, IOptimizationContext context) { + public boolean rewritePost(Mutable opRef, IOptimizationContext context) + throws AlgebricksException { AbstractLogicalOperator op = (AbstractLogicalOperator) opRef.getValue(); return OperatorManipulationUtil.setOperatorMode(op); } -- To view, visit https://asterix-gerrit.ics.uci.edu/1057 To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings Gerrit-MessageType: merged Gerrit-Change-Id: I2dec322b30835625430c06acd7626d902bada137 Gerrit-PatchSet: 26 Gerrit-Project: asterixdb Gerrit-Branch: master Gerrit-Owner: Steven Jacobs Gerrit-Reviewer: Ian Maxon Gerrit-Reviewer: Jenkins Gerrit-Reviewer: Michael Blow Gerrit-Reviewer: Preston Carman Gerrit-Reviewer: Steven Jacobs Gerrit-Reviewer: Taewoo Kim Gerrit-Reviewer: Till Westmann Gerrit-Reviewer: Yingyi Bu