You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@asterixdb.apache.org by al...@apache.org on 2023/02/27 20:29:04 UTC

[asterixdb] 04/16: [ASTERIXDB-3113] Allow using hash-join with redundant variables

This is an automated email from the ASF dual-hosted git repository.

alsuliman pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/asterixdb.git

commit d8e460e25cf578e5d3dd837e77188b56f75648c1
Author: Wail Alkowaileet <wa...@couchbase.com>
AuthorDate: Tue Feb 14 16:54:24 2023 -0800

    [ASTERIXDB-3113] Allow using hash-join with redundant variables
    
    - user model changes: no
    - storage format changes: no
    - interface changes: no
    
    Details:
    This patch introduces several optimizations:
    - Remove redundant equality checks in join conditions
    - Allow one attribute to be joined with two attributes
      using hash-join
    
    Change-Id: I3226457e4c9352fae0433cfeb39a158ec6562955
    Reviewed-on: https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/17377
    Tested-by: Jenkins <je...@fulliautomatix.ics.uci.edu>
    Integration-Tests: Jenkins <je...@fulliautomatix.ics.uci.edu>
    Reviewed-by: Wail Alkowaileet <wa...@gmail.com>
    Reviewed-by: Ali Alsuliman <al...@gmail.com>
---
 .../asterix/optimizer/base/RuleCollections.java    |    6 +
 .../rules/AbstractConditionExpressionRule.java     |   86 ++
 .../rules/ExtractRedundantVariablesInJoinRule.java |  187 +++
 ...neAndRemoveRedundantBooleanExpressionsRule.java |  107 ++
 ...emoveRedundantBooleanExpressionsInJoinRule.java |  155 ++
 .../optimizer/rules/cbo/EnumerateJoinsRule.java    |   91 +-
 .../hash-join-with-redundant-variable.1.sqlpp      |   63 +
 .../hash-join-with-redundant-variable.2.sqlpp      |   28 +
 .../hash-join-with-redundant-variable.3.sqlpp      |   30 +
 .../hash-join-with-redundant-variable.4.sqlpp      |   26 +
 .../hash-join-with-redundant-variable.1.plan       |   74 +
 .../hash-join-with-redundant-variable.2.plan       |   35 +
 .../hash-join-with-redundant-variable.3.plan       |   35 +
 .../hash-join-with-redundant-variable.4.plan       |   24 +
 .../hash-join-with-redundant-variable.1.plan       |   74 +
 .../hash-join-with-redundant-variable.01.ddl.sqlpp |  134 ++
 ...sh-join-with-redundant-variable.02.update.sqlpp |   36 +
 ...ash-join-with-redundant-variable.03.query.sqlpp |   32 +
 ...ash-join-with-redundant-variable.04.query.sqlpp |   32 +
 ...ash-join-with-redundant-variable.05.query.sqlpp |   39 +
 ...ash-join-with-redundant-variable.06.query.sqlpp |   39 +
 ...ash-join-with-redundant-variable.07.query.sqlpp |   39 +
 ...ash-join-with-redundant-variable.08.query.sqlpp |   39 +
 ...ash-join-with-redundant-variable.09.query.sqlpp |   35 +
 ...ash-join-with-redundant-variable.10.query.sqlpp |   35 +
 ...ash-join-with-redundant-variable.11.query.sqlpp |   35 +
 ...ash-join-with-redundant-variable.12.query.sqlpp |   35 +
 ...ash-join-with-redundant-variable.13.query.sqlpp |   39 +
 ...ash-join-with-redundant-variable.14.query.sqlpp |   39 +
 ...ash-join-with-redundant-variable.15.query.sqlpp |   40 +
 ...ash-join-with-redundant-variable.16.query.sqlpp |   40 +
 .../hash-join-with-redundant-variable.03.adm       |   58 +
 .../hash-join-with-redundant-variable.04.plan      |   64 +
 .../hash-join-with-redundant-variable.05.adm       |    3 +
 .../hash-join-with-redundant-variable.06.plan      |   50 +
 .../hash-join-with-redundant-variable.07.adm       | 1500 ++++++++++++++++++++
 .../hash-join-with-redundant-variable.08.plan      |   50 +
 .../hash-join-with-redundant-variable.09.adm       |   58 +
 .../hash-join-with-redundant-variable.10.plan      |   64 +
 .../hash-join-with-redundant-variable.11.adm       |   58 +
 .../hash-join-with-redundant-variable.12.plan      |   64 +
 .../hash-join-with-redundant-variable.13.adm       |   58 +
 .../hash-join-with-redundant-variable.14.plan      |   66 +
 .../hash-join-with-redundant-variable.15.adm       |   58 +
 .../hash-join-with-redundant-variable.16.plan      |   66 +
 .../test/resources/runtimets/testsuite_sqlpp.xml   |    5 +
 .../asterix/lang/common/util/FunctionUtil.java     |   69 +
 .../lang/expression/CommutativeEqualsTest.java     |   87 ++
 .../IPartitioningRequirementsCoordinator.java      |   26 +-
 49 files changed, 4015 insertions(+), 98 deletions(-)

diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/base/RuleCollections.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/base/RuleCollections.java
index 473c8ecb12..1c3ef03a11 100644
--- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/base/RuleCollections.java
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/base/RuleCollections.java
@@ -44,6 +44,7 @@ import org.apache.asterix.optimizer.rules.DisjunctivePredicateToJoinRule;
 import org.apache.asterix.optimizer.rules.ExtractBatchableExternalFunctionCallsRule;
 import org.apache.asterix.optimizer.rules.ExtractDistinctByExpressionsRule;
 import org.apache.asterix.optimizer.rules.ExtractOrderExpressionsRule;
+import org.apache.asterix.optimizer.rules.ExtractRedundantVariablesInJoinRule;
 import org.apache.asterix.optimizer.rules.ExtractWindowExpressionsRule;
 import org.apache.asterix.optimizer.rules.FeedScanCollectionToUnnest;
 import org.apache.asterix.optimizer.rules.FilterRefineSpatialJoinRuleForSTDistanceFunction;
@@ -82,6 +83,7 @@ import org.apache.asterix.optimizer.rules.PushValueAccessToExternalDataScanRule;
 import org.apache.asterix.optimizer.rules.RemoveDuplicateFieldsRule;
 import org.apache.asterix.optimizer.rules.RemoveLeftOuterUnnestForLeftOuterJoinRule;
 import org.apache.asterix.optimizer.rules.RemoveOrReplaceDefaultNullCastRule;
+import org.apache.asterix.optimizer.rules.RemoveRedundantBooleanExpressionsInJoinRule;
 import org.apache.asterix.optimizer.rules.RemoveRedundantListifyRule;
 import org.apache.asterix.optimizer.rules.RemoveRedundantSelectRule;
 import org.apache.asterix.optimizer.rules.RemoveSortInFeedIngestionRule;
@@ -350,6 +352,10 @@ public final class RuleCollections {
         planCleanupRules.add(new RemoveUnknownCheckForKnownTypeExpressionRule());
         // relies on RemoveOrReplaceDefaultNullCastRule AND RemoveUnknownCheckForKnownTypeExpressionRule
         planCleanupRules.add(new RemoveRedundantSelectRule());
+        planCleanupRules.add(new RemoveRedundantBooleanExpressionsInJoinRule());
+        // RemoveRedundantBooleanExpressionsInJoinRule has to run first to probably eliminate the need for
+        // introducing an assign operator in ExtractSimilarVariablesInJoinRule
+        planCleanupRules.add(new ExtractRedundantVariablesInJoinRule());
 
         // Needs to invoke ByNameToByIndexFieldAccessRule as the last logical optimization rule because
         // some rules can push a FieldAccessByName to a place where the name it tries to access is in the closed part.
diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/AbstractConditionExpressionRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/AbstractConditionExpressionRule.java
new file mode 100644
index 0000000000..2dded4ac1b
--- /dev/null
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/AbstractConditionExpressionRule.java
@@ -0,0 +1,86 @@
+/*
+ * 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.
+ */
+package org.apache.asterix.optimizer.rules;
+
+import org.apache.commons.lang3.mutable.Mutable;
+import org.apache.hyracks.algebricks.common.exceptions.AlgebricksException;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator;
+import org.apache.hyracks.algebricks.core.algebra.base.IOptimizationContext;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalExpressionTag;
+import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression;
+import org.apache.hyracks.algebricks.core.algebra.functions.FunctionIdentifier;
+import org.apache.hyracks.algebricks.core.algebra.functions.IFunctionInfo;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractBinaryJoinOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator;
+import org.apache.hyracks.algebricks.core.rewriter.base.IAlgebraicRewriteRule;
+
+public abstract class AbstractConditionExpressionRule implements IAlgebraicRewriteRule {
+    private IOptimizationContext context;
+
+    @Override
+    public boolean rewritePost(Mutable<ILogicalOperator> opRef, IOptimizationContext context)
+            throws AlgebricksException {
+        final ILogicalOperator op = opRef.getValue();
+        final Mutable<ILogicalExpression> condRef;
+        switch (op.getOperatorTag()) {
+            case SELECT:
+                final SelectOperator select = (SelectOperator) op;
+                condRef = select.getCondition();
+                break;
+            case INNERJOIN:
+            case LEFTOUTERJOIN:
+                final AbstractBinaryJoinOperator join = (AbstractBinaryJoinOperator) op;
+                condRef = join.getCondition();
+                break;
+            default:
+                return false;
+        }
+
+        this.context = context;
+
+        boolean changed = transform(condRef);
+        if (changed) {
+            context.computeAndSetTypeEnvironmentForOperator(op);
+        }
+
+        return changed;
+    }
+
+    protected final AbstractFunctionCallExpression getFunctionExpression(ILogicalExpression expression) {
+        if (expression.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
+            return null;
+        }
+
+        return (AbstractFunctionCallExpression) expression;
+    }
+
+    protected final IFunctionInfo getFunctionInfo(FunctionIdentifier fid) {
+        return context.getMetadataProvider().lookupFunction(fid);
+    }
+
+    /**
+     * Transform condition expression
+     *
+     * @param condRef SELECT or join condition reference
+     * @return {@code <code>true</code>} condition has been modified
+     * {@code <code>false</code>} otherwise.
+     */
+    protected abstract boolean transform(Mutable<ILogicalExpression> condRef);
+}
diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/ExtractRedundantVariablesInJoinRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/ExtractRedundantVariablesInJoinRule.java
new file mode 100644
index 0000000000..294dd28571
--- /dev/null
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/ExtractRedundantVariablesInJoinRule.java
@@ -0,0 +1,187 @@
+/*
+ * 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.
+ */
+package org.apache.asterix.optimizer.rules;
+
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+
+import org.apache.commons.lang3.mutable.Mutable;
+import org.apache.commons.lang3.mutable.MutableObject;
+import org.apache.hyracks.algebricks.common.exceptions.AlgebricksException;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator;
+import org.apache.hyracks.algebricks.core.algebra.base.IOptimizationContext;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalExpressionTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalOperatorTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
+import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression;
+import org.apache.hyracks.algebricks.core.algebra.expressions.VariableReferenceExpression;
+import org.apache.hyracks.algebricks.core.algebra.functions.AlgebricksBuiltinFunctions;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractBinaryJoinOperator;
+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.rewriter.base.IAlgebraicRewriteRule;
+import org.apache.hyracks.api.exceptions.SourceLocation;
+
+public class ExtractRedundantVariablesInJoinRule implements IAlgebraicRewriteRule {
+    private final Map<LogicalVariable, List<Mutable<ILogicalExpression>>> variableToExpressionsMap = new HashMap<>();
+    private final Set<LogicalVariable> leftLiveVars = new HashSet<>();
+
+    @Override
+    public boolean rewritePost(Mutable<ILogicalOperator> opRef, IOptimizationContext context)
+            throws AlgebricksException {
+        ILogicalOperator op = opRef.getValue();
+        if (op.getOperatorTag() != LogicalOperatorTag.INNERJOIN
+                && op.getOperatorTag() != LogicalOperatorTag.LEFTOUTERJOIN) {
+            return false;
+        }
+
+        AbstractBinaryJoinOperator joinOp = (AbstractBinaryJoinOperator) op;
+        if (!ensureAndExtractVarAndExpr(joinOp.getCondition().getValue())) {
+            return false;
+        }
+
+        setLeftLiveVariables(joinOp);
+
+        List<LogicalVariable> leftAssignVars = new ArrayList<>();
+        List<Mutable<ILogicalExpression>> leftAssignExprs = new ArrayList<>();
+
+        List<LogicalVariable> rightAssignVars = new ArrayList<>();
+        List<Mutable<ILogicalExpression>> rightAssignExprs = new ArrayList<>();
+
+        for (Map.Entry<LogicalVariable, List<Mutable<ILogicalExpression>>> kv : variableToExpressionsMap.entrySet()) {
+            LogicalVariable repeatedVariable = kv.getKey();
+            List<Mutable<ILogicalExpression>> repeatedReferences = kv.getValue();
+
+            if (leftLiveVars.contains(repeatedVariable)) {
+                reassignRepeatedVariables(context, repeatedVariable, repeatedReferences, leftAssignVars,
+                        leftAssignExprs);
+            } else {
+                reassignRepeatedVariables(context, repeatedVariable, repeatedReferences, rightAssignVars,
+                        rightAssignExprs);
+            }
+        }
+
+        SourceLocation sourceLocation = joinOp.getSourceLocation();
+        if (!leftAssignVars.isEmpty()) {
+            createAndSetAssign(context, sourceLocation, joinOp.getInputs().get(0), leftAssignVars, leftAssignExprs);
+        }
+
+        if (!rightAssignVars.isEmpty()) {
+            createAndSetAssign(context, sourceLocation, joinOp.getInputs().get(1), rightAssignVars, rightAssignExprs);
+        }
+
+        context.computeAndSetTypeEnvironmentForOperator(joinOp);
+
+        return true;
+    }
+
+    private void createAndSetAssign(IOptimizationContext context, SourceLocation sourceLocation,
+            Mutable<ILogicalOperator> joinInputRef, List<LogicalVariable> assignVars,
+            List<Mutable<ILogicalExpression>> assignExprs) throws AlgebricksException {
+        AssignOperator assignOp = new AssignOperator(assignVars, assignExprs);
+        assignOp.setSourceLocation(sourceLocation);
+        assignOp.getInputs().add(new MutableObject<>(joinInputRef.getValue()));
+        joinInputRef.setValue(assignOp);
+        context.computeAndSetTypeEnvironmentForOperator(assignOp);
+    }
+
+    private void setLeftLiveVariables(AbstractBinaryJoinOperator op) throws AlgebricksException {
+        ILogicalOperator leftOp = op.getInputs().get(0).getValue();
+        leftLiveVars.clear();
+        VariableUtilities.getLiveVariables(leftOp, leftLiveVars);
+    }
+
+    private void reassignRepeatedVariables(IOptimizationContext context, LogicalVariable repeatedVariable,
+            List<Mutable<ILogicalExpression>> repeatedReferences, List<LogicalVariable> assignVars,
+            List<Mutable<ILogicalExpression>> assignExprs) {
+
+        // keep one of the repeated references and reassign the others
+        for (int i = 1; i < repeatedReferences.size(); i++) {
+            Mutable<ILogicalExpression> exprRef = repeatedReferences.get(i);
+            SourceLocation sourceLocation = exprRef.getValue().getSourceLocation();
+            LogicalVariable newVar = context.newVar();
+
+            exprRef.setValue(new VariableReferenceExpression(newVar, sourceLocation));
+
+            assignVars.add(newVar);
+            assignExprs.add(new MutableObject<>(new VariableReferenceExpression(repeatedVariable, sourceLocation)));
+
+            // Prevent inlining the variable
+            context.addNotToBeInlinedVar(newVar);
+        }
+    }
+
+    private boolean ensureAndExtractVarAndExpr(ILogicalExpression expr) {
+        if (expr.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
+            return false;
+        }
+
+        AbstractFunctionCallExpression funcExpr = (AbstractFunctionCallExpression) expr;
+        if (!AlgebricksBuiltinFunctions.AND.equals(funcExpr.getFunctionIdentifier())) {
+            return false;
+        }
+
+        variableToExpressionsMap.clear();
+        boolean containsRepeatedReferences = false;
+        for (Mutable<ILogicalExpression> argRef : funcExpr.getArguments()) {
+            ILogicalExpression arg = argRef.getValue();
+            if (arg.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
+                return false;
+            }
+
+            AbstractFunctionCallExpression argFuncExpr = (AbstractFunctionCallExpression) arg;
+            if (!AlgebricksBuiltinFunctions.EQ.equals(argFuncExpr.getFunctionIdentifier())) {
+                return false;
+            }
+
+            List<Mutable<ILogicalExpression>> eqArgs = argFuncExpr.getArguments();
+            Mutable<ILogicalExpression> leftRef = eqArgs.get(0);
+            Mutable<ILogicalExpression> rightRef = eqArgs.get(1);
+
+            ILogicalExpression left = leftRef.getValue();
+            ILogicalExpression right = rightRef.getValue();
+
+            LogicalVariable leftVar = VariableUtilities.getVariable(left);
+            LogicalVariable rightVar = VariableUtilities.getVariable(right);
+
+            // shouldn't be possible. But here for sanity check
+            if (leftVar == null || rightVar == null) {
+                return false;
+            }
+
+            List<Mutable<ILogicalExpression>> leftList =
+                    variableToExpressionsMap.computeIfAbsent(leftVar, k -> new ArrayList<>());
+            leftList.add(leftRef);
+
+            List<Mutable<ILogicalExpression>> rightList =
+                    variableToExpressionsMap.computeIfAbsent(rightVar, k -> new ArrayList<>());
+            rightList.add(rightRef);
+
+            containsRepeatedReferences |= leftList.size() > 1 || rightList.size() > 1;
+        }
+
+        // return true only if there's a repeated reference to a variable
+        return containsRepeatedReferences;
+    }
+}
diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/InlineAndRemoveRedundantBooleanExpressionsRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/InlineAndRemoveRedundantBooleanExpressionsRule.java
new file mode 100644
index 0000000000..7eebf1ccef
--- /dev/null
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/InlineAndRemoveRedundantBooleanExpressionsRule.java
@@ -0,0 +1,107 @@
+/*
+ * 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.
+ */
+package org.apache.asterix.optimizer.rules;
+
+import java.util.List;
+
+import org.apache.asterix.lang.common.util.FunctionUtil;
+import org.apache.commons.lang3.mutable.Mutable;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
+import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression;
+import org.apache.hyracks.algebricks.core.algebra.functions.AlgebricksBuiltinFunctions;
+import org.apache.hyracks.algebricks.core.algebra.functions.FunctionIdentifier;
+
+/**
+ * Inline and remove redundant boolean expressions
+ * <p>
+ * Inline Example:
+ * and(x, and(y, and(z, w))) -> and(x, y, z, w)
+ * <p>
+ * Remove redundant example:
+ * or(x, y, y) -> or(x, y)
+ * TODO(wyk) include this rule in {@link org.apache.asterix.optimizer.base.RuleCollections}
+ */
+public class InlineAndRemoveRedundantBooleanExpressionsRule extends AbstractConditionExpressionRule {
+
+    @Override
+    protected boolean transform(Mutable<ILogicalExpression> condRef) {
+        AbstractFunctionCallExpression function = getFunctionExpression(condRef.getValue());
+        if (function == null) {
+            return false;
+        }
+
+        boolean changed = false;
+        for (Mutable<ILogicalExpression> argRef : function.getArguments()) {
+            changed |= transform(argRef);
+        }
+
+        final FunctionIdentifier fid = function.getFunctionIdentifier();
+        if (AlgebricksBuiltinFunctions.AND.equals(fid) || AlgebricksBuiltinFunctions.OR.equals(fid)) {
+            changed |= inlineCondition(function);
+            changed |= removeRedundantExpressions(function.getArguments());
+
+            //Special case: disjuncts/conjuncts have been factored out into a single (non-disjunct/conjunct) expression
+            if (function.getArguments().size() == 1) {
+                final ILogicalExpression newCond = function.getArguments().get(0).getValue();
+                condRef.setValue(newCond);
+            }
+        }
+
+        return changed;
+    }
+
+    private boolean inlineCondition(AbstractFunctionCallExpression function) {
+        final FunctionIdentifier fid = function.getFunctionIdentifier();
+        final List<Mutable<ILogicalExpression>> args = function.getArguments();
+
+        int i = 0;
+        boolean changed = false;
+        while (i < args.size()) {
+            final AbstractFunctionCallExpression argFunction = getFunctionExpression(args.get(i).getValue());
+            if (argFunction != null && fid.equals(argFunction.getFunctionIdentifier())) {
+                args.remove(i);
+                args.addAll(i, argFunction.getArguments());
+                changed = true;
+            } else {
+                i++;
+            }
+        }
+
+        return changed;
+    }
+
+    private boolean removeRedundantExpressions(List<Mutable<ILogicalExpression>> exprs) {
+        final int originalSize = exprs.size();
+        int i = 0;
+        while (i < exprs.size()) {
+            int j = i + 1;
+            while (j < exprs.size()) {
+                if (FunctionUtil.commutativeEquals(exprs.get(i).getValue(), exprs.get(j).getValue())) {
+                    exprs.remove(j);
+                } else {
+                    j++;
+                }
+            }
+            i++;
+        }
+
+        return exprs.size() != originalSize;
+    }
+
+}
diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/RemoveRedundantBooleanExpressionsInJoinRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/RemoveRedundantBooleanExpressionsInJoinRule.java
new file mode 100644
index 0000000000..f77fc2c078
--- /dev/null
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/RemoveRedundantBooleanExpressionsInJoinRule.java
@@ -0,0 +1,155 @@
+/*
+ * 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.
+ */
+package org.apache.asterix.optimizer.rules;
+
+import java.util.HashMap;
+import java.util.Map;
+
+import org.apache.commons.lang3.mutable.Mutable;
+import org.apache.commons.lang3.mutable.MutableObject;
+import org.apache.hyracks.algebricks.common.exceptions.AlgebricksException;
+import org.apache.hyracks.algebricks.core.algebra.base.EquivalenceClass;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalOperator;
+import org.apache.hyracks.algebricks.core.algebra.base.IOptimizationContext;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalExpressionTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalOperatorTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
+import org.apache.hyracks.algebricks.core.algebra.expressions.AbstractFunctionCallExpression;
+import org.apache.hyracks.algebricks.core.algebra.expressions.VariableReferenceExpression;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.AbstractBinaryJoinOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.FDsAndEquivClassesVisitor;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.visitors.VariableUtilities;
+
+public class RemoveRedundantBooleanExpressionsInJoinRule extends InlineAndRemoveRedundantBooleanExpressionsRule {
+    private final FDsAndEquivClassesVisitor visitor = new FDsAndEquivClassesVisitor();
+    private final Map<LogicalVariable, LogicalVariable> normalizedVariables = new HashMap<>();
+
+    @Override
+    public boolean rewritePre(Mutable<ILogicalOperator> opRef, IOptimizationContext context)
+            throws AlgebricksException {
+        return false;
+    }
+
+    @Override
+    public boolean rewritePost(Mutable<ILogicalOperator> opRef, IOptimizationContext context)
+            throws AlgebricksException {
+        ILogicalOperator op = opRef.getValue();
+        LogicalOperatorTag opTag = op.getOperatorTag();
+
+        if (context.checkIfInDontApplySet(this, op)) {
+            return false;
+        }
+
+        if (opTag != LogicalOperatorTag.INNERJOIN && opTag != LogicalOperatorTag.LEFTOUTERJOIN) {
+            // TODO FDsAndEquivClassesVisitor alters the distinct variables? We have seen bugs with distinct
+            // not sure if that related
+            if (op.getOperatorTag() != LogicalOperatorTag.DISTINCT) {
+                // Compute the equivalent classes for op
+                op.accept(visitor, context);
+            }
+            context.addToDontApplySet(this, op);
+            return false;
+        }
+
+        boolean changed = normalize(context, op);
+        // compute equivalent classes for the join op
+        op.accept(visitor, context);
+        context.addToDontApplySet(this, op);
+        return changed;
+    }
+
+    private boolean normalize(IOptimizationContext context, ILogicalOperator op) {
+        AbstractBinaryJoinOperator joinOp = (AbstractBinaryJoinOperator) op;
+        ILogicalOperator leftOp = joinOp.getInputs().get(0).getValue();
+        ILogicalOperator rightOp = joinOp.getInputs().get(1).getValue();
+
+        Map<LogicalVariable, EquivalenceClass> leftEqMap = context.getEquivalenceClassMap(leftOp);
+        Map<LogicalVariable, EquivalenceClass> rightEqMap = context.getEquivalenceClassMap(rightOp);
+
+        normalizedVariables.clear();
+
+        Mutable<ILogicalExpression> joinCondRef = joinOp.getCondition();
+        Mutable<ILogicalExpression> clonedCondition = new MutableObject<>(joinCondRef.getValue().cloneExpression());
+
+        if (normalizeVariables(leftEqMap, rightEqMap, clonedCondition) && transform(clonedCondition)) {
+            // replace the join condition iff the normalization led to a minimized circuit
+            joinCondRef.setValue(clonedCondition.getValue());
+            return true;
+        }
+
+        return false;
+    }
+
+    private boolean normalizeVariables(Map<LogicalVariable, EquivalenceClass> leftEqMap,
+            Map<LogicalVariable, EquivalenceClass> rightEqMap, Mutable<ILogicalExpression> exprRef) {
+        ILogicalExpression expr = exprRef.getValue();
+        if (expr.getExpressionTag() == LogicalExpressionTag.FUNCTION_CALL) {
+            return processFunction(leftEqMap, rightEqMap, (AbstractFunctionCallExpression) expr);
+        } else if (expr.getExpressionTag() == LogicalExpressionTag.CONSTANT) {
+            // TODO is this possible in joins?
+            return false;
+        }
+
+        LogicalVariable toNormalizeVariable = VariableUtilities.getVariable(expr);
+        LogicalVariable normalized =
+                getNormalizedVariableAndSetEquivalentsIfAny(leftEqMap, rightEqMap, toNormalizeVariable);
+
+        if (normalized == toNormalizeVariable) {
+            // both are the same, do nothing
+            return false;
+        }
+
+        // we need to replace the variable expression using the normalized expression
+        exprRef.setValue(new VariableReferenceExpression(normalized));
+        return true;
+    }
+
+    private LogicalVariable getNormalizedVariableAndSetEquivalentsIfAny(
+            Map<LogicalVariable, EquivalenceClass> leftEqMap, Map<LogicalVariable, EquivalenceClass> rightEqMap,
+            LogicalVariable toNormalizeVariable) {
+        if (normalizedVariables.containsKey(toNormalizeVariable)) {
+            // get the normalized variable
+            return normalizedVariables.get(toNormalizeVariable);
+        } else if (leftEqMap != null && leftEqMap.containsKey(toNormalizeVariable)) {
+            setNormalizedVariables(toNormalizeVariable, leftEqMap.get(toNormalizeVariable));
+        } else if (rightEqMap != null && rightEqMap.containsKey(toNormalizeVariable)) {
+            setNormalizedVariables(toNormalizeVariable, rightEqMap.get(toNormalizeVariable));
+        }
+
+        return toNormalizeVariable;
+    }
+
+    private void setNormalizedVariables(LogicalVariable toNormalizeVariable, EquivalenceClass equivalenceClass) {
+        for (LogicalVariable eqVar : equivalenceClass.getMembers()) {
+            normalizedVariables.put(eqVar, toNormalizeVariable);
+        }
+    }
+
+    private boolean processFunction(Map<LogicalVariable, EquivalenceClass> leftEqMap,
+            Map<LogicalVariable, EquivalenceClass> rightEqMap, AbstractFunctionCallExpression funcExpr) {
+
+        boolean changed = false;
+        for (Mutable<ILogicalExpression> argRef : funcExpr.getArguments()) {
+            changed |= normalizeVariables(leftEqMap, rightEqMap, argRef);
+        }
+
+        return changed;
+    }
+}
diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/cbo/EnumerateJoinsRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/cbo/EnumerateJoinsRule.java
index 40c25f920e..7ed12f21d0 100644
--- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/cbo/EnumerateJoinsRule.java
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/cbo/EnumerateJoinsRule.java
@@ -23,16 +23,13 @@ import java.util.ArrayList;
 import java.util.HashMap;
 import java.util.HashSet;
 import java.util.Iterator;
-import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Map;
-import java.util.Set;
 
 import org.apache.asterix.common.annotations.IndexedNLJoinExpressionAnnotation;
 import org.apache.asterix.metadata.entities.Index;
 import org.apache.commons.lang3.mutable.Mutable;
 import org.apache.commons.lang3.mutable.MutableInt;
-import org.apache.commons.lang3.mutable.MutableObject;
 import org.apache.hyracks.algebricks.common.exceptions.AlgebricksException;
 import org.apache.hyracks.algebricks.common.utils.Pair;
 import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
@@ -155,7 +152,6 @@ public class EnumerateJoinsRule implements IAlgebraicRewriteRule {
         PlanNode cheapestPlanNode = joinEnum.allPlans.get(cheapestPlan);
 
         if (numberOfFromTerms > 1) {
-            checkForMultipleUsesOfVariablesInJoinPreds(cheapestPlanNode, joinLeafInputsHashMap, context);
             buildNewTree(cheapestPlanNode, joinLeafInputsHashMap, joinOps, new MutableInt(0));
             printPlan(pp, (AbstractLogicalOperator) joinOps.get(0), "New Whole Plan after buildNewTree 1");
             ILogicalOperator root = addConstantInternalEdgesAtTheTop(joinOps.get(0), internalEdges);
@@ -222,7 +218,7 @@ public class EnumerateJoinsRule implements IAlgebraicRewriteRule {
     /**
      * Check to see if there is only one assign here and nothing below that other than a join.
      * have not seen cases where there is more than one assign in a leafinput.
-    */
+     */
     private boolean onlyOneAssign(ILogicalOperator nextOp) {
         if (nextOp.getOperatorTag() != LogicalOperatorTag.ASSIGN) {
             return false;
@@ -532,91 +528,6 @@ public class EnumerateJoinsRule implements IAlgebraicRewriteRule {
         }
     }
 
-    private boolean allEqualityPreds(ILogicalExpression expr) {
-        List<Mutable<ILogicalExpression>> conjs = new ArrayList<>();
-        // check that the expr is AND(EQ(), EQ(),..)
-        if (expr.splitIntoConjuncts(conjs)) {
-            for (Mutable<ILogicalExpression> conj : conjs) {
-                if (!(((AbstractFunctionCallExpression) conj.getValue()).getFunctionIdentifier()
-                        .equals(AlgebricksBuiltinFunctions.EQ))) {
-                    return false;
-                }
-            }
-            return true;
-        }
-        return false;
-    }
-
-    // This routine should not be needed! When the same variable is used multiple times in a join predicate as in
-    // AND(eq($$25, $$27), eq($$25, $$34)), JoinUtils.isHashJoinCondition() returns false which makes no sense.
-    // Tried changing the above routine but it always lead to some failures. Unable to figure out what the problem was,
-    // we just replace the duplicate occurrence of every variable with a new variable following by appropriate assign.
-    // We do this only once just once before we construct the final plan.
-    private void checkForMultipleUsesOfVariablesInJoinPreds(PlanNode plan,
-            HashMap<EmptyTupleSourceOperator, ILogicalOperator> joinLeafInputsHashMap, IOptimizationContext context)
-            throws AlgebricksException {
-        List<PlanNode> allPlans = joinEnum.getAllPlans();
-        if (plan.IsJoinNode()) {
-            ILogicalExpression exp = plan.getJoinExpr();
-            if (!allEqualityPreds(exp)) {
-                return;
-            }
-            boolean changes = true;
-            while (changes) {
-                changes = false;
-                List<LogicalVariable> vars = new ArrayList<>();
-                exp.getUsedVariables(vars);
-                Set<LogicalVariable> set = new LinkedHashSet<>(vars);
-                if (set.size() < vars.size()) {
-                    // walk thru vars and find the first instance of the duplicate
-                    for (int i = 0; i < vars.size() - 1; i++) {
-                        for (int j = i + 1; j < vars.size(); j++) {
-                            if (vars.get(i) == vars.get(j)) {
-                                /// find the leafInout that contains this vars(i)
-                                for (Map.Entry<EmptyTupleSourceOperator, ILogicalOperator> mapElement : joinLeafInputsHashMap
-                                        .entrySet()) {
-                                    ILogicalOperator joinLeafInput = mapElement.getValue();
-                                    EmptyTupleSourceOperator ets = mapElement.getKey();
-                                    HashSet<LogicalVariable> vars2 = new HashSet<>();
-                                    VariableUtilities.getLiveVariables(joinLeafInput, vars2);
-                                    if (vars2.contains(vars.get(i))) {
-                                        LogicalVariable newVar = context.newVar();
-                                        // replace one occurrence of vars(i) in exp
-                                        substituteVarOnce(exp, vars.get(i), newVar);
-                                        VariableReferenceExpression oldvarExpr =
-                                                new VariableReferenceExpression(vars.get(i));
-                                        AssignOperator assign =
-                                                new AssignOperator(newVar, new MutableObject<>(oldvarExpr));
-                                        // Now add an assign to the joinLeafInput : newvar <-- oldvar
-                                        assign.getInputs().add(new MutableObject<>(joinLeafInput));
-                                        context.computeAndSetTypeEnvironmentForOperator(assign);
-                                        context.addNotToBeInlinedVar(newVar);
-                                        context.addNotToBeInlinedVar(vars.get(i));
-
-                                        // also update the joinLeafInputsHashMap
-                                        joinLeafInputsHashMap.put(ets, assign);
-                                        changes = true;
-                                    }
-                                }
-                            }
-                        }
-                    }
-                }
-            }
-            // now traverse left and right side plans
-            int leftIndex = plan.getLeftPlanIndex();
-            int rightIndex = plan.getRightPlanIndex();
-            PlanNode leftPlan = allPlans.get(leftIndex);
-            PlanNode rightPlan = allPlans.get(rightIndex);
-            if (leftPlan.IsJoinNode()) {
-                checkForMultipleUsesOfVariablesInJoinPreds(leftPlan, joinLeafInputsHashMap, context);
-            }
-            if (rightPlan.IsJoinNode()) {
-                checkForMultipleUsesOfVariablesInJoinPreds(rightPlan, joinLeafInputsHashMap, context);
-            }
-        }
-    }
-
     // for every internal edge assign (again assuming only 1 for now), find the corresponding leafInput and place the assign
     // on top of that LeafInput. Modify the joinLeafInputsHashMap as well.
     private void pushAssignsIntoLeafInputs(HashMap<EmptyTupleSourceOperator, ILogicalOperator> joinLeafInputsHashMap,
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.sqlpp b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.sqlpp
new file mode 100644
index 0000000000..2ed73fb17e
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.sqlpp
@@ -0,0 +1,63 @@
+/*
+ * 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 test IF EXISTS;
+CREATE DATAVERSE test;
+
+USE test;
+
+CREATE TYPE OpenType AS {
+    uid: uuid
+};
+
+CREATE DATASET region(OpenType)
+PRIMARY KEY uid AUTOGENERATED;
+
+CREATE DATASET nation(OpenType)
+PRIMARY KEY uid AUTOGENERATED;
+
+CREATE DATASET customer(OpenType)
+PRIMARY KEY uid AUTOGENERATED;
+
+
+CREATE DATASET orders(OpenType)
+PRIMARY KEY uid AUTOGENERATED;
+
+
+CREATE DATASET lineitem(OpenType)
+PRIMARY KEY uid AUTOGENERATED;
+
+
+CREATE DATASET supplier(OpenType)
+PRIMARY KEY uid AUTOGENERATED;
+
+SELECT VALUE COUNT(*)
+FROM region r, nation n, customer c, orders o, lineitem l, supplier s
+WHERE c.c_custkey = o.o_custkey
+  AND l.l_orderkey = o.o_orderkey
+  AND l.l_suppkey = s.s_suppkey
+  AND s.s_nationkey = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  -- this predicate should be eliminated as it is implicitly
+  -- inferred by the two predicates above
+  AND c.c_nationkey = s.s_nationkey
+  AND n.n_regionkey = r.r_regionkey
+  AND r.r_name = 'EUROPE'
+  AND o.o_orderdate >= '1993-01-01'
+  AND o.o_orderdate < '1993-04-01';
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.2.sqlpp b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.2.sqlpp
new file mode 100644
index 0000000000..f2c56adda1
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.2.sqlpp
@@ -0,0 +1,28 @@
+/*
+ * 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 test;
+
+SELECT VALUE COUNT(*)
+FROM region r, orders o, nation n
+WHERE o.x = r.x
+  AND r.x = n.x
+    -- this predicate should be eliminated as it is implicitly
+    -- inferred by the two predicates above
+  AND o.x = n.x;
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.3.sqlpp b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.3.sqlpp
new file mode 100644
index 0000000000..78d7c900ac
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.3.sqlpp
@@ -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.
+ */
+
+USE test;
+
+SELECT VALUE COUNT(*)
+FROM region r, nation n, orders o
+WHERE o.x = r.x
+  AND r.x = n.x
+    -- this predicate should be eliminated as it is implicitly
+    -- inferred by the two predicates above
+  AND o.x = n.x
+  -- a redundant predicate (i.e., to o.x = r.x)
+  AND r.x = o.x;
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.4.sqlpp b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.4.sqlpp
new file mode 100644
index 0000000000..09bda7e929
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/hash-join-with-redundant-variable/hash-join-with-redundant-variable.4.sqlpp
@@ -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 test;
+
+SELECT VALUE COUNT(*)
+FROM region r, orders o
+WHERE r.x = o.x
+  -- r.x should be also be assigned to a new variable (a copy variable)
+  AND r.x = o.y;
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.plan
new file mode 100644
index 0000000000..8b2d960045
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.plan
@@ -0,0 +1,74 @@
+-- DISTRIBUTE_RESULT  |UNPARTITIONED|
+  -- ONE_TO_ONE_EXCHANGE  |UNPARTITIONED|
+    -- AGGREGATE  |UNPARTITIONED|
+      -- RANDOM_MERGE_EXCHANGE  |PARTITIONED|
+        -- AGGREGATE  |PARTITIONED|
+          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+            -- HYBRID_HASH_JOIN [$$127, $$117][$$128, $$116]  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$127, $$117]  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                    -- HYBRID_HASH_JOIN [$$132][$$131]  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$132]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            -- HYBRID_HASH_JOIN [$$117][$$115]  |PARTITIONED|
+                              -- HASH_PARTITION_EXCHANGE [$$117]  |PARTITIONED|
+                                -- STREAM_PROJECT  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- HYBRID_HASH_JOIN [$$130][$$129]  |PARTITIONED|
+                                      -- HASH_PARTITION_EXCHANGE [$$130]  |PARTITIONED|
+                                        -- STREAM_PROJECT  |PARTITIONED|
+                                          -- STREAM_SELECT  |PARTITIONED|
+                                            -- ASSIGN  |PARTITIONED|
+                                              -- STREAM_PROJECT  |PARTITIONED|
+                                                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                  -- DATASOURCE_SCAN (test.region)  |PARTITIONED|
+                                                    -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                      -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                                      -- HASH_PARTITION_EXCHANGE [$$129]  |PARTITIONED|
+                                        -- STREAM_PROJECT  |PARTITIONED|
+                                          -- ASSIGN  |PARTITIONED|
+                                            -- STREAM_PROJECT  |PARTITIONED|
+                                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                -- DATASOURCE_SCAN (test.nation)  |PARTITIONED|
+                                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                              -- HASH_PARTITION_EXCHANGE [$$115]  |PARTITIONED|
+                                -- STREAM_PROJECT  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- HYBRID_HASH_JOIN [$$125][$$126]  |PARTITIONED|
+                                      -- HASH_PARTITION_EXCHANGE [$$125]  |PARTITIONED|
+                                        -- STREAM_PROJECT  |PARTITIONED|
+                                          -- ASSIGN  |PARTITIONED|
+                                            -- STREAM_PROJECT  |PARTITIONED|
+                                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                -- DATASOURCE_SCAN (test.customer)  |PARTITIONED|
+                                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                                      -- HASH_PARTITION_EXCHANGE [$$126]  |PARTITIONED|
+                                        -- STREAM_PROJECT  |PARTITIONED|
+                                          -- STREAM_SELECT  |PARTITIONED|
+                                            -- STREAM_PROJECT  |PARTITIONED|
+                                              -- ASSIGN  |PARTITIONED|
+                                                -- STREAM_PROJECT  |PARTITIONED|
+                                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                    -- DATASOURCE_SCAN (test.orders)  |PARTITIONED|
+                                                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                        -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$131]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ASSIGN  |PARTITIONED|
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                -- DATASOURCE_SCAN (test.lineitem)  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$128, $$116]  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ASSIGN  |PARTITIONED|
+                    -- STREAM_PROJECT  |PARTITIONED|
+                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                        -- DATASOURCE_SCAN (test.supplier)  |PARTITIONED|
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.2.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.2.plan
new file mode 100644
index 0000000000..d56477e33f
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.2.plan
@@ -0,0 +1,35 @@
+-- DISTRIBUTE_RESULT  |UNPARTITIONED|
+  -- ONE_TO_ONE_EXCHANGE  |UNPARTITIONED|
+    -- AGGREGATE  |UNPARTITIONED|
+      -- RANDOM_MERGE_EXCHANGE  |PARTITIONED|
+        -- AGGREGATE  |PARTITIONED|
+          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+            -- HYBRID_HASH_JOIN [$$65][$$63]  |PARTITIONED|
+              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                    -- HYBRID_HASH_JOIN [$$65][$$64]  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$65]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ASSIGN  |PARTITIONED|
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                -- DATASOURCE_SCAN (test.region)  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$64]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ASSIGN  |PARTITIONED|
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                -- DATASOURCE_SCAN (test.orders)  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$63]  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ASSIGN  |PARTITIONED|
+                    -- STREAM_PROJECT  |PARTITIONED|
+                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                        -- DATASOURCE_SCAN (test.nation)  |PARTITIONED|
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.3.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.3.plan
new file mode 100644
index 0000000000..5264d56e89
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.3.plan
@@ -0,0 +1,35 @@
+-- DISTRIBUTE_RESULT  |UNPARTITIONED|
+  -- ONE_TO_ONE_EXCHANGE  |UNPARTITIONED|
+    -- AGGREGATE  |UNPARTITIONED|
+      -- RANDOM_MERGE_EXCHANGE  |PARTITIONED|
+        -- AGGREGATE  |PARTITIONED|
+          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+            -- HYBRID_HASH_JOIN [$$68][$$66]  |PARTITIONED|
+              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                    -- HYBRID_HASH_JOIN [$$67][$$68]  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$67]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ASSIGN  |PARTITIONED|
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                -- DATASOURCE_SCAN (test.region)  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$68]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ASSIGN  |PARTITIONED|
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                -- DATASOURCE_SCAN (test.nation)  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$66]  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ASSIGN  |PARTITIONED|
+                    -- STREAM_PROJECT  |PARTITIONED|
+                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                        -- DATASOURCE_SCAN (test.orders)  |PARTITIONED|
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.4.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.4.plan
new file mode 100644
index 0000000000..c898cde6eb
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/hash-join-with-redundant-variable/hash-join-with-redundant-variable.4.plan
@@ -0,0 +1,24 @@
+-- DISTRIBUTE_RESULT  |UNPARTITIONED|
+  -- ONE_TO_ONE_EXCHANGE  |UNPARTITIONED|
+    -- AGGREGATE  |UNPARTITIONED|
+      -- RANDOM_MERGE_EXCHANGE  |PARTITIONED|
+        -- AGGREGATE  |PARTITIONED|
+          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+            -- HYBRID_HASH_JOIN [$$49, $$56][$$52, $$53]  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$49, $$56]  |PARTITIONED|
+                -- ASSIGN  |PARTITIONED|
+                  -- STREAM_PROJECT  |PARTITIONED|
+                    -- ASSIGN  |PARTITIONED|
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          -- DATASOURCE_SCAN (test.region)  |PARTITIONED|
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$52, $$53]  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ASSIGN  |PARTITIONED|
+                    -- STREAM_PROJECT  |PARTITIONED|
+                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                        -- DATASOURCE_SCAN (test.orders)  |PARTITIONED|
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results_cbo/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results_cbo/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.plan
new file mode 100644
index 0000000000..65df3c22c6
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results_cbo/hash-join-with-redundant-variable/hash-join-with-redundant-variable.1.plan
@@ -0,0 +1,74 @@
+-- DISTRIBUTE_RESULT  |UNPARTITIONED|
+  -- ONE_TO_ONE_EXCHANGE  |UNPARTITIONED|
+    -- AGGREGATE  |UNPARTITIONED|
+      -- RANDOM_MERGE_EXCHANGE  |PARTITIONED|
+        -- AGGREGATE  |PARTITIONED|
+          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+            -- HYBRID_HASH_JOIN [$$130][$$129]  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$130]  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- STREAM_SELECT  |PARTITIONED|
+                    -- ASSIGN  |PARTITIONED|
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          -- DATASOURCE_SCAN (test.region)  |PARTITIONED|
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+              -- HASH_PARTITION_EXCHANGE [$$129]  |PARTITIONED|
+                -- STREAM_PROJECT  |PARTITIONED|
+                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                    -- HYBRID_HASH_JOIN [$$127, $$131][$$128, $$132]  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$127, $$131]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ASSIGN  |PARTITIONED|
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                -- DATASOURCE_SCAN (test.lineitem)  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                      -- HASH_PARTITION_EXCHANGE [$$128, $$132]  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            -- HYBRID_HASH_JOIN [$$126][$$125]  |PARTITIONED|
+                              -- HASH_PARTITION_EXCHANGE [$$126]  |PARTITIONED|
+                                -- STREAM_PROJECT  |PARTITIONED|
+                                  -- STREAM_SELECT  |PARTITIONED|
+                                    -- STREAM_PROJECT  |PARTITIONED|
+                                      -- ASSIGN  |PARTITIONED|
+                                        -- STREAM_PROJECT  |PARTITIONED|
+                                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                            -- DATASOURCE_SCAN (test.orders)  |PARTITIONED|
+                                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                              -- HASH_PARTITION_EXCHANGE [$$125]  |PARTITIONED|
+                                -- STREAM_PROJECT  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- HYBRID_HASH_JOIN [$$117][$$116]  |PARTITIONED|
+                                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                        -- STREAM_PROJECT  |PARTITIONED|
+                                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                            -- HYBRID_HASH_JOIN [$$117][$$115]  |PARTITIONED|
+                                              -- HASH_PARTITION_EXCHANGE [$$117]  |PARTITIONED|
+                                                -- STREAM_PROJECT  |PARTITIONED|
+                                                  -- ASSIGN  |PARTITIONED|
+                                                    -- STREAM_PROJECT  |PARTITIONED|
+                                                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                        -- DATASOURCE_SCAN (test.nation)  |PARTITIONED|
+                                                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                            -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                                              -- HASH_PARTITION_EXCHANGE [$$115]  |PARTITIONED|
+                                                -- STREAM_PROJECT  |PARTITIONED|
+                                                  -- ASSIGN  |PARTITIONED|
+                                                    -- STREAM_PROJECT  |PARTITIONED|
+                                                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                        -- DATASOURCE_SCAN (test.customer)  |PARTITIONED|
+                                                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                            -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                                      -- HASH_PARTITION_EXCHANGE [$$116]  |PARTITIONED|
+                                        -- STREAM_PROJECT  |PARTITIONED|
+                                          -- ASSIGN  |PARTITIONED|
+                                            -- STREAM_PROJECT  |PARTITIONED|
+                                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                -- DATASOURCE_SCAN (test.supplier)  |PARTITIONED|
+                                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.01.ddl.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.01.ddl.sqlpp
new file mode 100644
index 0000000000..48e46f65d8
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.01.ddl.sqlpp
@@ -0,0 +1,134 @@
+/*
+ * 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 tpch IF EXISTS;
+CREATE DATAVERSE tpch;
+
+USE tpch;
+
+
+CREATE TYPE tpch.LineItemType AS
+ CLOSED {
+  l_orderkey : integer,
+  l_partkey : integer,
+  l_suppkey : integer,
+  l_linenumber : integer,
+  l_quantity : integer,
+  l_extendedprice : double,
+  l_discount : double,
+  l_tax : double,
+  l_returnflag : string,
+  l_linestatus : string,
+  l_shipdate : string,
+  l_commitdate : string,
+  l_receiptdate : string,
+  l_shipinstruct : string,
+  l_shipmode : string,
+  l_comment : string
+};
+
+CREATE TYPE tpch.OrderType AS
+ CLOSED {
+  o_orderkey : integer,
+  o_custkey : integer,
+  o_orderstatus : string,
+  o_totalprice : double,
+  o_orderdate : string,
+  o_orderpriority : string,
+  o_clerk : string,
+  o_shippriority : integer,
+  o_comment : string
+};
+
+CREATE TYPE tpch.CustomerType AS
+ CLOSED {
+  c_custkey : integer,
+  c_name : string,
+  c_address : string,
+  c_nationkey : integer,
+  c_phone : string,
+  c_acctbal : double,
+  c_mktsegment : string,
+  c_comment : string
+};
+
+CREATE TYPE tpch.SupplierType AS
+ CLOSED {
+  s_suppkey : integer,
+  s_name : string,
+  s_address : string,
+  s_nationkey : integer,
+  s_phone : string,
+  s_acctbal : double,
+  s_comment : string
+};
+
+CREATE TYPE tpch.NationType AS
+ CLOSED {
+  n_nationkey : integer,
+  n_name : string,
+  n_regionkey : integer,
+  n_comment : string
+};
+
+CREATE TYPE tpch.RegionType AS
+ CLOSED {
+  r_regionkey : integer,
+  r_name : string,
+  r_comment : string
+};
+
+CREATE TYPE tpch.PartType AS
+ CLOSED {
+  p_partkey : integer,
+  p_name : string,
+  p_mfgr : string,
+  p_brand : string,
+  p_type : string,
+  p_size : integer,
+  p_container : string,
+  p_retailprice : double,
+  p_comment : string
+};
+
+CREATE TYPE tpch.PartSuppType AS
+ CLOSED {
+  ps_partkey : integer,
+  ps_suppkey : integer,
+  ps_availqty : integer,
+  ps_supplycost : double,
+  ps_comment : string
+};
+
+CREATE DATASET LineItem(LineItemType) PRIMARY KEY l_orderkey,l_linenumber;
+
+CREATE DATASET Orders(OrderType) PRIMARY KEY o_orderkey;
+
+CREATE DATASET Supplier(SupplierType) PRIMARY KEY s_suppkey;
+
+CREATE DATASET Region(RegionType) PRIMARY KEY r_regionkey;
+
+CREATE DATASET Nation(NationType) PRIMARY KEY n_nationkey;
+
+CREATE DATASET Part(PartType) PRIMARY KEY p_partkey;
+
+CREATE DATASET Partsupp(PartSuppType) PRIMARY KEY ps_partkey,ps_suppkey;
+
+CREATE DATASET Customer(CustomerType) PRIMARY KEY c_custkey;
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.02.update.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.02.update.sqlpp
new file mode 100644
index 0000000000..62a6ef46ec
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.02.update.sqlpp
@@ -0,0 +1,36 @@
+/*
+ * 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 tpch;
+
+LOAD DATASET LineItem USING localfs ((`path`=`asterix_nc1://data/tpch0.001/lineitem.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
+
+LOAD DATASET Orders USING localfs ((`path`=`asterix_nc1://data/tpch0.001/orders.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
+
+LOAD DATASET Supplier USING localfs ((`path`=`asterix_nc1://data/tpch0.001/supplier.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
+
+LOAD DATASET Region USING localfs ((`path`=`asterix_nc1://data/tpch0.001/region.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
+
+LOAD DATASET Nation USING localfs ((`path`=`asterix_nc1://data/tpch0.001/nation.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
+
+LOAD DATASET Part USING localfs ((`path`=`asterix_nc1://data/tpch0.001/part.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
+
+LOAD DATASET Partsupp USING localfs ((`path`=`asterix_nc1://data/tpch0.001/partsupp.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
+
+LOAD DATASET Customer USING localfs ((`path`=`asterix_nc1://data/tpch0.001/customer.tbl`),(`format`=`delimited-text`),(`delimiter`=`|`));
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.03.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.03.query.sqlpp
new file mode 100644
index 0000000000..f04553ede6
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.03.query.sqlpp
@@ -0,0 +1,32 @@
+/*
+ * 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 tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "0";
+
+
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Nation n, Supplier s, Customer c
+WHERE s.s_nationkey = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.04.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.04.query.sqlpp
new file mode 100644
index 0000000000..6de6bc922f
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.04.query.sqlpp
@@ -0,0 +1,32 @@
+/*
+ * 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 tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "0";
+
+EXPLAIN
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Nation n, Supplier s, Customer c
+WHERE s.s_nationkey = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.05.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.05.query.sqlpp
new file mode 100644
index 0000000000..336a54571a
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.05.query.sqlpp
@@ -0,0 +1,39 @@
+/*
+ * 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.
+ */
+
+/*
+ * This query does not make a logical sense; however,
+ * the idea here is to test that a hash-join can be used when we have a case
+ * where one relation uses one attribute to join on two attributes on the
+ * the other relation
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+-- To prevent plan changes
+SET `compiler.parallelism` "0";
+
+
+SELECT o.o_orderkey, l.l_orderkey, l.l_suppkey
+FROM Orders o, LineItem l
+WHERE o.o_orderkey = l.l_orderkey
+  AND o.o_orderkey = l.l_suppkey
+ORDER BY o.o_orderkey, l.l_orderkey, l.l_suppkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.06.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.06.query.sqlpp
new file mode 100644
index 0000000000..271a5289f6
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.06.query.sqlpp
@@ -0,0 +1,39 @@
+/*
+ * 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.
+ */
+
+/*
+ * This query does not make a logical sense; however,
+ * the idea here is to test that a hash-join can be used when we have a case
+ * where one relation uses one attribute to join on two attributes on the
+ * the other relation
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+-- To prevent plan changes
+SET `compiler.parallelism` "0";
+
+EXPLAIN
+SELECT o.o_orderkey, l.l_orderkey, l.l_suppkey
+FROM Orders o, LineItem l
+WHERE o.o_orderkey = l.l_orderkey
+  AND o.o_orderkey = l.l_suppkey
+ORDER BY o.o_orderkey, l.l_orderkey, l.l_suppkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.07.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.07.query.sqlpp
new file mode 100644
index 0000000000..5bd7308153
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.07.query.sqlpp
@@ -0,0 +1,39 @@
+/*
+ * 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.
+ */
+
+/*
+ * This query does not make a logical sense; however,
+ * the idea here is to test that a hash-join can be used when we have a case
+ * where one relation uses one attribute to join on two attributes on the
+ * the other relation
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "0";
+
+
+SELECT o.o_orderkey, l.l_orderkey, l.l_suppkey
+FROM            Orders o
+LEFT OUTER JOIN LineItem l
+             ON o.o_orderkey = l.l_orderkey
+            AND o.o_orderkey = l.l_suppkey
+ORDER BY o.o_orderkey, l.l_orderkey, l.l_suppkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.08.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.08.query.sqlpp
new file mode 100644
index 0000000000..5be6513454
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.08.query.sqlpp
@@ -0,0 +1,39 @@
+/*
+ * 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.
+ */
+
+/*
+ * This query does not make a logical sense; however,
+ * the idea here is to test that a hash-join can be used when we have a case
+ * where one relation uses one attribute to join on two attributes on the
+ * the other relation
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "0";
+
+EXPLAIN
+SELECT o.o_orderkey, l.l_orderkey, l.l_suppkey
+FROM            Orders o
+LEFT OUTER JOIN LineItem l
+             ON o.o_orderkey = l.l_orderkey
+            AND o.o_orderkey = l.l_suppkey
+ORDER BY o.o_orderkey, l.l_orderkey, l.l_suppkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.09.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.09.query.sqlpp
new file mode 100644
index 0000000000..ee5a92d51b
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.09.query.sqlpp
@@ -0,0 +1,35 @@
+/*
+ * 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 tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+-- To ensure that Nation's key is hash partitioned
+SET `compiler.parallelism` "3";
+-- To disable reordering joins
+SET `compiler.cbo` "true";
+
+
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Nation n, Supplier s, Customer c
+WHERE s.s_nationkey = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.10.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.10.query.sqlpp
new file mode 100644
index 0000000000..5a13fc32d0
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.10.query.sqlpp
@@ -0,0 +1,35 @@
+/*
+ * 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 tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+-- To ensure that Nation's key is hash partitioned
+SET `compiler.parallelism` "3";
+-- To disable reordering joins
+SET `compiler.cbo` "true";
+
+EXPLAIN
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Nation n, Supplier s, Customer c
+WHERE s.s_nationkey = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.11.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.11.query.sqlpp
new file mode 100644
index 0000000000..59f3fb8c67
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.11.query.sqlpp
@@ -0,0 +1,35 @@
+/*
+ * 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 tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+-- To ensure that Nation's key is hash partitioned
+SET `compiler.parallelism` "-1";
+-- To disable reordering joins
+SET `compiler.cbo` "true";
+
+
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Nation n, Supplier s, Customer c
+WHERE s.s_nationkey = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.12.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.12.query.sqlpp
new file mode 100644
index 0000000000..85077fe0c1
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.12.query.sqlpp
@@ -0,0 +1,35 @@
+/*
+ * 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 tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+-- To ensure that Nation's key is hash partitioned
+SET `compiler.parallelism` "-1";
+-- To disable reordering joins
+SET `compiler.cbo` "true";
+
+EXPLAIN
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Nation n, Supplier s, Customer c
+WHERE s.s_nationkey = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.13.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.13.query.sqlpp
new file mode 100644
index 0000000000..9f52f75ad7
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.13.query.sqlpp
@@ -0,0 +1,39 @@
+/*
+ * 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.
+ */
+/*
+ * Similar to hash-join-with-redundant-variable.04.query.sqlpp
+ * But with Index NL. The plan of this test should has two
+ * hash-partition-exchange -- both of which with one variable.
+ * One from 'Supplier' and one from 'Customer'. Nation should not be
+ * partitioned here to join with Customer (already partitioned by its PK)
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "0";
+
+
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Supplier s, Nation n, Customer c
+WHERE s.s_nationkey /* +indexnl */ = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.14.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.14.query.sqlpp
new file mode 100644
index 0000000000..379055a1bd
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.14.query.sqlpp
@@ -0,0 +1,39 @@
+/*
+ * 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.
+ */
+/*
+ * Similar to hash-join-with-redundant-variable.04.query.sqlpp
+ * But with Index NL. The plan of this test should has two
+ * hash-partition-exchange -- both of which with one variable.
+ * One from 'Supplier' and one from 'Customer'. Nation should not be
+ * partitioned here to join with Customer (already partitioned by its PK)
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "0";
+
+EXPLAIN
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Supplier s, Nation n, Customer c
+WHERE s.s_nationkey /* +indexnl */ = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.15.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.15.query.sqlpp
new file mode 100644
index 0000000000..c77b25e3df
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.15.query.sqlpp
@@ -0,0 +1,40 @@
+/*
+ * 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.
+ */
+/*
+ * Similar to hash-join-with-redundant-variable.04.query.sqlpp
+ * But with Index NL. The plan of this test should has three
+ * hash-partition-exchange (as opposed to test 13 & 14). Because the parallelism
+ * is set to 3, then the last join requires both sides to be hash partitioned.
+ * Customer will need to duplicate its variable to join both with Nation and Supplier.
+ * This is the effect of using Index NL with parallelism != # of partitions
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "3";
+
+
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Supplier s, Nation n, Customer c
+WHERE s.s_nationkey /* +indexnl */ = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.16.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.16.query.sqlpp
new file mode 100644
index 0000000000..90aff2ea41
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.16.query.sqlpp
@@ -0,0 +1,40 @@
+/*
+ * 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.
+ */
+/*
+ * Similar to hash-join-with-redundant-variable.04.query.sqlpp
+ * But with Index NL. The plan of this test should has three
+ * hash-partition-exchange (as opposed to test 13 & 14). Because the parallelism
+ * is set to 3, then the last join requires both sides to be hash partitioned.
+ * Customer will need to duplicate its variable to join both with Nation and Supplier.
+ * This is the effect of using Index NL with parallelism != # of partitions
+ */
+
+USE tpch;
+
+-- Disabled for a simpler plan
+SET `compiler.sort.parallel` "false";
+SET `compiler.parallelism` "3";
+
+EXPLAIN
+SELECT n.n_nationkey, s.s_nationkey, c.c_nationkey
+FROM Supplier s, Nation n, Customer c
+WHERE s.s_nationkey /* +indexnl */ = n.n_nationkey
+  AND c.c_nationkey = n.n_nationkey
+  AND s.s_nationkey = c.c_nationkey
+ORDER BY n.n_nationkey, s.s_nationkey, c.c_nationkey
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.03.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.03.adm
new file mode 100644
index 0000000000..322cb104da
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.03.adm
@@ -0,0 +1,58 @@
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 24, "s_nationkey": 24, "c_nationkey": 24 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.04.plan b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.04.plan
new file mode 100644
index 0000000000..d5cc785fc0
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.04.plan
@@ -0,0 +1,64 @@
+distribute result [$$48]
+-- DISTRIBUTE_RESULT  |PARTITIONED|
+  exchange
+  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+    project ([$$48])
+    -- STREAM_PROJECT  |PARTITIONED|
+      assign [$$48] <- [{"n_nationkey": $$55, "s_nationkey": $$53, "c_nationkey": $$52}]
+      -- ASSIGN  |PARTITIONED|
+        exchange
+        -- SORT_MERGE_EXCHANGE [$$55(ASC), $$53(ASC), $$52(ASC) ]  |PARTITIONED|
+          order (ASC, $$55) (ASC, $$53) (ASC, $$52)
+          -- STABLE_SORT [$$55(ASC), $$53(ASC), $$52(ASC)]  |PARTITIONED|
+            exchange
+            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+              join (eq($$52, $$55))
+              -- HYBRID_HASH_JOIN [$$55][$$52]  |PARTITIONED|
+                exchange
+                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                  join (eq($$53, $$55))
+                  -- HYBRID_HASH_JOIN [$$55][$$53]  |PARTITIONED|
+                    exchange
+                    -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                      project ([$$55])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          data-scan []<-[$$55, $$n] <- tpch.Nation
+                          -- DATASOURCE_SCAN  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              empty-tuple-source
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$53]  |PARTITIONED|
+                      project ([$$53])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        assign [$$53] <- [$$s.getField(3)]
+                        -- ASSIGN  |PARTITIONED|
+                          project ([$$s])
+                          -- STREAM_PROJECT  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              data-scan []<-[$$56, $$s] <- tpch.Supplier
+                              -- DATASOURCE_SCAN  |PARTITIONED|
+                                exchange
+                                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                  empty-tuple-source
+                                  -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                exchange
+                -- HASH_PARTITION_EXCHANGE [$$52]  |PARTITIONED|
+                  project ([$$52])
+                  -- STREAM_PROJECT  |PARTITIONED|
+                    assign [$$52] <- [$$c.getField(3)]
+                    -- ASSIGN  |PARTITIONED|
+                      project ([$$c])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          data-scan []<-[$$57, $$c] <- tpch.Customer
+                          -- DATASOURCE_SCAN  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              empty-tuple-source
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.05.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.05.adm
new file mode 100644
index 0000000000..92abec0cfd
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.05.adm
@@ -0,0 +1,3 @@
+{ "o_orderkey": 2, "l_orderkey": 2, "l_suppkey": 2 }
+{ "o_orderkey": 5, "l_orderkey": 5, "l_suppkey": 5 }
+{ "o_orderkey": 6, "l_orderkey": 6, "l_suppkey": 6 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.06.plan b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.06.plan
new file mode 100644
index 0000000000..977b1db706
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.06.plan
@@ -0,0 +1,50 @@
+distribute result [$$34]
+-- DISTRIBUTE_RESULT  |PARTITIONED|
+  exchange
+  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+    project ([$$34])
+    -- STREAM_PROJECT  |PARTITIONED|
+      assign [$$34] <- [{"o_orderkey": $$41, "l_orderkey": $$42, "l_suppkey": $$40}]
+      -- ASSIGN  |PARTITIONED|
+        exchange
+        -- SORT_MERGE_EXCHANGE [$$41(ASC), $$42(ASC), $$40(ASC) ]  |PARTITIONED|
+          order (ASC, $$41) (ASC, $$42) (ASC, $$40)
+          -- STABLE_SORT [$$41(ASC), $$42(ASC), $$40(ASC)]  |PARTITIONED|
+            exchange
+            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+              project ([$$41, $$42, $$40])
+              -- STREAM_PROJECT  |PARTITIONED|
+                exchange
+                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                  join (and(eq($$41, $$42), eq($$47, $$40)))
+                  -- HYBRID_HASH_JOIN [$$41, $$47][$$42, $$40]  |PARTITIONED|
+                    exchange
+                    -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                      assign [$$47] <- [$$41]
+                      -- ASSIGN  |PARTITIONED|
+                        project ([$$41])
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          exchange
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            data-scan []<-[$$41, $$o] <- tpch.Orders
+                            -- DATASOURCE_SCAN  |PARTITIONED|
+                              exchange
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                empty-tuple-source
+                                -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$42]  |PARTITIONED|
+                      project ([$$42, $$40])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        assign [$$40] <- [$$l.getField(2)]
+                        -- ASSIGN  |PARTITIONED|
+                          project ([$$42, $$l])
+                          -- STREAM_PROJECT  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              data-scan []<-[$$42, $$43, $$l] <- tpch.LineItem
+                              -- DATASOURCE_SCAN  |PARTITIONED|
+                                exchange
+                                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                  empty-tuple-source
+                                  -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.07.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.07.adm
new file mode 100644
index 0000000000..55acc860a6
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.07.adm
@@ -0,0 +1,1500 @@
+{ "o_orderkey": 1 }
+{ "o_orderkey": 2, "l_orderkey": 2, "l_suppkey": 2 }
+{ "o_orderkey": 3 }
+{ "o_orderkey": 4 }
+{ "o_orderkey": 5, "l_orderkey": 5, "l_suppkey": 5 }
+{ "o_orderkey": 6, "l_orderkey": 6, "l_suppkey": 6 }
+{ "o_orderkey": 7 }
+{ "o_orderkey": 32 }
+{ "o_orderkey": 33 }
+{ "o_orderkey": 34 }
+{ "o_orderkey": 35 }
+{ "o_orderkey": 36 }
+{ "o_orderkey": 37 }
+{ "o_orderkey": 38 }
+{ "o_orderkey": 39 }
+{ "o_orderkey": 64 }
+{ "o_orderkey": 65 }
+{ "o_orderkey": 66 }
+{ "o_orderkey": 67 }
+{ "o_orderkey": 68 }
+{ "o_orderkey": 69 }
+{ "o_orderkey": 70 }
+{ "o_orderkey": 71 }
+{ "o_orderkey": 96 }
+{ "o_orderkey": 97 }
+{ "o_orderkey": 98 }
+{ "o_orderkey": 99 }
+{ "o_orderkey": 100 }
+{ "o_orderkey": 101 }
+{ "o_orderkey": 102 }
+{ "o_orderkey": 103 }
+{ "o_orderkey": 128 }
+{ "o_orderkey": 129 }
+{ "o_orderkey": 130 }
+{ "o_orderkey": 131 }
+{ "o_orderkey": 132 }
+{ "o_orderkey": 133 }
+{ "o_orderkey": 134 }
+{ "o_orderkey": 135 }
+{ "o_orderkey": 160 }
+{ "o_orderkey": 161 }
+{ "o_orderkey": 162 }
+{ "o_orderkey": 163 }
+{ "o_orderkey": 164 }
+{ "o_orderkey": 165 }
+{ "o_orderkey": 166 }
+{ "o_orderkey": 167 }
+{ "o_orderkey": 192 }
+{ "o_orderkey": 193 }
+{ "o_orderkey": 194 }
+{ "o_orderkey": 195 }
+{ "o_orderkey": 196 }
+{ "o_orderkey": 197 }
+{ "o_orderkey": 198 }
+{ "o_orderkey": 199 }
+{ "o_orderkey": 224 }
+{ "o_orderkey": 225 }
+{ "o_orderkey": 226 }
+{ "o_orderkey": 227 }
+{ "o_orderkey": 228 }
+{ "o_orderkey": 229 }
+{ "o_orderkey": 230 }
+{ "o_orderkey": 231 }
+{ "o_orderkey": 256 }
+{ "o_orderkey": 257 }
+{ "o_orderkey": 258 }
+{ "o_orderkey": 259 }
+{ "o_orderkey": 260 }
+{ "o_orderkey": 261 }
+{ "o_orderkey": 262 }
+{ "o_orderkey": 263 }
+{ "o_orderkey": 288 }
+{ "o_orderkey": 289 }
+{ "o_orderkey": 290 }
+{ "o_orderkey": 291 }
+{ "o_orderkey": 292 }
+{ "o_orderkey": 293 }
+{ "o_orderkey": 294 }
+{ "o_orderkey": 295 }
+{ "o_orderkey": 320 }
+{ "o_orderkey": 321 }
+{ "o_orderkey": 322 }
+{ "o_orderkey": 323 }
+{ "o_orderkey": 324 }
+{ "o_orderkey": 325 }
+{ "o_orderkey": 326 }
+{ "o_orderkey": 327 }
+{ "o_orderkey": 352 }
+{ "o_orderkey": 353 }
+{ "o_orderkey": 354 }
+{ "o_orderkey": 355 }
+{ "o_orderkey": 356 }
+{ "o_orderkey": 357 }
+{ "o_orderkey": 358 }
+{ "o_orderkey": 359 }
+{ "o_orderkey": 384 }
+{ "o_orderkey": 385 }
+{ "o_orderkey": 386 }
+{ "o_orderkey": 387 }
+{ "o_orderkey": 388 }
+{ "o_orderkey": 389 }
+{ "o_orderkey": 390 }
+{ "o_orderkey": 391 }
+{ "o_orderkey": 416 }
+{ "o_orderkey": 417 }
+{ "o_orderkey": 418 }
+{ "o_orderkey": 419 }
+{ "o_orderkey": 420 }
+{ "o_orderkey": 421 }
+{ "o_orderkey": 422 }
+{ "o_orderkey": 423 }
+{ "o_orderkey": 448 }
+{ "o_orderkey": 449 }
+{ "o_orderkey": 450 }
+{ "o_orderkey": 451 }
+{ "o_orderkey": 452 }
+{ "o_orderkey": 453 }
+{ "o_orderkey": 454 }
+{ "o_orderkey": 455 }
+{ "o_orderkey": 480 }
+{ "o_orderkey": 481 }
+{ "o_orderkey": 482 }
+{ "o_orderkey": 483 }
+{ "o_orderkey": 484 }
+{ "o_orderkey": 485 }
+{ "o_orderkey": 486 }
+{ "o_orderkey": 487 }
+{ "o_orderkey": 512 }
+{ "o_orderkey": 513 }
+{ "o_orderkey": 514 }
+{ "o_orderkey": 515 }
+{ "o_orderkey": 516 }
+{ "o_orderkey": 517 }
+{ "o_orderkey": 518 }
+{ "o_orderkey": 519 }
+{ "o_orderkey": 544 }
+{ "o_orderkey": 545 }
+{ "o_orderkey": 546 }
+{ "o_orderkey": 547 }
+{ "o_orderkey": 548 }
+{ "o_orderkey": 549 }
+{ "o_orderkey": 550 }
+{ "o_orderkey": 551 }
+{ "o_orderkey": 576 }
+{ "o_orderkey": 577 }
+{ "o_orderkey": 578 }
+{ "o_orderkey": 579 }
+{ "o_orderkey": 580 }
+{ "o_orderkey": 581 }
+{ "o_orderkey": 582 }
+{ "o_orderkey": 583 }
+{ "o_orderkey": 608 }
+{ "o_orderkey": 609 }
+{ "o_orderkey": 610 }
+{ "o_orderkey": 611 }
+{ "o_orderkey": 612 }
+{ "o_orderkey": 613 }
+{ "o_orderkey": 614 }
+{ "o_orderkey": 615 }
+{ "o_orderkey": 640 }
+{ "o_orderkey": 641 }
+{ "o_orderkey": 642 }
+{ "o_orderkey": 643 }
+{ "o_orderkey": 644 }
+{ "o_orderkey": 645 }
+{ "o_orderkey": 646 }
+{ "o_orderkey": 647 }
+{ "o_orderkey": 672 }
+{ "o_orderkey": 673 }
+{ "o_orderkey": 674 }
+{ "o_orderkey": 675 }
+{ "o_orderkey": 676 }
+{ "o_orderkey": 677 }
+{ "o_orderkey": 678 }
+{ "o_orderkey": 679 }
+{ "o_orderkey": 704 }
+{ "o_orderkey": 705 }
+{ "o_orderkey": 706 }
+{ "o_orderkey": 707 }
+{ "o_orderkey": 708 }
+{ "o_orderkey": 709 }
+{ "o_orderkey": 710 }
+{ "o_orderkey": 711 }
+{ "o_orderkey": 736 }
+{ "o_orderkey": 737 }
+{ "o_orderkey": 738 }
+{ "o_orderkey": 739 }
+{ "o_orderkey": 740 }
+{ "o_orderkey": 741 }
+{ "o_orderkey": 742 }
+{ "o_orderkey": 743 }
+{ "o_orderkey": 768 }
+{ "o_orderkey": 769 }
+{ "o_orderkey": 770 }
+{ "o_orderkey": 771 }
+{ "o_orderkey": 772 }
+{ "o_orderkey": 773 }
+{ "o_orderkey": 774 }
+{ "o_orderkey": 775 }
+{ "o_orderkey": 800 }
+{ "o_orderkey": 801 }
+{ "o_orderkey": 802 }
+{ "o_orderkey": 803 }
+{ "o_orderkey": 804 }
+{ "o_orderkey": 805 }
+{ "o_orderkey": 806 }
+{ "o_orderkey": 807 }
+{ "o_orderkey": 832 }
+{ "o_orderkey": 833 }
+{ "o_orderkey": 834 }
+{ "o_orderkey": 835 }
+{ "o_orderkey": 836 }
+{ "o_orderkey": 837 }
+{ "o_orderkey": 838 }
+{ "o_orderkey": 839 }
+{ "o_orderkey": 864 }
+{ "o_orderkey": 865 }
+{ "o_orderkey": 866 }
+{ "o_orderkey": 867 }
+{ "o_orderkey": 868 }
+{ "o_orderkey": 869 }
+{ "o_orderkey": 870 }
+{ "o_orderkey": 871 }
+{ "o_orderkey": 896 }
+{ "o_orderkey": 897 }
+{ "o_orderkey": 898 }
+{ "o_orderkey": 899 }
+{ "o_orderkey": 900 }
+{ "o_orderkey": 901 }
+{ "o_orderkey": 902 }
+{ "o_orderkey": 903 }
+{ "o_orderkey": 928 }
+{ "o_orderkey": 929 }
+{ "o_orderkey": 930 }
+{ "o_orderkey": 931 }
+{ "o_orderkey": 932 }
+{ "o_orderkey": 933 }
+{ "o_orderkey": 934 }
+{ "o_orderkey": 935 }
+{ "o_orderkey": 960 }
+{ "o_orderkey": 961 }
+{ "o_orderkey": 962 }
+{ "o_orderkey": 963 }
+{ "o_orderkey": 964 }
+{ "o_orderkey": 965 }
+{ "o_orderkey": 966 }
+{ "o_orderkey": 967 }
+{ "o_orderkey": 992 }
+{ "o_orderkey": 993 }
+{ "o_orderkey": 994 }
+{ "o_orderkey": 995 }
+{ "o_orderkey": 996 }
+{ "o_orderkey": 997 }
+{ "o_orderkey": 998 }
+{ "o_orderkey": 999 }
+{ "o_orderkey": 1024 }
+{ "o_orderkey": 1025 }
+{ "o_orderkey": 1026 }
+{ "o_orderkey": 1027 }
+{ "o_orderkey": 1028 }
+{ "o_orderkey": 1029 }
+{ "o_orderkey": 1030 }
+{ "o_orderkey": 1031 }
+{ "o_orderkey": 1056 }
+{ "o_orderkey": 1057 }
+{ "o_orderkey": 1058 }
+{ "o_orderkey": 1059 }
+{ "o_orderkey": 1060 }
+{ "o_orderkey": 1061 }
+{ "o_orderkey": 1062 }
+{ "o_orderkey": 1063 }
+{ "o_orderkey": 1088 }
+{ "o_orderkey": 1089 }
+{ "o_orderkey": 1090 }
+{ "o_orderkey": 1091 }
+{ "o_orderkey": 1092 }
+{ "o_orderkey": 1093 }
+{ "o_orderkey": 1094 }
+{ "o_orderkey": 1095 }
+{ "o_orderkey": 1120 }
+{ "o_orderkey": 1121 }
+{ "o_orderkey": 1122 }
+{ "o_orderkey": 1123 }
+{ "o_orderkey": 1124 }
+{ "o_orderkey": 1125 }
+{ "o_orderkey": 1126 }
+{ "o_orderkey": 1127 }
+{ "o_orderkey": 1152 }
+{ "o_orderkey": 1153 }
+{ "o_orderkey": 1154 }
+{ "o_orderkey": 1155 }
+{ "o_orderkey": 1156 }
+{ "o_orderkey": 1157 }
+{ "o_orderkey": 1158 }
+{ "o_orderkey": 1159 }
+{ "o_orderkey": 1184 }
+{ "o_orderkey": 1185 }
+{ "o_orderkey": 1186 }
+{ "o_orderkey": 1187 }
+{ "o_orderkey": 1188 }
+{ "o_orderkey": 1189 }
+{ "o_orderkey": 1190 }
+{ "o_orderkey": 1191 }
+{ "o_orderkey": 1216 }
+{ "o_orderkey": 1217 }
+{ "o_orderkey": 1218 }
+{ "o_orderkey": 1219 }
+{ "o_orderkey": 1220 }
+{ "o_orderkey": 1221 }
+{ "o_orderkey": 1222 }
+{ "o_orderkey": 1223 }
+{ "o_orderkey": 1248 }
+{ "o_orderkey": 1249 }
+{ "o_orderkey": 1250 }
+{ "o_orderkey": 1251 }
+{ "o_orderkey": 1252 }
+{ "o_orderkey": 1253 }
+{ "o_orderkey": 1254 }
+{ "o_orderkey": 1255 }
+{ "o_orderkey": 1280 }
+{ "o_orderkey": 1281 }
+{ "o_orderkey": 1282 }
+{ "o_orderkey": 1283 }
+{ "o_orderkey": 1284 }
+{ "o_orderkey": 1285 }
+{ "o_orderkey": 1286 }
+{ "o_orderkey": 1287 }
+{ "o_orderkey": 1312 }
+{ "o_orderkey": 1313 }
+{ "o_orderkey": 1314 }
+{ "o_orderkey": 1315 }
+{ "o_orderkey": 1316 }
+{ "o_orderkey": 1317 }
+{ "o_orderkey": 1318 }
+{ "o_orderkey": 1319 }
+{ "o_orderkey": 1344 }
+{ "o_orderkey": 1345 }
+{ "o_orderkey": 1346 }
+{ "o_orderkey": 1347 }
+{ "o_orderkey": 1348 }
+{ "o_orderkey": 1349 }
+{ "o_orderkey": 1350 }
+{ "o_orderkey": 1351 }
+{ "o_orderkey": 1376 }
+{ "o_orderkey": 1377 }
+{ "o_orderkey": 1378 }
+{ "o_orderkey": 1379 }
+{ "o_orderkey": 1380 }
+{ "o_orderkey": 1381 }
+{ "o_orderkey": 1382 }
+{ "o_orderkey": 1383 }
+{ "o_orderkey": 1408 }
+{ "o_orderkey": 1409 }
+{ "o_orderkey": 1410 }
+{ "o_orderkey": 1411 }
+{ "o_orderkey": 1412 }
+{ "o_orderkey": 1413 }
+{ "o_orderkey": 1414 }
+{ "o_orderkey": 1415 }
+{ "o_orderkey": 1440 }
+{ "o_orderkey": 1441 }
+{ "o_orderkey": 1442 }
+{ "o_orderkey": 1443 }
+{ "o_orderkey": 1444 }
+{ "o_orderkey": 1445 }
+{ "o_orderkey": 1446 }
+{ "o_orderkey": 1447 }
+{ "o_orderkey": 1472 }
+{ "o_orderkey": 1473 }
+{ "o_orderkey": 1474 }
+{ "o_orderkey": 1475 }
+{ "o_orderkey": 1476 }
+{ "o_orderkey": 1477 }
+{ "o_orderkey": 1478 }
+{ "o_orderkey": 1479 }
+{ "o_orderkey": 1504 }
+{ "o_orderkey": 1505 }
+{ "o_orderkey": 1506 }
+{ "o_orderkey": 1507 }
+{ "o_orderkey": 1508 }
+{ "o_orderkey": 1509 }
+{ "o_orderkey": 1510 }
+{ "o_orderkey": 1511 }
+{ "o_orderkey": 1536 }
+{ "o_orderkey": 1537 }
+{ "o_orderkey": 1538 }
+{ "o_orderkey": 1539 }
+{ "o_orderkey": 1540 }
+{ "o_orderkey": 1541 }
+{ "o_orderkey": 1542 }
+{ "o_orderkey": 1543 }
+{ "o_orderkey": 1568 }
+{ "o_orderkey": 1569 }
+{ "o_orderkey": 1570 }
+{ "o_orderkey": 1571 }
+{ "o_orderkey": 1572 }
+{ "o_orderkey": 1573 }
+{ "o_orderkey": 1574 }
+{ "o_orderkey": 1575 }
+{ "o_orderkey": 1600 }
+{ "o_orderkey": 1601 }
+{ "o_orderkey": 1602 }
+{ "o_orderkey": 1603 }
+{ "o_orderkey": 1604 }
+{ "o_orderkey": 1605 }
+{ "o_orderkey": 1606 }
+{ "o_orderkey": 1607 }
+{ "o_orderkey": 1632 }
+{ "o_orderkey": 1633 }
+{ "o_orderkey": 1634 }
+{ "o_orderkey": 1635 }
+{ "o_orderkey": 1636 }
+{ "o_orderkey": 1637 }
+{ "o_orderkey": 1638 }
+{ "o_orderkey": 1639 }
+{ "o_orderkey": 1664 }
+{ "o_orderkey": 1665 }
+{ "o_orderkey": 1666 }
+{ "o_orderkey": 1667 }
+{ "o_orderkey": 1668 }
+{ "o_orderkey": 1669 }
+{ "o_orderkey": 1670 }
+{ "o_orderkey": 1671 }
+{ "o_orderkey": 1696 }
+{ "o_orderkey": 1697 }
+{ "o_orderkey": 1698 }
+{ "o_orderkey": 1699 }
+{ "o_orderkey": 1700 }
+{ "o_orderkey": 1701 }
+{ "o_orderkey": 1702 }
+{ "o_orderkey": 1703 }
+{ "o_orderkey": 1728 }
+{ "o_orderkey": 1729 }
+{ "o_orderkey": 1730 }
+{ "o_orderkey": 1731 }
+{ "o_orderkey": 1732 }
+{ "o_orderkey": 1733 }
+{ "o_orderkey": 1734 }
+{ "o_orderkey": 1735 }
+{ "o_orderkey": 1760 }
+{ "o_orderkey": 1761 }
+{ "o_orderkey": 1762 }
+{ "o_orderkey": 1763 }
+{ "o_orderkey": 1764 }
+{ "o_orderkey": 1765 }
+{ "o_orderkey": 1766 }
+{ "o_orderkey": 1767 }
+{ "o_orderkey": 1792 }
+{ "o_orderkey": 1793 }
+{ "o_orderkey": 1794 }
+{ "o_orderkey": 1795 }
+{ "o_orderkey": 1796 }
+{ "o_orderkey": 1797 }
+{ "o_orderkey": 1798 }
+{ "o_orderkey": 1799 }
+{ "o_orderkey": 1824 }
+{ "o_orderkey": 1825 }
+{ "o_orderkey": 1826 }
+{ "o_orderkey": 1827 }
+{ "o_orderkey": 1828 }
+{ "o_orderkey": 1829 }
+{ "o_orderkey": 1830 }
+{ "o_orderkey": 1831 }
+{ "o_orderkey": 1856 }
+{ "o_orderkey": 1857 }
+{ "o_orderkey": 1858 }
+{ "o_orderkey": 1859 }
+{ "o_orderkey": 1860 }
+{ "o_orderkey": 1861 }
+{ "o_orderkey": 1862 }
+{ "o_orderkey": 1863 }
+{ "o_orderkey": 1888 }
+{ "o_orderkey": 1889 }
+{ "o_orderkey": 1890 }
+{ "o_orderkey": 1891 }
+{ "o_orderkey": 1892 }
+{ "o_orderkey": 1893 }
+{ "o_orderkey": 1894 }
+{ "o_orderkey": 1895 }
+{ "o_orderkey": 1920 }
+{ "o_orderkey": 1921 }
+{ "o_orderkey": 1922 }
+{ "o_orderkey": 1923 }
+{ "o_orderkey": 1924 }
+{ "o_orderkey": 1925 }
+{ "o_orderkey": 1926 }
+{ "o_orderkey": 1927 }
+{ "o_orderkey": 1952 }
+{ "o_orderkey": 1953 }
+{ "o_orderkey": 1954 }
+{ "o_orderkey": 1955 }
+{ "o_orderkey": 1956 }
+{ "o_orderkey": 1957 }
+{ "o_orderkey": 1958 }
+{ "o_orderkey": 1959 }
+{ "o_orderkey": 1984 }
+{ "o_orderkey": 1985 }
+{ "o_orderkey": 1986 }
+{ "o_orderkey": 1987 }
+{ "o_orderkey": 1988 }
+{ "o_orderkey": 1989 }
+{ "o_orderkey": 1990 }
+{ "o_orderkey": 1991 }
+{ "o_orderkey": 2016 }
+{ "o_orderkey": 2017 }
+{ "o_orderkey": 2018 }
+{ "o_orderkey": 2019 }
+{ "o_orderkey": 2020 }
+{ "o_orderkey": 2021 }
+{ "o_orderkey": 2022 }
+{ "o_orderkey": 2023 }
+{ "o_orderkey": 2048 }
+{ "o_orderkey": 2049 }
+{ "o_orderkey": 2050 }
+{ "o_orderkey": 2051 }
+{ "o_orderkey": 2052 }
+{ "o_orderkey": 2053 }
+{ "o_orderkey": 2054 }
+{ "o_orderkey": 2055 }
+{ "o_orderkey": 2080 }
+{ "o_orderkey": 2081 }
+{ "o_orderkey": 2082 }
+{ "o_orderkey": 2083 }
+{ "o_orderkey": 2084 }
+{ "o_orderkey": 2085 }
+{ "o_orderkey": 2086 }
+{ "o_orderkey": 2087 }
+{ "o_orderkey": 2112 }
+{ "o_orderkey": 2113 }
+{ "o_orderkey": 2114 }
+{ "o_orderkey": 2115 }
+{ "o_orderkey": 2116 }
+{ "o_orderkey": 2117 }
+{ "o_orderkey": 2118 }
+{ "o_orderkey": 2119 }
+{ "o_orderkey": 2144 }
+{ "o_orderkey": 2145 }
+{ "o_orderkey": 2146 }
+{ "o_orderkey": 2147 }
+{ "o_orderkey": 2148 }
+{ "o_orderkey": 2149 }
+{ "o_orderkey": 2150 }
+{ "o_orderkey": 2151 }
+{ "o_orderkey": 2176 }
+{ "o_orderkey": 2177 }
+{ "o_orderkey": 2178 }
+{ "o_orderkey": 2179 }
+{ "o_orderkey": 2180 }
+{ "o_orderkey": 2181 }
+{ "o_orderkey": 2182 }
+{ "o_orderkey": 2183 }
+{ "o_orderkey": 2208 }
+{ "o_orderkey": 2209 }
+{ "o_orderkey": 2210 }
+{ "o_orderkey": 2211 }
+{ "o_orderkey": 2212 }
+{ "o_orderkey": 2213 }
+{ "o_orderkey": 2214 }
+{ "o_orderkey": 2215 }
+{ "o_orderkey": 2240 }
+{ "o_orderkey": 2241 }
+{ "o_orderkey": 2242 }
+{ "o_orderkey": 2243 }
+{ "o_orderkey": 2244 }
+{ "o_orderkey": 2245 }
+{ "o_orderkey": 2246 }
+{ "o_orderkey": 2247 }
+{ "o_orderkey": 2272 }
+{ "o_orderkey": 2273 }
+{ "o_orderkey": 2274 }
+{ "o_orderkey": 2275 }
+{ "o_orderkey": 2276 }
+{ "o_orderkey": 2277 }
+{ "o_orderkey": 2278 }
+{ "o_orderkey": 2279 }
+{ "o_orderkey": 2304 }
+{ "o_orderkey": 2305 }
+{ "o_orderkey": 2306 }
+{ "o_orderkey": 2307 }
+{ "o_orderkey": 2308 }
+{ "o_orderkey": 2309 }
+{ "o_orderkey": 2310 }
+{ "o_orderkey": 2311 }
+{ "o_orderkey": 2336 }
+{ "o_orderkey": 2337 }
+{ "o_orderkey": 2338 }
+{ "o_orderkey": 2339 }
+{ "o_orderkey": 2340 }
+{ "o_orderkey": 2341 }
+{ "o_orderkey": 2342 }
+{ "o_orderkey": 2343 }
+{ "o_orderkey": 2368 }
+{ "o_orderkey": 2369 }
+{ "o_orderkey": 2370 }
+{ "o_orderkey": 2371 }
+{ "o_orderkey": 2372 }
+{ "o_orderkey": 2373 }
+{ "o_orderkey": 2374 }
+{ "o_orderkey": 2375 }
+{ "o_orderkey": 2400 }
+{ "o_orderkey": 2401 }
+{ "o_orderkey": 2402 }
+{ "o_orderkey": 2403 }
+{ "o_orderkey": 2404 }
+{ "o_orderkey": 2405 }
+{ "o_orderkey": 2406 }
+{ "o_orderkey": 2407 }
+{ "o_orderkey": 2432 }
+{ "o_orderkey": 2433 }
+{ "o_orderkey": 2434 }
+{ "o_orderkey": 2435 }
+{ "o_orderkey": 2436 }
+{ "o_orderkey": 2437 }
+{ "o_orderkey": 2438 }
+{ "o_orderkey": 2439 }
+{ "o_orderkey": 2464 }
+{ "o_orderkey": 2465 }
+{ "o_orderkey": 2466 }
+{ "o_orderkey": 2467 }
+{ "o_orderkey": 2468 }
+{ "o_orderkey": 2469 }
+{ "o_orderkey": 2470 }
+{ "o_orderkey": 2471 }
+{ "o_orderkey": 2496 }
+{ "o_orderkey": 2497 }
+{ "o_orderkey": 2498 }
+{ "o_orderkey": 2499 }
+{ "o_orderkey": 2500 }
+{ "o_orderkey": 2501 }
+{ "o_orderkey": 2502 }
+{ "o_orderkey": 2503 }
+{ "o_orderkey": 2528 }
+{ "o_orderkey": 2529 }
+{ "o_orderkey": 2530 }
+{ "o_orderkey": 2531 }
+{ "o_orderkey": 2532 }
+{ "o_orderkey": 2533 }
+{ "o_orderkey": 2534 }
+{ "o_orderkey": 2535 }
+{ "o_orderkey": 2560 }
+{ "o_orderkey": 2561 }
+{ "o_orderkey": 2562 }
+{ "o_orderkey": 2563 }
+{ "o_orderkey": 2564 }
+{ "o_orderkey": 2565 }
+{ "o_orderkey": 2566 }
+{ "o_orderkey": 2567 }
+{ "o_orderkey": 2592 }
+{ "o_orderkey": 2593 }
+{ "o_orderkey": 2594 }
+{ "o_orderkey": 2595 }
+{ "o_orderkey": 2596 }
+{ "o_orderkey": 2597 }
+{ "o_orderkey": 2598 }
+{ "o_orderkey": 2599 }
+{ "o_orderkey": 2624 }
+{ "o_orderkey": 2625 }
+{ "o_orderkey": 2626 }
+{ "o_orderkey": 2627 }
+{ "o_orderkey": 2628 }
+{ "o_orderkey": 2629 }
+{ "o_orderkey": 2630 }
+{ "o_orderkey": 2631 }
+{ "o_orderkey": 2656 }
+{ "o_orderkey": 2657 }
+{ "o_orderkey": 2658 }
+{ "o_orderkey": 2659 }
+{ "o_orderkey": 2660 }
+{ "o_orderkey": 2661 }
+{ "o_orderkey": 2662 }
+{ "o_orderkey": 2663 }
+{ "o_orderkey": 2688 }
+{ "o_orderkey": 2689 }
+{ "o_orderkey": 2690 }
+{ "o_orderkey": 2691 }
+{ "o_orderkey": 2692 }
+{ "o_orderkey": 2693 }
+{ "o_orderkey": 2694 }
+{ "o_orderkey": 2695 }
+{ "o_orderkey": 2720 }
+{ "o_orderkey": 2721 }
+{ "o_orderkey": 2722 }
+{ "o_orderkey": 2723 }
+{ "o_orderkey": 2724 }
+{ "o_orderkey": 2725 }
+{ "o_orderkey": 2726 }
+{ "o_orderkey": 2727 }
+{ "o_orderkey": 2752 }
+{ "o_orderkey": 2753 }
+{ "o_orderkey": 2754 }
+{ "o_orderkey": 2755 }
+{ "o_orderkey": 2756 }
+{ "o_orderkey": 2757 }
+{ "o_orderkey": 2758 }
+{ "o_orderkey": 2759 }
+{ "o_orderkey": 2784 }
+{ "o_orderkey": 2785 }
+{ "o_orderkey": 2786 }
+{ "o_orderkey": 2787 }
+{ "o_orderkey": 2788 }
+{ "o_orderkey": 2789 }
+{ "o_orderkey": 2790 }
+{ "o_orderkey": 2791 }
+{ "o_orderkey": 2816 }
+{ "o_orderkey": 2817 }
+{ "o_orderkey": 2818 }
+{ "o_orderkey": 2819 }
+{ "o_orderkey": 2820 }
+{ "o_orderkey": 2821 }
+{ "o_orderkey": 2822 }
+{ "o_orderkey": 2823 }
+{ "o_orderkey": 2848 }
+{ "o_orderkey": 2849 }
+{ "o_orderkey": 2850 }
+{ "o_orderkey": 2851 }
+{ "o_orderkey": 2852 }
+{ "o_orderkey": 2853 }
+{ "o_orderkey": 2854 }
+{ "o_orderkey": 2855 }
+{ "o_orderkey": 2880 }
+{ "o_orderkey": 2881 }
+{ "o_orderkey": 2882 }
+{ "o_orderkey": 2883 }
+{ "o_orderkey": 2884 }
+{ "o_orderkey": 2885 }
+{ "o_orderkey": 2886 }
+{ "o_orderkey": 2887 }
+{ "o_orderkey": 2912 }
+{ "o_orderkey": 2913 }
+{ "o_orderkey": 2914 }
+{ "o_orderkey": 2915 }
+{ "o_orderkey": 2916 }
+{ "o_orderkey": 2917 }
+{ "o_orderkey": 2918 }
+{ "o_orderkey": 2919 }
+{ "o_orderkey": 2944 }
+{ "o_orderkey": 2945 }
+{ "o_orderkey": 2946 }
+{ "o_orderkey": 2947 }
+{ "o_orderkey": 2948 }
+{ "o_orderkey": 2949 }
+{ "o_orderkey": 2950 }
+{ "o_orderkey": 2951 }
+{ "o_orderkey": 2976 }
+{ "o_orderkey": 2977 }
+{ "o_orderkey": 2978 }
+{ "o_orderkey": 2979 }
+{ "o_orderkey": 2980 }
+{ "o_orderkey": 2981 }
+{ "o_orderkey": 2982 }
+{ "o_orderkey": 2983 }
+{ "o_orderkey": 3008 }
+{ "o_orderkey": 3009 }
+{ "o_orderkey": 3010 }
+{ "o_orderkey": 3011 }
+{ "o_orderkey": 3012 }
+{ "o_orderkey": 3013 }
+{ "o_orderkey": 3014 }
+{ "o_orderkey": 3015 }
+{ "o_orderkey": 3040 }
+{ "o_orderkey": 3041 }
+{ "o_orderkey": 3042 }
+{ "o_orderkey": 3043 }
+{ "o_orderkey": 3044 }
+{ "o_orderkey": 3045 }
+{ "o_orderkey": 3046 }
+{ "o_orderkey": 3047 }
+{ "o_orderkey": 3072 }
+{ "o_orderkey": 3073 }
+{ "o_orderkey": 3074 }
+{ "o_orderkey": 3075 }
+{ "o_orderkey": 3076 }
+{ "o_orderkey": 3077 }
+{ "o_orderkey": 3078 }
+{ "o_orderkey": 3079 }
+{ "o_orderkey": 3104 }
+{ "o_orderkey": 3105 }
+{ "o_orderkey": 3106 }
+{ "o_orderkey": 3107 }
+{ "o_orderkey": 3108 }
+{ "o_orderkey": 3109 }
+{ "o_orderkey": 3110 }
+{ "o_orderkey": 3111 }
+{ "o_orderkey": 3136 }
+{ "o_orderkey": 3137 }
+{ "o_orderkey": 3138 }
+{ "o_orderkey": 3139 }
+{ "o_orderkey": 3140 }
+{ "o_orderkey": 3141 }
+{ "o_orderkey": 3142 }
+{ "o_orderkey": 3143 }
+{ "o_orderkey": 3168 }
+{ "o_orderkey": 3169 }
+{ "o_orderkey": 3170 }
+{ "o_orderkey": 3171 }
+{ "o_orderkey": 3172 }
+{ "o_orderkey": 3173 }
+{ "o_orderkey": 3174 }
+{ "o_orderkey": 3175 }
+{ "o_orderkey": 3200 }
+{ "o_orderkey": 3201 }
+{ "o_orderkey": 3202 }
+{ "o_orderkey": 3203 }
+{ "o_orderkey": 3204 }
+{ "o_orderkey": 3205 }
+{ "o_orderkey": 3206 }
+{ "o_orderkey": 3207 }
+{ "o_orderkey": 3232 }
+{ "o_orderkey": 3233 }
+{ "o_orderkey": 3234 }
+{ "o_orderkey": 3235 }
+{ "o_orderkey": 3236 }
+{ "o_orderkey": 3237 }
+{ "o_orderkey": 3238 }
+{ "o_orderkey": 3239 }
+{ "o_orderkey": 3264 }
+{ "o_orderkey": 3265 }
+{ "o_orderkey": 3266 }
+{ "o_orderkey": 3267 }
+{ "o_orderkey": 3268 }
+{ "o_orderkey": 3269 }
+{ "o_orderkey": 3270 }
+{ "o_orderkey": 3271 }
+{ "o_orderkey": 3296 }
+{ "o_orderkey": 3297 }
+{ "o_orderkey": 3298 }
+{ "o_orderkey": 3299 }
+{ "o_orderkey": 3300 }
+{ "o_orderkey": 3301 }
+{ "o_orderkey": 3302 }
+{ "o_orderkey": 3303 }
+{ "o_orderkey": 3328 }
+{ "o_orderkey": 3329 }
+{ "o_orderkey": 3330 }
+{ "o_orderkey": 3331 }
+{ "o_orderkey": 3332 }
+{ "o_orderkey": 3333 }
+{ "o_orderkey": 3334 }
+{ "o_orderkey": 3335 }
+{ "o_orderkey": 3360 }
+{ "o_orderkey": 3361 }
+{ "o_orderkey": 3362 }
+{ "o_orderkey": 3363 }
+{ "o_orderkey": 3364 }
+{ "o_orderkey": 3365 }
+{ "o_orderkey": 3366 }
+{ "o_orderkey": 3367 }
+{ "o_orderkey": 3392 }
+{ "o_orderkey": 3393 }
+{ "o_orderkey": 3394 }
+{ "o_orderkey": 3395 }
+{ "o_orderkey": 3396 }
+{ "o_orderkey": 3397 }
+{ "o_orderkey": 3398 }
+{ "o_orderkey": 3399 }
+{ "o_orderkey": 3424 }
+{ "o_orderkey": 3425 }
+{ "o_orderkey": 3426 }
+{ "o_orderkey": 3427 }
+{ "o_orderkey": 3428 }
+{ "o_orderkey": 3429 }
+{ "o_orderkey": 3430 }
+{ "o_orderkey": 3431 }
+{ "o_orderkey": 3456 }
+{ "o_orderkey": 3457 }
+{ "o_orderkey": 3458 }
+{ "o_orderkey": 3459 }
+{ "o_orderkey": 3460 }
+{ "o_orderkey": 3461 }
+{ "o_orderkey": 3462 }
+{ "o_orderkey": 3463 }
+{ "o_orderkey": 3488 }
+{ "o_orderkey": 3489 }
+{ "o_orderkey": 3490 }
+{ "o_orderkey": 3491 }
+{ "o_orderkey": 3492 }
+{ "o_orderkey": 3493 }
+{ "o_orderkey": 3494 }
+{ "o_orderkey": 3495 }
+{ "o_orderkey": 3520 }
+{ "o_orderkey": 3521 }
+{ "o_orderkey": 3522 }
+{ "o_orderkey": 3523 }
+{ "o_orderkey": 3524 }
+{ "o_orderkey": 3525 }
+{ "o_orderkey": 3526 }
+{ "o_orderkey": 3527 }
+{ "o_orderkey": 3552 }
+{ "o_orderkey": 3553 }
+{ "o_orderkey": 3554 }
+{ "o_orderkey": 3555 }
+{ "o_orderkey": 3556 }
+{ "o_orderkey": 3557 }
+{ "o_orderkey": 3558 }
+{ "o_orderkey": 3559 }
+{ "o_orderkey": 3584 }
+{ "o_orderkey": 3585 }
+{ "o_orderkey": 3586 }
+{ "o_orderkey": 3587 }
+{ "o_orderkey": 3588 }
+{ "o_orderkey": 3589 }
+{ "o_orderkey": 3590 }
+{ "o_orderkey": 3591 }
+{ "o_orderkey": 3616 }
+{ "o_orderkey": 3617 }
+{ "o_orderkey": 3618 }
+{ "o_orderkey": 3619 }
+{ "o_orderkey": 3620 }
+{ "o_orderkey": 3621 }
+{ "o_orderkey": 3622 }
+{ "o_orderkey": 3623 }
+{ "o_orderkey": 3648 }
+{ "o_orderkey": 3649 }
+{ "o_orderkey": 3650 }
+{ "o_orderkey": 3651 }
+{ "o_orderkey": 3652 }
+{ "o_orderkey": 3653 }
+{ "o_orderkey": 3654 }
+{ "o_orderkey": 3655 }
+{ "o_orderkey": 3680 }
+{ "o_orderkey": 3681 }
+{ "o_orderkey": 3682 }
+{ "o_orderkey": 3683 }
+{ "o_orderkey": 3684 }
+{ "o_orderkey": 3685 }
+{ "o_orderkey": 3686 }
+{ "o_orderkey": 3687 }
+{ "o_orderkey": 3712 }
+{ "o_orderkey": 3713 }
+{ "o_orderkey": 3714 }
+{ "o_orderkey": 3715 }
+{ "o_orderkey": 3716 }
+{ "o_orderkey": 3717 }
+{ "o_orderkey": 3718 }
+{ "o_orderkey": 3719 }
+{ "o_orderkey": 3744 }
+{ "o_orderkey": 3745 }
+{ "o_orderkey": 3746 }
+{ "o_orderkey": 3747 }
+{ "o_orderkey": 3748 }
+{ "o_orderkey": 3749 }
+{ "o_orderkey": 3750 }
+{ "o_orderkey": 3751 }
+{ "o_orderkey": 3776 }
+{ "o_orderkey": 3777 }
+{ "o_orderkey": 3778 }
+{ "o_orderkey": 3779 }
+{ "o_orderkey": 3780 }
+{ "o_orderkey": 3781 }
+{ "o_orderkey": 3782 }
+{ "o_orderkey": 3783 }
+{ "o_orderkey": 3808 }
+{ "o_orderkey": 3809 }
+{ "o_orderkey": 3810 }
+{ "o_orderkey": 3811 }
+{ "o_orderkey": 3812 }
+{ "o_orderkey": 3813 }
+{ "o_orderkey": 3814 }
+{ "o_orderkey": 3815 }
+{ "o_orderkey": 3840 }
+{ "o_orderkey": 3841 }
+{ "o_orderkey": 3842 }
+{ "o_orderkey": 3843 }
+{ "o_orderkey": 3844 }
+{ "o_orderkey": 3845 }
+{ "o_orderkey": 3846 }
+{ "o_orderkey": 3847 }
+{ "o_orderkey": 3872 }
+{ "o_orderkey": 3873 }
+{ "o_orderkey": 3874 }
+{ "o_orderkey": 3875 }
+{ "o_orderkey": 3876 }
+{ "o_orderkey": 3877 }
+{ "o_orderkey": 3878 }
+{ "o_orderkey": 3879 }
+{ "o_orderkey": 3904 }
+{ "o_orderkey": 3905 }
+{ "o_orderkey": 3906 }
+{ "o_orderkey": 3907 }
+{ "o_orderkey": 3908 }
+{ "o_orderkey": 3909 }
+{ "o_orderkey": 3910 }
+{ "o_orderkey": 3911 }
+{ "o_orderkey": 3936 }
+{ "o_orderkey": 3937 }
+{ "o_orderkey": 3938 }
+{ "o_orderkey": 3939 }
+{ "o_orderkey": 3940 }
+{ "o_orderkey": 3941 }
+{ "o_orderkey": 3942 }
+{ "o_orderkey": 3943 }
+{ "o_orderkey": 3968 }
+{ "o_orderkey": 3969 }
+{ "o_orderkey": 3970 }
+{ "o_orderkey": 3971 }
+{ "o_orderkey": 3972 }
+{ "o_orderkey": 3973 }
+{ "o_orderkey": 3974 }
+{ "o_orderkey": 3975 }
+{ "o_orderkey": 4000 }
+{ "o_orderkey": 4001 }
+{ "o_orderkey": 4002 }
+{ "o_orderkey": 4003 }
+{ "o_orderkey": 4004 }
+{ "o_orderkey": 4005 }
+{ "o_orderkey": 4006 }
+{ "o_orderkey": 4007 }
+{ "o_orderkey": 4032 }
+{ "o_orderkey": 4033 }
+{ "o_orderkey": 4034 }
+{ "o_orderkey": 4035 }
+{ "o_orderkey": 4036 }
+{ "o_orderkey": 4037 }
+{ "o_orderkey": 4038 }
+{ "o_orderkey": 4039 }
+{ "o_orderkey": 4064 }
+{ "o_orderkey": 4065 }
+{ "o_orderkey": 4066 }
+{ "o_orderkey": 4067 }
+{ "o_orderkey": 4068 }
+{ "o_orderkey": 4069 }
+{ "o_orderkey": 4070 }
+{ "o_orderkey": 4071 }
+{ "o_orderkey": 4096 }
+{ "o_orderkey": 4097 }
+{ "o_orderkey": 4098 }
+{ "o_orderkey": 4099 }
+{ "o_orderkey": 4100 }
+{ "o_orderkey": 4101 }
+{ "o_orderkey": 4102 }
+{ "o_orderkey": 4103 }
+{ "o_orderkey": 4128 }
+{ "o_orderkey": 4129 }
+{ "o_orderkey": 4130 }
+{ "o_orderkey": 4131 }
+{ "o_orderkey": 4132 }
+{ "o_orderkey": 4133 }
+{ "o_orderkey": 4134 }
+{ "o_orderkey": 4135 }
+{ "o_orderkey": 4160 }
+{ "o_orderkey": 4161 }
+{ "o_orderkey": 4162 }
+{ "o_orderkey": 4163 }
+{ "o_orderkey": 4164 }
+{ "o_orderkey": 4165 }
+{ "o_orderkey": 4166 }
+{ "o_orderkey": 4167 }
+{ "o_orderkey": 4192 }
+{ "o_orderkey": 4193 }
+{ "o_orderkey": 4194 }
+{ "o_orderkey": 4195 }
+{ "o_orderkey": 4196 }
+{ "o_orderkey": 4197 }
+{ "o_orderkey": 4198 }
+{ "o_orderkey": 4199 }
+{ "o_orderkey": 4224 }
+{ "o_orderkey": 4225 }
+{ "o_orderkey": 4226 }
+{ "o_orderkey": 4227 }
+{ "o_orderkey": 4228 }
+{ "o_orderkey": 4229 }
+{ "o_orderkey": 4230 }
+{ "o_orderkey": 4231 }
+{ "o_orderkey": 4256 }
+{ "o_orderkey": 4257 }
+{ "o_orderkey": 4258 }
+{ "o_orderkey": 4259 }
+{ "o_orderkey": 4260 }
+{ "o_orderkey": 4261 }
+{ "o_orderkey": 4262 }
+{ "o_orderkey": 4263 }
+{ "o_orderkey": 4288 }
+{ "o_orderkey": 4289 }
+{ "o_orderkey": 4290 }
+{ "o_orderkey": 4291 }
+{ "o_orderkey": 4292 }
+{ "o_orderkey": 4293 }
+{ "o_orderkey": 4294 }
+{ "o_orderkey": 4295 }
+{ "o_orderkey": 4320 }
+{ "o_orderkey": 4321 }
+{ "o_orderkey": 4322 }
+{ "o_orderkey": 4323 }
+{ "o_orderkey": 4324 }
+{ "o_orderkey": 4325 }
+{ "o_orderkey": 4326 }
+{ "o_orderkey": 4327 }
+{ "o_orderkey": 4352 }
+{ "o_orderkey": 4353 }
+{ "o_orderkey": 4354 }
+{ "o_orderkey": 4355 }
+{ "o_orderkey": 4356 }
+{ "o_orderkey": 4357 }
+{ "o_orderkey": 4358 }
+{ "o_orderkey": 4359 }
+{ "o_orderkey": 4384 }
+{ "o_orderkey": 4385 }
+{ "o_orderkey": 4386 }
+{ "o_orderkey": 4387 }
+{ "o_orderkey": 4388 }
+{ "o_orderkey": 4389 }
+{ "o_orderkey": 4390 }
+{ "o_orderkey": 4391 }
+{ "o_orderkey": 4416 }
+{ "o_orderkey": 4417 }
+{ "o_orderkey": 4418 }
+{ "o_orderkey": 4419 }
+{ "o_orderkey": 4420 }
+{ "o_orderkey": 4421 }
+{ "o_orderkey": 4422 }
+{ "o_orderkey": 4423 }
+{ "o_orderkey": 4448 }
+{ "o_orderkey": 4449 }
+{ "o_orderkey": 4450 }
+{ "o_orderkey": 4451 }
+{ "o_orderkey": 4452 }
+{ "o_orderkey": 4453 }
+{ "o_orderkey": 4454 }
+{ "o_orderkey": 4455 }
+{ "o_orderkey": 4480 }
+{ "o_orderkey": 4481 }
+{ "o_orderkey": 4482 }
+{ "o_orderkey": 4483 }
+{ "o_orderkey": 4484 }
+{ "o_orderkey": 4485 }
+{ "o_orderkey": 4486 }
+{ "o_orderkey": 4487 }
+{ "o_orderkey": 4512 }
+{ "o_orderkey": 4513 }
+{ "o_orderkey": 4514 }
+{ "o_orderkey": 4515 }
+{ "o_orderkey": 4516 }
+{ "o_orderkey": 4517 }
+{ "o_orderkey": 4518 }
+{ "o_orderkey": 4519 }
+{ "o_orderkey": 4544 }
+{ "o_orderkey": 4545 }
+{ "o_orderkey": 4546 }
+{ "o_orderkey": 4547 }
+{ "o_orderkey": 4548 }
+{ "o_orderkey": 4549 }
+{ "o_orderkey": 4550 }
+{ "o_orderkey": 4551 }
+{ "o_orderkey": 4576 }
+{ "o_orderkey": 4577 }
+{ "o_orderkey": 4578 }
+{ "o_orderkey": 4579 }
+{ "o_orderkey": 4580 }
+{ "o_orderkey": 4581 }
+{ "o_orderkey": 4582 }
+{ "o_orderkey": 4583 }
+{ "o_orderkey": 4608 }
+{ "o_orderkey": 4609 }
+{ "o_orderkey": 4610 }
+{ "o_orderkey": 4611 }
+{ "o_orderkey": 4612 }
+{ "o_orderkey": 4613 }
+{ "o_orderkey": 4614 }
+{ "o_orderkey": 4615 }
+{ "o_orderkey": 4640 }
+{ "o_orderkey": 4641 }
+{ "o_orderkey": 4642 }
+{ "o_orderkey": 4643 }
+{ "o_orderkey": 4644 }
+{ "o_orderkey": 4645 }
+{ "o_orderkey": 4646 }
+{ "o_orderkey": 4647 }
+{ "o_orderkey": 4672 }
+{ "o_orderkey": 4673 }
+{ "o_orderkey": 4674 }
+{ "o_orderkey": 4675 }
+{ "o_orderkey": 4676 }
+{ "o_orderkey": 4677 }
+{ "o_orderkey": 4678 }
+{ "o_orderkey": 4679 }
+{ "o_orderkey": 4704 }
+{ "o_orderkey": 4705 }
+{ "o_orderkey": 4706 }
+{ "o_orderkey": 4707 }
+{ "o_orderkey": 4708 }
+{ "o_orderkey": 4709 }
+{ "o_orderkey": 4710 }
+{ "o_orderkey": 4711 }
+{ "o_orderkey": 4736 }
+{ "o_orderkey": 4737 }
+{ "o_orderkey": 4738 }
+{ "o_orderkey": 4739 }
+{ "o_orderkey": 4740 }
+{ "o_orderkey": 4741 }
+{ "o_orderkey": 4742 }
+{ "o_orderkey": 4743 }
+{ "o_orderkey": 4768 }
+{ "o_orderkey": 4769 }
+{ "o_orderkey": 4770 }
+{ "o_orderkey": 4771 }
+{ "o_orderkey": 4772 }
+{ "o_orderkey": 4773 }
+{ "o_orderkey": 4774 }
+{ "o_orderkey": 4775 }
+{ "o_orderkey": 4800 }
+{ "o_orderkey": 4801 }
+{ "o_orderkey": 4802 }
+{ "o_orderkey": 4803 }
+{ "o_orderkey": 4804 }
+{ "o_orderkey": 4805 }
+{ "o_orderkey": 4806 }
+{ "o_orderkey": 4807 }
+{ "o_orderkey": 4832 }
+{ "o_orderkey": 4833 }
+{ "o_orderkey": 4834 }
+{ "o_orderkey": 4835 }
+{ "o_orderkey": 4836 }
+{ "o_orderkey": 4837 }
+{ "o_orderkey": 4838 }
+{ "o_orderkey": 4839 }
+{ "o_orderkey": 4864 }
+{ "o_orderkey": 4865 }
+{ "o_orderkey": 4866 }
+{ "o_orderkey": 4867 }
+{ "o_orderkey": 4868 }
+{ "o_orderkey": 4869 }
+{ "o_orderkey": 4870 }
+{ "o_orderkey": 4871 }
+{ "o_orderkey": 4896 }
+{ "o_orderkey": 4897 }
+{ "o_orderkey": 4898 }
+{ "o_orderkey": 4899 }
+{ "o_orderkey": 4900 }
+{ "o_orderkey": 4901 }
+{ "o_orderkey": 4902 }
+{ "o_orderkey": 4903 }
+{ "o_orderkey": 4928 }
+{ "o_orderkey": 4929 }
+{ "o_orderkey": 4930 }
+{ "o_orderkey": 4931 }
+{ "o_orderkey": 4932 }
+{ "o_orderkey": 4933 }
+{ "o_orderkey": 4934 }
+{ "o_orderkey": 4935 }
+{ "o_orderkey": 4960 }
+{ "o_orderkey": 4961 }
+{ "o_orderkey": 4962 }
+{ "o_orderkey": 4963 }
+{ "o_orderkey": 4964 }
+{ "o_orderkey": 4965 }
+{ "o_orderkey": 4966 }
+{ "o_orderkey": 4967 }
+{ "o_orderkey": 4992 }
+{ "o_orderkey": 4993 }
+{ "o_orderkey": 4994 }
+{ "o_orderkey": 4995 }
+{ "o_orderkey": 4996 }
+{ "o_orderkey": 4997 }
+{ "o_orderkey": 4998 }
+{ "o_orderkey": 4999 }
+{ "o_orderkey": 5024 }
+{ "o_orderkey": 5025 }
+{ "o_orderkey": 5026 }
+{ "o_orderkey": 5027 }
+{ "o_orderkey": 5028 }
+{ "o_orderkey": 5029 }
+{ "o_orderkey": 5030 }
+{ "o_orderkey": 5031 }
+{ "o_orderkey": 5056 }
+{ "o_orderkey": 5057 }
+{ "o_orderkey": 5058 }
+{ "o_orderkey": 5059 }
+{ "o_orderkey": 5060 }
+{ "o_orderkey": 5061 }
+{ "o_orderkey": 5062 }
+{ "o_orderkey": 5063 }
+{ "o_orderkey": 5088 }
+{ "o_orderkey": 5089 }
+{ "o_orderkey": 5090 }
+{ "o_orderkey": 5091 }
+{ "o_orderkey": 5092 }
+{ "o_orderkey": 5093 }
+{ "o_orderkey": 5094 }
+{ "o_orderkey": 5095 }
+{ "o_orderkey": 5120 }
+{ "o_orderkey": 5121 }
+{ "o_orderkey": 5122 }
+{ "o_orderkey": 5123 }
+{ "o_orderkey": 5124 }
+{ "o_orderkey": 5125 }
+{ "o_orderkey": 5126 }
+{ "o_orderkey": 5127 }
+{ "o_orderkey": 5152 }
+{ "o_orderkey": 5153 }
+{ "o_orderkey": 5154 }
+{ "o_orderkey": 5155 }
+{ "o_orderkey": 5156 }
+{ "o_orderkey": 5157 }
+{ "o_orderkey": 5158 }
+{ "o_orderkey": 5159 }
+{ "o_orderkey": 5184 }
+{ "o_orderkey": 5185 }
+{ "o_orderkey": 5186 }
+{ "o_orderkey": 5187 }
+{ "o_orderkey": 5188 }
+{ "o_orderkey": 5189 }
+{ "o_orderkey": 5190 }
+{ "o_orderkey": 5191 }
+{ "o_orderkey": 5216 }
+{ "o_orderkey": 5217 }
+{ "o_orderkey": 5218 }
+{ "o_orderkey": 5219 }
+{ "o_orderkey": 5220 }
+{ "o_orderkey": 5221 }
+{ "o_orderkey": 5222 }
+{ "o_orderkey": 5223 }
+{ "o_orderkey": 5248 }
+{ "o_orderkey": 5249 }
+{ "o_orderkey": 5250 }
+{ "o_orderkey": 5251 }
+{ "o_orderkey": 5252 }
+{ "o_orderkey": 5253 }
+{ "o_orderkey": 5254 }
+{ "o_orderkey": 5255 }
+{ "o_orderkey": 5280 }
+{ "o_orderkey": 5281 }
+{ "o_orderkey": 5282 }
+{ "o_orderkey": 5283 }
+{ "o_orderkey": 5284 }
+{ "o_orderkey": 5285 }
+{ "o_orderkey": 5286 }
+{ "o_orderkey": 5287 }
+{ "o_orderkey": 5312 }
+{ "o_orderkey": 5313 }
+{ "o_orderkey": 5314 }
+{ "o_orderkey": 5315 }
+{ "o_orderkey": 5316 }
+{ "o_orderkey": 5317 }
+{ "o_orderkey": 5318 }
+{ "o_orderkey": 5319 }
+{ "o_orderkey": 5344 }
+{ "o_orderkey": 5345 }
+{ "o_orderkey": 5346 }
+{ "o_orderkey": 5347 }
+{ "o_orderkey": 5348 }
+{ "o_orderkey": 5349 }
+{ "o_orderkey": 5350 }
+{ "o_orderkey": 5351 }
+{ "o_orderkey": 5376 }
+{ "o_orderkey": 5377 }
+{ "o_orderkey": 5378 }
+{ "o_orderkey": 5379 }
+{ "o_orderkey": 5380 }
+{ "o_orderkey": 5381 }
+{ "o_orderkey": 5382 }
+{ "o_orderkey": 5383 }
+{ "o_orderkey": 5408 }
+{ "o_orderkey": 5409 }
+{ "o_orderkey": 5410 }
+{ "o_orderkey": 5411 }
+{ "o_orderkey": 5412 }
+{ "o_orderkey": 5413 }
+{ "o_orderkey": 5414 }
+{ "o_orderkey": 5415 }
+{ "o_orderkey": 5440 }
+{ "o_orderkey": 5441 }
+{ "o_orderkey": 5442 }
+{ "o_orderkey": 5443 }
+{ "o_orderkey": 5444 }
+{ "o_orderkey": 5445 }
+{ "o_orderkey": 5446 }
+{ "o_orderkey": 5447 }
+{ "o_orderkey": 5472 }
+{ "o_orderkey": 5473 }
+{ "o_orderkey": 5474 }
+{ "o_orderkey": 5475 }
+{ "o_orderkey": 5476 }
+{ "o_orderkey": 5477 }
+{ "o_orderkey": 5478 }
+{ "o_orderkey": 5479 }
+{ "o_orderkey": 5504 }
+{ "o_orderkey": 5505 }
+{ "o_orderkey": 5506 }
+{ "o_orderkey": 5507 }
+{ "o_orderkey": 5508 }
+{ "o_orderkey": 5509 }
+{ "o_orderkey": 5510 }
+{ "o_orderkey": 5511 }
+{ "o_orderkey": 5536 }
+{ "o_orderkey": 5537 }
+{ "o_orderkey": 5538 }
+{ "o_orderkey": 5539 }
+{ "o_orderkey": 5540 }
+{ "o_orderkey": 5541 }
+{ "o_orderkey": 5542 }
+{ "o_orderkey": 5543 }
+{ "o_orderkey": 5568 }
+{ "o_orderkey": 5569 }
+{ "o_orderkey": 5570 }
+{ "o_orderkey": 5571 }
+{ "o_orderkey": 5572 }
+{ "o_orderkey": 5573 }
+{ "o_orderkey": 5574 }
+{ "o_orderkey": 5575 }
+{ "o_orderkey": 5600 }
+{ "o_orderkey": 5601 }
+{ "o_orderkey": 5602 }
+{ "o_orderkey": 5603 }
+{ "o_orderkey": 5604 }
+{ "o_orderkey": 5605 }
+{ "o_orderkey": 5606 }
+{ "o_orderkey": 5607 }
+{ "o_orderkey": 5632 }
+{ "o_orderkey": 5633 }
+{ "o_orderkey": 5634 }
+{ "o_orderkey": 5635 }
+{ "o_orderkey": 5636 }
+{ "o_orderkey": 5637 }
+{ "o_orderkey": 5638 }
+{ "o_orderkey": 5639 }
+{ "o_orderkey": 5664 }
+{ "o_orderkey": 5665 }
+{ "o_orderkey": 5666 }
+{ "o_orderkey": 5667 }
+{ "o_orderkey": 5668 }
+{ "o_orderkey": 5669 }
+{ "o_orderkey": 5670 }
+{ "o_orderkey": 5671 }
+{ "o_orderkey": 5696 }
+{ "o_orderkey": 5697 }
+{ "o_orderkey": 5698 }
+{ "o_orderkey": 5699 }
+{ "o_orderkey": 5700 }
+{ "o_orderkey": 5701 }
+{ "o_orderkey": 5702 }
+{ "o_orderkey": 5703 }
+{ "o_orderkey": 5728 }
+{ "o_orderkey": 5729 }
+{ "o_orderkey": 5730 }
+{ "o_orderkey": 5731 }
+{ "o_orderkey": 5732 }
+{ "o_orderkey": 5733 }
+{ "o_orderkey": 5734 }
+{ "o_orderkey": 5735 }
+{ "o_orderkey": 5760 }
+{ "o_orderkey": 5761 }
+{ "o_orderkey": 5762 }
+{ "o_orderkey": 5763 }
+{ "o_orderkey": 5764 }
+{ "o_orderkey": 5765 }
+{ "o_orderkey": 5766 }
+{ "o_orderkey": 5767 }
+{ "o_orderkey": 5792 }
+{ "o_orderkey": 5793 }
+{ "o_orderkey": 5794 }
+{ "o_orderkey": 5795 }
+{ "o_orderkey": 5796 }
+{ "o_orderkey": 5797 }
+{ "o_orderkey": 5798 }
+{ "o_orderkey": 5799 }
+{ "o_orderkey": 5824 }
+{ "o_orderkey": 5825 }
+{ "o_orderkey": 5826 }
+{ "o_orderkey": 5827 }
+{ "o_orderkey": 5828 }
+{ "o_orderkey": 5829 }
+{ "o_orderkey": 5830 }
+{ "o_orderkey": 5831 }
+{ "o_orderkey": 5856 }
+{ "o_orderkey": 5857 }
+{ "o_orderkey": 5858 }
+{ "o_orderkey": 5859 }
+{ "o_orderkey": 5860 }
+{ "o_orderkey": 5861 }
+{ "o_orderkey": 5862 }
+{ "o_orderkey": 5863 }
+{ "o_orderkey": 5888 }
+{ "o_orderkey": 5889 }
+{ "o_orderkey": 5890 }
+{ "o_orderkey": 5891 }
+{ "o_orderkey": 5892 }
+{ "o_orderkey": 5893 }
+{ "o_orderkey": 5894 }
+{ "o_orderkey": 5895 }
+{ "o_orderkey": 5920 }
+{ "o_orderkey": 5921 }
+{ "o_orderkey": 5922 }
+{ "o_orderkey": 5923 }
+{ "o_orderkey": 5924 }
+{ "o_orderkey": 5925 }
+{ "o_orderkey": 5926 }
+{ "o_orderkey": 5927 }
+{ "o_orderkey": 5952 }
+{ "o_orderkey": 5953 }
+{ "o_orderkey": 5954 }
+{ "o_orderkey": 5955 }
+{ "o_orderkey": 5956 }
+{ "o_orderkey": 5957 }
+{ "o_orderkey": 5958 }
+{ "o_orderkey": 5959 }
+{ "o_orderkey": 5984 }
+{ "o_orderkey": 5985 }
+{ "o_orderkey": 5986 }
+{ "o_orderkey": 5987 }
+{ "o_orderkey": 5988 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.08.plan b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.08.plan
new file mode 100644
index 0000000000..a14d818ceb
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.08.plan
@@ -0,0 +1,50 @@
+distribute result [$$36]
+-- DISTRIBUTE_RESULT  |PARTITIONED|
+  exchange
+  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+    project ([$$36])
+    -- STREAM_PROJECT  |PARTITIONED|
+      assign [$$36] <- [{"o_orderkey": $$41, "l_orderkey": $$42, "l_suppkey": $$45}]
+      -- ASSIGN  |PARTITIONED|
+        exchange
+        -- SORT_MERGE_EXCHANGE [$$41(ASC), $$42(ASC), $$45(ASC) ]  |PARTITIONED|
+          order (ASC, $$41) (ASC, $$42) (ASC, $$45)
+          -- STABLE_SORT [$$41(ASC), $$42(ASC), $$45(ASC)]  |PARTITIONED|
+            exchange
+            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+              project ([$$41, $$42, $$45])
+              -- STREAM_PROJECT  |PARTITIONED|
+                exchange
+                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                  left outer join (and(eq($$41, $$42), eq($$54, $$45)))
+                  -- HYBRID_HASH_JOIN [$$41, $$54][$$42, $$45]  |PARTITIONED|
+                    exchange
+                    -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                      assign [$$54] <- [$$41]
+                      -- ASSIGN  |PARTITIONED|
+                        project ([$$41])
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          exchange
+                          -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                            data-scan []<-[$$41, $$o] <- tpch.Orders
+                            -- DATASOURCE_SCAN  |PARTITIONED|
+                              exchange
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                empty-tuple-source
+                                -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$42]  |PARTITIONED|
+                      project ([$$42, $$45])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        assign [$$45] <- [$$l.getField(2)]
+                        -- ASSIGN  |PARTITIONED|
+                          project ([$$42, $$l])
+                          -- STREAM_PROJECT  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              data-scan []<-[$$42, $$43, $$l] <- tpch.LineItem
+                              -- DATASOURCE_SCAN  |PARTITIONED|
+                                exchange
+                                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                  empty-tuple-source
+                                  -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.09.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.09.adm
new file mode 100644
index 0000000000..322cb104da
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.09.adm
@@ -0,0 +1,58 @@
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 24, "s_nationkey": 24, "c_nationkey": 24 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.10.plan b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.10.plan
new file mode 100644
index 0000000000..1fadf108e6
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.10.plan
@@ -0,0 +1,64 @@
+distribute result [$$48]
+-- DISTRIBUTE_RESULT  |PARTITIONED|
+  exchange
+  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+    project ([$$48])
+    -- STREAM_PROJECT  |PARTITIONED|
+      assign [$$48] <- [{"n_nationkey": $$55, "s_nationkey": $$53, "c_nationkey": $$52}]
+      -- ASSIGN  |PARTITIONED|
+        exchange
+        -- SORT_MERGE_EXCHANGE [$$55(ASC), $$53(ASC), $$52(ASC) ]  |PARTITIONED|
+          order (ASC, $$55) (ASC, $$53) (ASC, $$52)
+          -- STABLE_SORT [$$55(ASC), $$53(ASC), $$52(ASC)]  |PARTITIONED|
+            exchange
+            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+              join (eq($$52, $$55))
+              -- HYBRID_HASH_JOIN [$$55][$$52]  |PARTITIONED|
+                exchange
+                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                  join (eq($$53, $$55))
+                  -- HYBRID_HASH_JOIN [$$55][$$53]  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$55]  |PARTITIONED|
+                      project ([$$55])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          data-scan []<-[$$55, $$n] <- tpch.Nation
+                          -- DATASOURCE_SCAN  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              empty-tuple-source
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$53]  |PARTITIONED|
+                      project ([$$53])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        assign [$$53] <- [$$s.getField(3)]
+                        -- ASSIGN  |PARTITIONED|
+                          project ([$$s])
+                          -- STREAM_PROJECT  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              data-scan []<-[$$56, $$s] <- tpch.Supplier
+                              -- DATASOURCE_SCAN  |PARTITIONED|
+                                exchange
+                                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                  empty-tuple-source
+                                  -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                exchange
+                -- HASH_PARTITION_EXCHANGE [$$52]  |PARTITIONED|
+                  project ([$$52])
+                  -- STREAM_PROJECT  |PARTITIONED|
+                    assign [$$52] <- [$$c.getField(3)]
+                    -- ASSIGN  |PARTITIONED|
+                      project ([$$c])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          data-scan []<-[$$57, $$c] <- tpch.Customer
+                          -- DATASOURCE_SCAN  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              empty-tuple-source
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.11.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.11.adm
new file mode 100644
index 0000000000..322cb104da
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.11.adm
@@ -0,0 +1,58 @@
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 24, "s_nationkey": 24, "c_nationkey": 24 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.12.plan b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.12.plan
new file mode 100644
index 0000000000..1fadf108e6
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.12.plan
@@ -0,0 +1,64 @@
+distribute result [$$48]
+-- DISTRIBUTE_RESULT  |PARTITIONED|
+  exchange
+  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+    project ([$$48])
+    -- STREAM_PROJECT  |PARTITIONED|
+      assign [$$48] <- [{"n_nationkey": $$55, "s_nationkey": $$53, "c_nationkey": $$52}]
+      -- ASSIGN  |PARTITIONED|
+        exchange
+        -- SORT_MERGE_EXCHANGE [$$55(ASC), $$53(ASC), $$52(ASC) ]  |PARTITIONED|
+          order (ASC, $$55) (ASC, $$53) (ASC, $$52)
+          -- STABLE_SORT [$$55(ASC), $$53(ASC), $$52(ASC)]  |PARTITIONED|
+            exchange
+            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+              join (eq($$52, $$55))
+              -- HYBRID_HASH_JOIN [$$55][$$52]  |PARTITIONED|
+                exchange
+                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                  join (eq($$53, $$55))
+                  -- HYBRID_HASH_JOIN [$$55][$$53]  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$55]  |PARTITIONED|
+                      project ([$$55])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          data-scan []<-[$$55, $$n] <- tpch.Nation
+                          -- DATASOURCE_SCAN  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              empty-tuple-source
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$53]  |PARTITIONED|
+                      project ([$$53])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        assign [$$53] <- [$$s.getField(3)]
+                        -- ASSIGN  |PARTITIONED|
+                          project ([$$s])
+                          -- STREAM_PROJECT  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              data-scan []<-[$$56, $$s] <- tpch.Supplier
+                              -- DATASOURCE_SCAN  |PARTITIONED|
+                                exchange
+                                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                  empty-tuple-source
+                                  -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                exchange
+                -- HASH_PARTITION_EXCHANGE [$$52]  |PARTITIONED|
+                  project ([$$52])
+                  -- STREAM_PROJECT  |PARTITIONED|
+                    assign [$$52] <- [$$c.getField(3)]
+                    -- ASSIGN  |PARTITIONED|
+                      project ([$$c])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          data-scan []<-[$$57, $$c] <- tpch.Customer
+                          -- DATASOURCE_SCAN  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              empty-tuple-source
+                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.13.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.13.adm
new file mode 100644
index 0000000000..322cb104da
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.13.adm
@@ -0,0 +1,58 @@
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 24, "s_nationkey": 24, "c_nationkey": 24 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.14.plan b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.14.plan
new file mode 100644
index 0000000000..76b66b233f
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.14.plan
@@ -0,0 +1,66 @@
+distribute result [$$48]
+-- DISTRIBUTE_RESULT  |PARTITIONED|
+  exchange
+  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+    project ([$$48])
+    -- STREAM_PROJECT  |PARTITIONED|
+      assign [$$48] <- [{"n_nationkey": $$56, "s_nationkey": $$53, "c_nationkey": $$52}]
+      -- ASSIGN  |PARTITIONED|
+        exchange
+        -- SORT_MERGE_EXCHANGE [$$56(ASC), $$53(ASC), $$52(ASC) ]  |PARTITIONED|
+          order (ASC, $$56) (ASC, $$53) (ASC, $$52)
+          -- STABLE_SORT [$$56(ASC), $$53(ASC), $$52(ASC)]  |PARTITIONED|
+            exchange
+            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+              project ([$$56, $$53, $$52])
+              -- STREAM_PROJECT  |PARTITIONED|
+                exchange
+                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                  join (and(eq($$52, $$56), eq($$53, $$63)))
+                  -- HYBRID_HASH_JOIN [$$56, $$53][$$52, $$63]  |PARTITIONED|
+                    exchange
+                    -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                      project ([$$53, $$56])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          unnest-map [$$56, $$n] <- index-search("Nation", 0, "tpch", "Nation", true, true, 1, $$53, 1, $$53, true, true, true)
+                          -- BTREE_SEARCH  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              order (ASC, $$53)
+                              -- STABLE_SORT [$$53(ASC)]  |PARTITIONED|
+                                exchange
+                                -- HASH_PARTITION_EXCHANGE [$$53]  |PARTITIONED|
+                                  project ([$$53])
+                                  -- STREAM_PROJECT  |PARTITIONED|
+                                    assign [$$53] <- [$$s.getField(3)]
+                                    -- ASSIGN  |PARTITIONED|
+                                      project ([$$s])
+                                      -- STREAM_PROJECT  |PARTITIONED|
+                                        exchange
+                                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                          data-scan []<-[$$55, $$s] <- tpch.Supplier
+                                          -- DATASOURCE_SCAN  |PARTITIONED|
+                                            exchange
+                                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                              empty-tuple-source
+                                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$52]  |PARTITIONED|
+                      assign [$$63] <- [$$52]
+                      -- ASSIGN  |PARTITIONED|
+                        project ([$$52])
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          assign [$$52] <- [$$c.getField(3)]
+                          -- ASSIGN  |PARTITIONED|
+                            project ([$$c])
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              exchange
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                data-scan []<-[$$57, $$c] <- tpch.Customer
+                                -- DATASOURCE_SCAN  |PARTITIONED|
+                                  exchange
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    empty-tuple-source
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.15.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.15.adm
new file mode 100644
index 0000000000..322cb104da
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.15.adm
@@ -0,0 +1,58 @@
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 1, "s_nationkey": 1, "c_nationkey": 1 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 5, "s_nationkey": 5, "c_nationkey": 5 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 10, "s_nationkey": 10, "c_nationkey": 10 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 11, "s_nationkey": 11, "c_nationkey": 11 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 14, "s_nationkey": 14, "c_nationkey": 14 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 15, "s_nationkey": 15, "c_nationkey": 15 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 17, "s_nationkey": 17, "c_nationkey": 17 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 23, "s_nationkey": 23, "c_nationkey": 23 }
+{ "n_nationkey": 24, "s_nationkey": 24, "c_nationkey": 24 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.16.plan b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.16.plan
new file mode 100644
index 0000000000..8288361a9d
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/hash-join-with-redundant-variable/hash-join-with-redundant-variable.16.plan
@@ -0,0 +1,66 @@
+distribute result [$$48]
+-- DISTRIBUTE_RESULT  |PARTITIONED|
+  exchange
+  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+    project ([$$48])
+    -- STREAM_PROJECT  |PARTITIONED|
+      assign [$$48] <- [{"n_nationkey": $$56, "s_nationkey": $$53, "c_nationkey": $$52}]
+      -- ASSIGN  |PARTITIONED|
+        exchange
+        -- SORT_MERGE_EXCHANGE [$$56(ASC), $$53(ASC), $$52(ASC) ]  |PARTITIONED|
+          order (ASC, $$56) (ASC, $$53) (ASC, $$52)
+          -- STABLE_SORT [$$56(ASC), $$53(ASC), $$52(ASC)]  |PARTITIONED|
+            exchange
+            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+              project ([$$56, $$53, $$52])
+              -- STREAM_PROJECT  |PARTITIONED|
+                exchange
+                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                  join (and(eq($$52, $$56), eq($$53, $$63)))
+                  -- HYBRID_HASH_JOIN [$$56, $$53][$$52, $$63]  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$56, $$53]  |PARTITIONED|
+                      project ([$$53, $$56])
+                      -- STREAM_PROJECT  |PARTITIONED|
+                        exchange
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          unnest-map [$$56, $$n] <- index-search("Nation", 0, "tpch", "Nation", true, true, 1, $$53, 1, $$53, true, true, true)
+                          -- BTREE_SEARCH  |PARTITIONED|
+                            exchange
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              order (ASC, $$53)
+                              -- STABLE_SORT [$$53(ASC)]  |PARTITIONED|
+                                exchange
+                                -- HASH_PARTITION_EXCHANGE [$$53]  |PARTITIONED|
+                                  project ([$$53])
+                                  -- STREAM_PROJECT  |PARTITIONED|
+                                    assign [$$53] <- [$$s.getField(3)]
+                                    -- ASSIGN  |PARTITIONED|
+                                      project ([$$s])
+                                      -- STREAM_PROJECT  |PARTITIONED|
+                                        exchange
+                                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                          data-scan []<-[$$55, $$s] <- tpch.Supplier
+                                          -- DATASOURCE_SCAN  |PARTITIONED|
+                                            exchange
+                                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                              empty-tuple-source
+                                              -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                    exchange
+                    -- HASH_PARTITION_EXCHANGE [$$52, $$63]  |PARTITIONED|
+                      assign [$$63] <- [$$52]
+                      -- ASSIGN  |PARTITIONED|
+                        project ([$$52])
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          assign [$$52] <- [$$c.getField(3)]
+                          -- ASSIGN  |PARTITIONED|
+                            project ([$$c])
+                            -- STREAM_PROJECT  |PARTITIONED|
+                              exchange
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                data-scan []<-[$$57, $$c] <- tpch.Customer
+                                -- DATASOURCE_SCAN  |PARTITIONED|
+                                  exchange
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    empty-tuple-source
+                                    -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/testsuite_sqlpp.xml b/asterixdb/asterix-app/src/test/resources/runtimets/testsuite_sqlpp.xml
index addced1707..54b4ffc5af 100644
--- a/asterixdb/asterix-app/src/test/resources/runtimets/testsuite_sqlpp.xml
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/testsuite_sqlpp.xml
@@ -6646,6 +6646,11 @@
         <expected-error>ASX1077: Cannot find dataset x in dataverse Default nor an alias with name x (in line 26, at column 39)</expected-error>
       </compilation-unit>
     </test-case>
+    <test-case FilePath="join">
+      <compilation-unit name="hash-join-with-redundant-variable">
+        <output-dir compare="Text">hash-join-with-redundant-variable</output-dir>
+      </compilation-unit>
+    </test-case>
     <test-case FilePath="join">
       <compilation-unit name="hash_join_array">
         <output-dir compare="Text">hash_join_array</output-dir>
diff --git a/asterixdb/asterix-lang-common/src/main/java/org/apache/asterix/lang/common/util/FunctionUtil.java b/asterixdb/asterix-lang-common/src/main/java/org/apache/asterix/lang/common/util/FunctionUtil.java
index 1cf88c2c35..cce0bd8d31 100644
--- a/asterixdb/asterix-lang-common/src/main/java/org/apache/asterix/lang/common/util/FunctionUtil.java
+++ b/asterixdb/asterix-lang-common/src/main/java/org/apache/asterix/lang/common/util/FunctionUtil.java
@@ -21,11 +21,13 @@ package org.apache.asterix.lang.common.util;
 
 import java.io.StringReader;
 import java.util.ArrayList;
+import java.util.BitSet;
 import java.util.Collection;
 import java.util.Collections;
 import java.util.List;
 import java.util.Map;
 import java.util.Objects;
+import java.util.Set;
 import java.util.function.BiFunction;
 
 import org.apache.asterix.common.exceptions.AsterixException;
@@ -67,6 +69,8 @@ import org.apache.hyracks.api.exceptions.SourceLocation;
 public class FunctionUtil {
 
     public static final String IMPORT_PRIVATE_FUNCTIONS = "import-private-functions";
+    //TODO(wyk) add Multiply and Add
+    private static final Set<FunctionIdentifier> COMMUTATIVE_FUNCTIONS = Set.of(BuiltinFunctions.EQ);
 
     private static final DataverseName FN_DATASET_DATAVERSE_NAME =
             FunctionSignature.getDataverseName(BuiltinFunctions.DATASET);
@@ -316,4 +320,69 @@ public class FunctionUtil {
         return BuiltinFunctions.FIELD_ACCESS_BY_INDEX.equals(fid) || BuiltinFunctions.FIELD_ACCESS_BY_NAME.equals(fid)
                 || BuiltinFunctions.FIELD_ACCESS_NESTED.equals(fid);
     }
+
+    /**
+     * Compares two commutative expressions
+     * TODO It doesn't support add and multiply (e.g., add(x, add(y, z) & add(add(x, y), z) will return false)
+     *
+     * @param expr1 left expression
+     * @param expr2 right expression
+     * @return true if equals, false otherwise
+     */
+    public static boolean commutativeEquals(ILogicalExpression expr1, ILogicalExpression expr2) {
+        if (expr1.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL
+                || expr2.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
+            return expr1.equals(expr2);
+        }
+
+        AbstractFunctionCallExpression funcExpr1 = (AbstractFunctionCallExpression) expr1;
+        AbstractFunctionCallExpression funcExpr2 = (AbstractFunctionCallExpression) expr2;
+
+        FunctionIdentifier fid1 = funcExpr1.getFunctionIdentifier();
+        FunctionIdentifier fid2 = funcExpr2.getFunctionIdentifier();
+
+        if (!fid1.equals(fid2) || funcExpr1.getArguments().size() != funcExpr2.getArguments().size()) {
+            return false;
+        } else if (!COMMUTATIVE_FUNCTIONS.contains(fid1)) {
+            return expr1.equals(expr2);
+        }
+
+        return commutativeEquals(expr1, expr2, new BitSet());
+    }
+
+    private static boolean commutativeEquals(ILogicalExpression expr1, ILogicalExpression expr2, BitSet matched) {
+        if (expr1.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL
+                || expr2.getExpressionTag() != LogicalExpressionTag.FUNCTION_CALL) {
+            return expr1.equals(expr2);
+        }
+
+        AbstractFunctionCallExpression funcExpr1 = (AbstractFunctionCallExpression) expr1;
+        AbstractFunctionCallExpression funcExpr2 = (AbstractFunctionCallExpression) expr2;
+
+        List<Mutable<ILogicalExpression>> args1 = funcExpr1.getArguments();
+        List<Mutable<ILogicalExpression>> args2 = funcExpr2.getArguments();
+
+        BitSet childrenSet = new BitSet();
+        int numberOfMatches = 0;
+        for (Mutable<ILogicalExpression> arg1 : args1) {
+            int prevNumberOfMatches = numberOfMatches;
+
+            for (int i = 0; i < args2.size(); i++) {
+                Mutable<ILogicalExpression> arg2 = args2.get(i);
+                childrenSet.clear();
+                if (!matched.get(i) && commutativeEquals(arg1.getValue(), arg2.getValue(), childrenSet)) {
+                    matched.set(i);
+                    numberOfMatches++;
+                    break;
+                }
+            }
+
+            if (numberOfMatches == prevNumberOfMatches) {
+                // Early exit as one operand didn't match with any of the other operands
+                return false;
+            }
+        }
+
+        return numberOfMatches == args1.size();
+    }
 }
diff --git a/asterixdb/asterix-lang-sqlpp/src/test/java/org/apache/asterix/lang/expression/CommutativeEqualsTest.java b/asterixdb/asterix-lang-sqlpp/src/test/java/org/apache/asterix/lang/expression/CommutativeEqualsTest.java
new file mode 100644
index 0000000000..06f5ba7187
--- /dev/null
+++ b/asterixdb/asterix-lang-sqlpp/src/test/java/org/apache/asterix/lang/expression/CommutativeEqualsTest.java
@@ -0,0 +1,87 @@
+/*
+ * 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.
+ */
+package org.apache.asterix.lang.expression;
+
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.asterix.lang.common.util.FunctionUtil;
+import org.apache.asterix.om.functions.BuiltinFunctions;
+import org.apache.commons.lang3.mutable.Mutable;
+import org.apache.commons.lang3.mutable.MutableObject;
+import org.apache.hyracks.algebricks.core.algebra.base.ILogicalExpression;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
+import org.apache.hyracks.algebricks.core.algebra.expressions.ScalarFunctionCallExpression;
+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.functions.IFunctionInfo;
+import org.junit.Assert;
+import org.junit.Test;
+
+/**
+ * TODO include multiply and add when supported
+ *
+ * @see FunctionUtil#commutativeEquals(ILogicalExpression, ILogicalExpression)
+ */
+public class CommutativeEqualsTest {
+    private final Map<Character, LogicalVariable> varNameToVarMap = new HashMap<>();
+    private int varCounter;
+
+    @Test
+    public void testTwoOperands() {
+        // EQ
+        reset();
+        ILogicalExpression expr1 = createExpression(BuiltinFunctions.EQ, 'x', 'y');
+        ILogicalExpression expr2 = createExpression(BuiltinFunctions.EQ, 'x', 'y');
+        Assert.assertTrue(FunctionUtil.commutativeEquals(expr1, expr2));
+
+        reset();
+        expr1 = createExpression(BuiltinFunctions.EQ, 'x', 'y');
+        expr2 = createExpression(BuiltinFunctions.EQ, 'y', 'x');
+        Assert.assertTrue(FunctionUtil.commutativeEquals(expr1, expr2));
+
+        reset();
+        expr1 = createExpression(BuiltinFunctions.EQ, 'x', 'x');
+        expr2 = createExpression(BuiltinFunctions.EQ, 'x', 'y');
+        Assert.assertFalse(FunctionUtil.commutativeEquals(expr1, expr2));
+    }
+
+    private void reset() {
+        varCounter = 0;
+        varNameToVarMap.clear();
+    }
+
+    private ILogicalExpression createExpression(FunctionIdentifier fid, char left, char right) {
+        List<Mutable<ILogicalExpression>> args = new ArrayList<>();
+
+        args.add(getVariableExpression(left));
+        args.add(getVariableExpression(right));
+
+        IFunctionInfo funcInfo = BuiltinFunctions.getBuiltinFunctionInfo(fid);
+        return new ScalarFunctionCallExpression(funcInfo, args);
+    }
+
+    private Mutable<ILogicalExpression> getVariableExpression(Character displayName) {
+        LogicalVariable variable = varNameToVarMap.computeIfAbsent(displayName,
+                k -> new LogicalVariable(varCounter++, displayName.toString()));
+        return new MutableObject<>(new VariableReferenceExpression(variable));
+    }
+}
diff --git a/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/properties/IPartitioningRequirementsCoordinator.java b/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/properties/IPartitioningRequirementsCoordinator.java
index d515fcfc6b..a49c4b3bb7 100644
--- a/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/properties/IPartitioningRequirementsCoordinator.java
+++ b/hyracks-fullstack/algebricks/algebricks-core/src/main/java/org/apache/hyracks/algebricks/core/algebra/properties/IPartitioningRequirementsCoordinator.java
@@ -29,6 +29,7 @@ import org.apache.hyracks.algebricks.core.algebra.base.EquivalenceClass;
 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.base.LogicalVariable;
+import org.apache.hyracks.api.exceptions.ErrorCode;
 
 /**
  * Implements constraints in between requirements for the children of the same
@@ -70,20 +71,29 @@ public interface IPartitioningRequirementsCoordinator {
                                 for (LogicalVariable v : set1) {
                                     EquivalenceClass ecFirst = eqmap.get(v);
                                     for (LogicalVariable r : uppreq.getColumnSet()) {
-                                        EquivalenceClass ec = eqmap.get(r);
-                                        if (ecFirst == ec) {
-                                            covered.add(v);
-                                            modifuppreq.add(r);
-                                            break;
+                                        if (!modifuppreq.contains(r)) {
+                                            EquivalenceClass ec = eqmap.get(r);
+                                            if (ecFirst == ec) {
+                                                covered.add(v);
+                                                modifuppreq.add(r);
+                                                break;
+                                            }
                                         }
                                     }
                                 }
 
                                 if (!covered.equals(set1)) {
-                                    throw new AlgebricksException("Could not modify " + rqdpp
-                                            + " to agree with partitioning property " + firstDeliveredPartitioning
-                                            + " delivered by previous input operator.");
+                                    throw new AlgebricksException(ErrorCode.ILLEGAL_STATE,
+                                            "Could not modify " + rqdpp + " to agree with partitioning property "
+                                                    + firstDeliveredPartitioning
+                                                    + " delivered by previous input operator.");
                                 }
+
+                                if (modifuppreq.size() != set1.size()) {
+                                    throw new AlgebricksException(ErrorCode.ILLEGAL_STATE,
+                                            "The number of variables are not equal in both partitioning sides");
+                                }
+
                                 UnorderedPartitionedProperty upp2 =
                                         new UnorderedPartitionedProperty(modifuppreq, rqdpp.getNodeDomain());
                                 return new Pair<Boolean, IPartitioningProperty>(false, upp2);