Merge "Merge branch 'gerrit/trinity' into 'master'"
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 7a06f4c..67dab65 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
@@ -115,6 +115,7 @@
 import org.apache.hyracks.algebricks.rewriter.rules.BreakSelectIntoConjunctsRule;
 import org.apache.hyracks.algebricks.rewriter.rules.ComplexUnnestToProductRule;
 import org.apache.hyracks.algebricks.rewriter.rules.ConsolidateAssignsRule;
+import org.apache.hyracks.algebricks.rewriter.rules.ConsolidateLeftOuterJoinSelectsRule;
 import org.apache.hyracks.algebricks.rewriter.rules.ConsolidateSelectsRule;
 import org.apache.hyracks.algebricks.rewriter.rules.CopyLimitDownRule;
 import org.apache.hyracks.algebricks.rewriter.rules.EliminateGroupByEmptyKeyRule;
@@ -346,6 +347,8 @@
         planCleanupRules.add(new RemoveRedundantVariablesInUnionRule()); // relies on RemoveUnusedAssignAndAggregateRule
         planCleanupRules.add(new PushProjectDownRule());
         planCleanupRules.add(new PushSelectDownRule());
+        // Must run after PushSelectDownRule
+        planCleanupRules.add(new ConsolidateLeftOuterJoinSelectsRule());
         planCleanupRules.add(new SetClosedRecordConstructorsRule());
         planCleanupRules.add(new IntroduceDynamicTypeCastRule());
         planCleanupRules.add(new IntroduceDynamicTypeCastForExternalFunctionRule());
diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/RemoveUnusedOneToOneEquiJoinRule.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/RemoveUnusedOneToOneEquiJoinRule.java
index 61361f6..265791e 100644
--- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/RemoveUnusedOneToOneEquiJoinRule.java
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/RemoveUnusedOneToOneEquiJoinRule.java
@@ -24,6 +24,7 @@
 import java.util.List;
 import java.util.Set;
 
+import org.apache.asterix.common.config.DatasetConfig;
 import org.apache.asterix.metadata.declared.DataSource;
 import org.apache.asterix.metadata.declared.DatasetDataSource;
 import org.apache.asterix.metadata.entities.Dataset;
@@ -165,10 +166,14 @@
         // Check that all datascans scan the same dataset, and that the join condition
         // only used primary key variables of those datascans.
         for (int i = 0; i < dataScans.size(); i++) {
+            DatasetDataSource currentDataSource = (DatasetDataSource) dataScans.get(i).getDataSource();
+            if (currentDataSource.getDataset().getDatasetType() == DatasetConfig.DatasetType.EXTERNAL) {
+                // The PK condition is not satisfied when external datasets are involved (no PKs)
+                return -1;
+            }
             if (i > 0) {
-                DatasetDataSource prevAqlDataSource = (DatasetDataSource) dataScans.get(i - 1).getDataSource();
-                DatasetDataSource currAqlDataSource = (DatasetDataSource) dataScans.get(i).getDataSource();
-                if (!prevAqlDataSource.getDataset().equals(currAqlDataSource.getDataset())) {
+                DatasetDataSource previousDataSource = (DatasetDataSource) dataScans.get(i - 1).getDataSource();
+                if (!previousDataSource.getDataset().equals(currentDataSource.getDataset())) {
                     return -1;
                 }
             }
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceRequestParameters.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceRequestParameters.java
index b6913e4..20d79d5 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceRequestParameters.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceRequestParameters.java
@@ -19,6 +19,8 @@
 
 package org.apache.asterix.api.http.server;
 
+import static org.apache.asterix.utils.RedactionUtil.REDACTED_SENSITIVE_ENTRY_VALUE;
+
 import java.io.IOException;
 import java.util.HashMap;
 import java.util.Iterator;
@@ -43,6 +45,7 @@
 import org.apache.hyracks.http.api.IServletRequest;
 import org.apache.hyracks.http.server.utils.HttpUtil;
 import org.apache.hyracks.util.JSONUtil;
+import org.apache.hyracks.util.LogRedactionUtil;
 
 import com.fasterxml.jackson.core.JsonParseException;
 import com.fasterxml.jackson.core.JsonProcessingException;
@@ -80,7 +83,8 @@
         SIGNATURE("signature"),
         MULTI_STATEMENT("multi-statement"),
         MAX_WARNINGS("max-warnings"),
-        SQL_COMPAT("sql-compat");
+        SQL_COMPAT("sql-compat"),
+        SOURCE("source");
 
         private final String str;
 
@@ -124,6 +128,7 @@
     private String statement;
     private String clientContextID;
     private String dataverse;
+    private String source;
     private ClientType clientType = ClientType.ASTERIX;
     private OutputFormat format = OutputFormat.CLEAN_JSON;
     private ResultDelivery mode = ResultDelivery.IMMEDIATE;
@@ -172,6 +177,14 @@
         this.statement = statement;
     }
 
+    public String getSource() {
+        return source;
+    }
+
+    public void setSource(String source) {
+        this.source = source;
+    }
+
     public OutputFormat getFormat() {
         return format;
     }
@@ -380,7 +393,8 @@
         ObjectNode object = OBJECT_MAPPER.createObjectNode();
         object.put("host", host);
         object.put("path", path);
-        object.put("statement", statement != null ? JSONUtil.escape(new StringBuilder(), statement).toString() : null);
+        object.put("statement", statement != null
+                ? LogRedactionUtil.statement(JSONUtil.escape(new StringBuilder(), statement).toString()) : null);
         object.put("pretty", pretty);
         object.put("mode", mode.getName());
         object.put("clientContextID", clientContextID);
@@ -402,9 +416,10 @@
         object.put("readOnly", readOnly);
         object.put("maxWarnings", maxWarnings);
         object.put("sqlCompat", sqlCompatMode);
+        object.put("source", source);
         if (statementParams != null) {
             for (Map.Entry<String, JsonNode> statementParam : statementParams.entrySet()) {
-                object.set('$' + statementParam.getKey(), statementParam.getValue());
+                object.set('$' + statementParam.getKey(), REDACTED_SENSITIVE_ENTRY_VALUE);
             }
         }
         return object;
@@ -486,6 +501,7 @@
         setSignature(parseBoolean(req, Parameter.SIGNATURE.str(), valGetter, isSignature()));
         setClientType(parseIfExists(req, Parameter.CLIENT_TYPE.str(), valGetter, getClientType(), clientTypes::get));
         setSQLCompatMode(parseBoolean(req, Parameter.SQL_COMPAT.str(), valGetter, isSQLCompatMode()));
+        setSource(valGetter.apply(req, Parameter.SOURCE.str()));
     }
 
     protected void setExtraParams(JsonNode jsonRequest) throws HyracksDataException {
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
index 63cd915..3c013e7 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
@@ -90,7 +90,6 @@
 import org.apache.hyracks.http.api.IServletRequest;
 import org.apache.hyracks.http.api.IServletResponse;
 import org.apache.hyracks.http.server.utils.HttpUtil;
-import org.apache.hyracks.util.LogRedactionUtil;
 import org.apache.logging.log4j.LogManager;
 import org.apache.logging.log4j.Logger;
 
@@ -284,10 +283,10 @@
             if (statement != null && (statement.startsWith("UPSERT") || statement.startsWith("INSERT"))
                     && LOGGER.isDebugEnabled()) {
                 LOGGER.debug("handleRequest: uuid={}, clientContextID={}, {}", requestRef.getUuid(),
-                        param.getClientContextID(), LogRedactionUtil.statement(param.toString()));
+                        param.getClientContextID(), param.toString());
             } else if (LOGGER.isInfoEnabled()) {
                 LOGGER.info("handleRequest: uuid={}, clientContextID={}, {}", requestRef.getUuid(),
-                        param.getClientContextID(), LogRedactionUtil.statement(param.toString()));
+                        param.getClientContextID(), param.toString());
             }
             delivery = param.getMode();
             setSessionConfig(sessionOutput, param, delivery);
@@ -437,14 +436,12 @@
                     executionState.setStatus(ResultStatus.FATAL, HttpResponseStatus.BAD_REQUEST);
                     return true;
                 case REQUEST_TIMEOUT:
-                    LOGGER.info(() -> "handleException: request execution timed out: "
-                            + LogRedactionUtil.userData(param.toString()));
+                    LOGGER.info(() -> "handleException: request execution timed out: " + param.toString());
                     executionState.setStatus(ResultStatus.TIMEOUT, HttpResponseStatus.OK);
                     return true;
                 case REJECT_NODE_UNREGISTERED:
                 case REJECT_BAD_CLUSTER_STATE:
-                    LOGGER.warn(() -> "handleException: " + ex.getMessage() + ": "
-                            + LogRedactionUtil.userData(param.toString()));
+                    LOGGER.warn(() -> "handleException: " + ex.getMessage() + ": " + param.toString());
                     executionState.setStatus(ResultStatus.FATAL, HttpResponseStatus.SERVICE_UNAVAILABLE);
                     return true;
                 default:
@@ -464,11 +461,9 @@
             QueryServiceRequestParameters param, IServletResponse response) {
         if (t instanceof org.apache.asterix.lang.sqlpp.parser.TokenMgrError || t instanceof AlgebricksException) {
             if (LOGGER.isDebugEnabled()) {
-                LOGGER.debug("handleException: {}: {}", t.getMessage(), LogRedactionUtil.statement(param.toString()),
-                        t);
+                LOGGER.debug("handleException: {}: {}", t.getMessage(), param.toString(), t);
             } else {
-                LOGGER.info(() -> "handleException: " + t.getMessage() + ": "
-                        + LogRedactionUtil.statement(param.toString()));
+                LOGGER.info(() -> "handleException: " + t.getMessage() + ": " + param.toString());
             }
             executionState.setStatus(ResultStatus.FATAL, HttpResponseStatus.BAD_REQUEST);
             return;
@@ -480,7 +475,7 @@
                 return;
             }
         }
-        LOGGER.warn(() -> "handleException: unexpected exception: " + LogRedactionUtil.userData(param.toString()), t);
+        LOGGER.warn(() -> "handleException: unexpected exception: " + param.toString(), t);
         executionState.setStatus(ResultStatus.FATAL, HttpResponseStatus.INTERNAL_SERVER_ERROR);
     }
 
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/utils/RedactionUtil.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/utils/RedactionUtil.java
index 48cf511..d8b47b8 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/utils/RedactionUtil.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/utils/RedactionUtil.java
@@ -26,6 +26,8 @@
 
 import org.apache.hyracks.util.ILogRedactor;
 
+import com.fasterxml.jackson.databind.node.TextNode;
+
 public class RedactionUtil {
     private RedactionUtil() {
         throw new AssertionError("do not instantiate");
@@ -34,6 +36,7 @@
     private static final Pattern STATEMENT_PATTERN =
             Pattern.compile("(" + SECRET_ACCESS_KEY_FIELD_NAME + ").*", CASE_INSENSITIVE | DOTALL);
     private static final String STATEMENT_REPLACEMENT = "$1...<redacted sensitive data>";
+    public static final TextNode REDACTED_SENSITIVE_ENTRY_VALUE = new TextNode("<redacted sensitive entry>");
 
     public static final ILogRedactor LOG_REDACTOR = new ILogRedactor() {
         @Override
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/external-cross-product.sqlpp b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/external-cross-product.sqlpp
new file mode 100644
index 0000000..ec96012
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/external-cross-product.sqlpp
@@ -0,0 +1,34 @@
+/*
+ * 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 {
+};
+
+CREATE EXTERNAL DATASET Orders(OpenType) USING localfs
+(
+   ("path"="asterix_nc1://data/json/double-150-11.json"),
+   ("format"="json")
+);
+
+SELECT COUNT(*)
+FROM Orders o1, Orders o2, Orders o3;
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/query_issue3316.sqlpp b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/query_issue3316.sqlpp
index 74ecb1b..cc9bb60 100644
--- a/asterixdb/asterix-app/src/test/resources/optimizerts/queries/query_issue3316.sqlpp
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/queries/query_issue3316.sqlpp
@@ -17,19 +17,18 @@
  * under the License.
  */
 /*
- * Description  : This test case is to verify the fix for issue827
- * https://code.google.com/p/asterixdb/issues/detail?id=849
+ * Description  : This test case is to verify the fix for issue 3316
  * Expected Res : SUCCESS
- * Date         : 2nd Feb. 2015
  */
 
 drop  dataverse test if exists;
 create  dataverse test;
 
 use test;
+create type dt1 as {id:int};
 
-create dataset collection0 primary key (id:int);
-create dataset collection1 primary key (id:int);
+create dataset collection0(dt1) primary key id;
+create dataset collection1(dt1) primary key id;
 
 SELECT DISTINCT T1.*,
         (SELECT H.u
diff --git a/asterixdb/asterix-app/src/test/resources/optimizerts/results/external-cross-product.plan b/asterixdb/asterix-app/src/test/resources/optimizerts/results/external-cross-product.plan
new file mode 100644
index 0000000..fd95cc1
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/optimizerts/results/external-cross-product.plan
@@ -0,0 +1,45 @@
+-- DISTRIBUTE_RESULT  |UNPARTITIONED|
+  -- ONE_TO_ONE_EXCHANGE  |UNPARTITIONED|
+    -- STREAM_PROJECT  |UNPARTITIONED|
+      -- ASSIGN  |UNPARTITIONED|
+        -- AGGREGATE  |UNPARTITIONED|
+          -- RANDOM_MERGE_EXCHANGE  |PARTITIONED|
+            -- AGGREGATE  |PARTITIONED|
+              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                -- NESTED_LOOP  |PARTITIONED|
+                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                    -- NESTED_LOOP  |PARTITIONED|
+                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                        -- STREAM_PROJECT  |PARTITIONED|
+                          -- ASSIGN  |PARTITIONED|
+                            -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                              -- REPLICATE  |PARTITIONED|
+                                -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                  -- STREAM_PROJECT  |PARTITIONED|
+                                    -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                      -- DATASOURCE_SCAN (test.Orders)  |PARTITIONED|
+                                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                          -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                        -- REPLICATE  |PARTITIONED|
+                          -- BROADCAST_EXCHANGE  |PARTITIONED|
+                            -- REPLICATE  |PARTITIONED|
+                              -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                -- STREAM_PROJECT  |PARTITIONED|
+                                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                    -- DATASOURCE_SCAN (test.Orders)  |PARTITIONED|
+                                      -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                                        -- EMPTY_TUPLE_SOURCE  |PARTITIONED|
+                  -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                    -- STREAM_PROJECT  |PARTITIONED|
+                      -- ASSIGN  |PARTITIONED|
+                        -- ONE_TO_ONE_EXCHANGE  |PARTITIONED|
+                          -- REPLICATE  |PARTITIONED|
+                            -- BROADCAST_EXCHANGE  |PARTITIONED|
+                              -- REPLICATE  |PARTITIONED|
+                                -- ONE_TO_ONE_EXCHANGE  |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/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.00.ddl.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.00.ddl.sqlpp
new file mode 100644
index 0000000..a4d3984
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.00.ddl.sqlpp
@@ -0,0 +1,57 @@
+/*
+ * 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 CustomerType AS CLOSED {
+  c_custkey : bigint,
+  c_name : string,
+  c_address : string,
+  c_nationkey : bigint,
+  c_phone : string,
+  c_acctbal : double,
+  c_mktsegment : string,
+  c_comment : string
+};
+
+CREATE TYPE OrderType AS CLOSED {
+  o_orderkey : bigint,
+  o_custkey : bigint,
+  o_orderstatus : string,
+  o_totalprice : double,
+  o_orderdate : string,
+  o_orderpriority : string,
+  o_clerk : string,
+  o_shippriority : bigint,
+  o_comment : string
+};
+
+CREATE DATASET Customer(CustomerType)
+PRIMARY KEY c_custkey;
+
+CREATE DATASET Orders(OrderType)
+PRIMARY KEY o_orderkey;
+
+
+CREATE INDEX customer_fk_nation ON Customer (c_nationkey);
+CREATE INDEX orders_fk_customer ON Orders (o_custkey);
+CREATE INDEX orders_orderdateIx ON Orders (o_orderdate);
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.01.update.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.01.update.sqlpp
new file mode 100644
index 0000000..3336fd7
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.01.update.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;
+
+LOAD DATASET Orders USING localfs (
+   (`path`=`asterix_nc1://data/tpch0.001/orders.tbl`),
+   (`format`=`delimited-text`),
+   (`delimiter`=`|`)
+);
+
+LOAD DATASET Customer using localfs (
+   (`path`=`asterix_nc1://data/tpch0.001/customer.tbl`),
+   (`format`=`delimited-text`),
+   (`delimiter`=`|`)
+);
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.10.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.10.query.sqlpp
new file mode 100644
index 0000000..45fa904
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.10.query.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 tpch;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey  = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.11.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.11.query.sqlpp
new file mode 100644
index 0000000..0d270a7
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.11.query.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 tpch;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c 
+LEFT OUTER JOIN Orders o 
+ON  c.c_custkey /*+indexnl*/ = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.20.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.20.query.sqlpp
new file mode 100644
index 0000000..5f5562b
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.20.query.sqlpp
@@ -0,0 +1,27 @@
+/*
+ * 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;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey  = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
+AND o.o_totalprice > 100000
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.21.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.21.query.sqlpp
new file mode 100644
index 0000000..3057cd5
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.21.query.sqlpp
@@ -0,0 +1,27 @@
+/*
+ * 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;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey /*+indexnl*/ = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
+AND o.o_totalprice > 100000
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.30.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.30.query.sqlpp
new file mode 100644
index 0000000..717da7f
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.30.query.sqlpp
@@ -0,0 +1,27 @@
+/*
+ * 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;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey  = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
+AND (CASE WHEN o.o_totalprice > 100000.0 THEN TRUE ELSE FALSE END)
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.31.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.31.query.sqlpp
new file mode 100644
index 0000000..5ba532e
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.31.query.sqlpp
@@ -0,0 +1,27 @@
+/*
+ * 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;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey /*+indexnl*/ = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
+AND (CASE WHEN o.o_totalprice > 100000.0 THEN TRUE ELSE FALSE END)
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.40.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.40.query.sqlpp
new file mode 100644
index 0000000..10475eb
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.40.query.sqlpp
@@ -0,0 +1,27 @@
+/*
+ * 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;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey  = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
+AND (SELECT VALUE SUM(o.o_orderkey) FROM range(1, 10) x)[0] > 30000
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.41.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.41.query.sqlpp
new file mode 100644
index 0000000..d8b1908
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.41.query.sqlpp
@@ -0,0 +1,27 @@
+/*
+ * 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;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey /*+indexnl*/ = o.o_custkey
+AND o.o_comment NOT LIKE '%special%requests%'
+AND (SELECT VALUE SUM(o.o_orderkey) FROM range(1, 10) x)[0] > 30000
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.50.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.50.query.sqlpp
new file mode 100644
index 0000000..39f82b7
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.50.query.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 tpch;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey  = o.o_custkey
+WHERE o.o_comment NOT LIKE '%special%requests%'
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.51.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.51.query.sqlpp
new file mode 100644
index 0000000..e15d56e
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/join/nlj-outer/nlj-outer.51.query.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 tpch;
+
+SELECT VALUE COUNT(*)
+FROM            Customer c
+LEFT OUTER JOIN Orders o
+ON  c.c_custkey /*+indexnl*/ = o.o_custkey
+WHERE o.o_comment NOT LIKE '%special%requests%'
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/limit/offset_without_limit/offset_without_limit.7.query.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/limit/offset_without_limit/offset_without_limit.7.query.sqlpp
new file mode 100644
index 0000000..e0df538
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/limit/offset_without_limit/offset_without_limit.7.query.sqlpp
@@ -0,0 +1,27 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+/*
+ * Description     : Test that offset without limit is NOT pushed into a primary scan
+ * Expected Result : Success
+ */
+
+use test;
+
+select x, count(*) as count from [{"id":10, "x":1},{"id":10, "x":2}] a group by x order by x offset 1;
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/misc/query-ASTERIXDB-3316/query-ASTERIXDB-3316.1.ddl.sqlpp b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/misc/query-ASTERIXDB-3316/query-ASTERIXDB-3316.1.ddl.sqlpp
index baada5f..fbc5858 100644
--- a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/misc/query-ASTERIXDB-3316/query-ASTERIXDB-3316.1.ddl.sqlpp
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/misc/query-ASTERIXDB-3316/query-ASTERIXDB-3316.1.ddl.sqlpp
@@ -26,5 +26,6 @@
 
 use test;
 
-create dataset collection0 primary key (id:int);
-create dataset collection1 primary key (id:int);
+create type dt1 as {id:int};
+create dataset collection0(dt1) primary key id;
+create dataset collection1(dt1) primary key id;
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.10.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.10.adm
new file mode 100644
index 0000000..f768f58
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.10.adm
@@ -0,0 +1 @@
+1535
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.11.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.11.adm
new file mode 100644
index 0000000..f768f58
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.11.adm
@@ -0,0 +1 @@
+1535
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.20.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.20.adm
new file mode 100644
index 0000000..6bf42b9
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.20.adm
@@ -0,0 +1 @@
+757
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.21.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.21.adm
new file mode 100644
index 0000000..6bf42b9
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.21.adm
@@ -0,0 +1 @@
+757
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.30.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.30.adm
new file mode 100644
index 0000000..6bf42b9
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.30.adm
@@ -0,0 +1 @@
+757
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.31.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.31.adm
new file mode 100644
index 0000000..6bf42b9
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.31.adm
@@ -0,0 +1 @@
+757
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.40.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.40.adm
new file mode 100644
index 0000000..6822652
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.40.adm
@@ -0,0 +1 @@
+793
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.41.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.41.adm
new file mode 100644
index 0000000..6822652
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.41.adm
@@ -0,0 +1 @@
+793
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.50.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.50.adm
new file mode 100644
index 0000000..000b173
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.50.adm
@@ -0,0 +1 @@
+1485
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.51.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.51.adm
new file mode 100644
index 0000000..000b173
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/join/nlj-outer/nlj-outer.51.adm
@@ -0,0 +1 @@
+1485
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/results/limit/offset_without_limit/offset_without_limit.7.adm b/asterixdb/asterix-app/src/test/resources/runtimets/results/limit/offset_without_limit/offset_without_limit.7.adm
new file mode 100644
index 0000000..d29535d
--- /dev/null
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/results/limit/offset_without_limit/offset_without_limit.7.adm
@@ -0,0 +1 @@
+{ "x": 2, "count": 1 }
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/sqlpp_queries.xml b/asterixdb/asterix-app/src/test/resources/runtimets/sqlpp_queries.xml
index c6fe190..86f2b0b 100644
--- a/asterixdb/asterix-app/src/test/resources/runtimets/sqlpp_queries.xml
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/sqlpp_queries.xml
@@ -6749,6 +6749,11 @@
       </compilation-unit>
     </test-case>
     <test-case FilePath="join">
+      <compilation-unit name="nlj-outer">
+        <output-dir compare="Text">nlj-outer</output-dir>
+      </compilation-unit>
+    </test-case>
+    <test-case FilePath="join">
       <compilation-unit name="nlj_partitioning_property">
         <output-dir compare="Text">nlj_partitioning_property</output-dir>
       </compilation-unit>
diff --git a/asterixdb/asterix-lang-sqlpp/src/main/java/org/apache/asterix/lang/sqlpp/visitor/FreeVariableVisitor.java b/asterixdb/asterix-lang-sqlpp/src/main/java/org/apache/asterix/lang/sqlpp/visitor/FreeVariableVisitor.java
index 45fd4bf..37dd1f1 100644
--- a/asterixdb/asterix-lang-sqlpp/src/main/java/org/apache/asterix/lang/sqlpp/visitor/FreeVariableVisitor.java
+++ b/asterixdb/asterix-lang-sqlpp/src/main/java/org/apache/asterix/lang/sqlpp/visitor/FreeVariableVisitor.java
@@ -333,7 +333,12 @@
 
     @Override
     public Void visit(LimitClause limitClause, Collection<VariableExpr> freeVars) throws CompilationException {
-        limitClause.getLimitExpr().accept(this, freeVars);
+        if (limitClause.hasLimitExpr()) {
+            limitClause.getLimitExpr().accept(this, freeVars);
+        }
+        if (limitClause.hasOffset()) {
+            limitClause.getOffset().accept(this, freeVars);
+        }
         return null;
     }
 
diff --git a/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateLeftOuterJoinSelectsRule.java b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateLeftOuterJoinSelectsRule.java
new file mode 100644
index 0000000..3114524
--- /dev/null
+++ b/hyracks-fullstack/algebricks/algebricks-rewriter/src/main/java/org/apache/hyracks/algebricks/rewriter/rules/ConsolidateLeftOuterJoinSelectsRule.java
@@ -0,0 +1,112 @@
+/*
+ * 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.hyracks.algebricks.rewriter.rules;
+
+import java.util.ArrayList;
+import java.util.List;
+
+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.LogicalOperatorTag;
+import org.apache.hyracks.algebricks.core.algebra.base.LogicalVariable;
+import org.apache.hyracks.algebricks.core.algebra.expressions.IAlgebricksConstantValue;
+import org.apache.hyracks.algebricks.core.algebra.expressions.ScalarFunctionCallExpression;
+import org.apache.hyracks.algebricks.core.algebra.functions.AlgebricksBuiltinFunctions;
+import org.apache.hyracks.algebricks.core.algebra.functions.IFunctionInfo;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterJoinOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.LeftOuterUnnestMapOperator;
+import org.apache.hyracks.algebricks.core.algebra.operators.logical.SelectOperator;
+import org.apache.hyracks.algebricks.core.rewriter.base.IAlgebraicRewriteRule;
+
+/**
+ * In case {@link LeftOuterJoinOperator} gets transformed into {@link LeftOuterUnnestMapOperator}, the plan could
+ * end up with multiple SELECT operators (broken up by {@link BreakSelectIntoConjunctsRule}).
+ * This rule consolidate those SELECT operators back again.
+ * <p>
+ * Example:
+ * select (and([JOIN_CONDITION], eq($$o.getField("x"), 1)))
+ * -- unnest $$o <- dataset(...)
+ * <p>
+ * After {@link BreakSelectIntoConjunctsRule}
+ * select ([JOIN_CONDITION])
+ * -- select (eq($$o.getField("x"), 1))
+ * -- -- unnest $$o <- dataset(...)
+ * <p>
+ * Before accessMethod rewrite:
+ * left outer join ([JOIN_CONDITION])
+ * -- data-scan []<-[$$56, $$c] <- ...
+ * -- select (eq($$o.getField("x"), 1))
+ * -- -- data-scan []<-[$$57, $$o] <- ...
+ * <p>
+ * After accessMethod rewrite:
+ * select ([JOIN_CONDITION]) retain-untrue (... <- missing)
+ * -- select (eq($$o.getField("x"), 1))
+ * -- -- left-outer-unnest-map ...
+ * -- -- -- ...
+ * <p>
+ * After this rule:
+ * select (and([JOIN_CONDITION], eq($$o.getField("x"), 1))) retain-untrue (... <- missing)
+ * -- left-outer-unnest-map ...
+ */
+public class ConsolidateLeftOuterJoinSelectsRule implements IAlgebraicRewriteRule {
+    private final List<Mutable<ILogicalExpression>> conditions = new ArrayList<>();
+
+    @Override
+    public boolean rewritePre(Mutable<ILogicalOperator> opRef, IOptimizationContext context)
+            throws AlgebricksException {
+        ILogicalOperator op = opRef.getValue();
+        if (op.getOperatorTag() != LogicalOperatorTag.SELECT) {
+            return false;
+        }
+
+        conditions.clear();
+        IAlgebricksConstantValue missingValue = null;
+        LogicalVariable missingVariable = null;
+        ILogicalOperator nextOp = op;
+        do {
+            SelectOperator selectOp = (SelectOperator) nextOp;
+            if (missingValue == null) {
+                // Capture the SELECT that contains the retain missing value and variable placeholder
+                missingValue = selectOp.getRetainMissingAsValue();
+                missingVariable = selectOp.getMissingPlaceholderVariable();
+            }
+            conditions.add(new MutableObject<>(selectOp.getCondition().getValue()));
+            nextOp = nextOp.getInputs().get(0).getValue();
+        } while (nextOp.getOperatorTag() == LogicalOperatorTag.SELECT);
+
+        if (conditions.size() < 2 || missingValue == null) {
+            return false;
+        }
+
+        SelectOperator newSelect = new SelectOperator(createAndCondition(context), missingValue, missingVariable);
+        newSelect.getInputs().add(new MutableObject<>(nextOp));
+        opRef.setValue(newSelect);
+        context.computeAndSetTypeEnvironmentForOperator(newSelect);
+        return true;
+    }
+
+    private Mutable<ILogicalExpression> createAndCondition(IOptimizationContext context) {
+        IFunctionInfo fInfo = context.getMetadataProvider().lookupFunction(AlgebricksBuiltinFunctions.AND);
+        return new MutableObject<>(new ScalarFunctionCallExpression(fInfo, new ArrayList<>(conditions)));
+    }
+}
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/util/ExceptionUtils.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/util/ExceptionUtils.java
index e07cdd4..d616eb5 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/util/ExceptionUtils.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/util/ExceptionUtils.java
@@ -100,7 +100,7 @@
      *            the subsequent failure
      * @return the root exception, or null if both parameters are null
      */
-    public static Throwable suppress(Throwable first, Throwable second) {
+    public static <T extends Throwable> T suppress(T first, T second) {
         if (second instanceof InterruptedException) {
             Thread.currentThread().interrupt();
         }
diff --git a/hyracks-fullstack/hyracks/hyracks-http/src/main/java/org/apache/hyracks/http/server/HttpServer.java b/hyracks-fullstack/hyracks/hyracks-http/src/main/java/org/apache/hyracks/http/server/HttpServer.java
index 97b8859..271de53 100644
--- a/hyracks-fullstack/hyracks/hyracks-http/src/main/java/org/apache/hyracks/http/server/HttpServer.java
+++ b/hyracks-fullstack/hyracks/hyracks-http/src/main/java/org/apache/hyracks/http/server/HttpServer.java
@@ -21,7 +21,9 @@
 import java.io.IOException;
 import java.net.InetSocketAddress;
 import java.util.ArrayList;
+import java.util.Collection;
 import java.util.Collections;
+import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Set;
 import java.util.concurrent.ConcurrentHashMap;
@@ -30,7 +32,10 @@
 import java.util.concurrent.ThreadPoolExecutor;
 import java.util.concurrent.TimeUnit;
 import java.util.concurrent.atomic.AtomicInteger;
+import java.util.stream.Collectors;
 
+import org.apache.commons.lang3.tuple.Pair;
+import org.apache.hyracks.api.util.ExceptionUtils;
 import org.apache.hyracks.http.api.IChannelClosedHandler;
 import org.apache.hyracks.http.api.IServlet;
 import org.apache.hyracks.util.MXHelper;
@@ -80,8 +85,7 @@
     private final ServletRegistry servlets;
     private final EventLoopGroup bossGroup;
     private final EventLoopGroup workerGroup;
-    private final InetSocketAddress defaultAddress;
-    private final List<InetSocketAddress> addresses;
+    private final Set<InetSocketAddress> addresses;
     private final ThreadPoolExecutor executor;
     // Mutable members
     private volatile int state = STOPPED;
@@ -114,15 +118,14 @@
         this(bossGroup, workerGroup, Collections.singletonList(address), config, closeHandler);
     }
 
-    public HttpServer(EventLoopGroup bossGroup, EventLoopGroup workerGroup, List<InetSocketAddress> addresses,
+    public HttpServer(EventLoopGroup bossGroup, EventLoopGroup workerGroup, Collection<InetSocketAddress> addresses,
             HttpServerConfig config, IChannelClosedHandler closeHandler) {
         if (addresses.isEmpty()) {
             throw new IllegalArgumentException("no addresses specified");
         }
         this.bossGroup = bossGroup;
         this.workerGroup = workerGroup;
-        this.addresses = addresses;
-        defaultAddress = addresses.get(0);
+        this.addresses = new LinkedHashSet<>(addresses);
         this.closedHandler = closeHandler;
         this.config = config;
         channels = new ArrayList<>();
@@ -130,9 +133,16 @@
         servlets = new ServletRegistry();
         workQueue = new LinkedBlockingQueue<>(config.getRequestQueueSize());
         int numExecutorThreads = config.getThreadCount();
+        int[] ports = this.addresses.stream().mapToInt(InetSocketAddress::getPort).distinct().toArray();
+        String desc;
+        if (ports.length > 1) {
+            desc = this.addresses.stream().map(a -> a.getAddress().getHostAddress() + ":" + a.getPort())
+                    .collect(Collectors.joining(",", "[", "]"));
+        } else {
+            desc = "port:" + ports[0];
+        }
         executor = new ThreadPoolExecutor(numExecutorThreads, numExecutorThreads, 0L, TimeUnit.MILLISECONDS, workQueue,
-                runnable -> new Thread(runnable,
-                        "HttpExecutor(port:" + defaultAddress.getPort() + ")-" + threadId.getAndIncrement()));
+                runnable -> new Thread(runnable, "HttpExecutor(" + desc + ")-" + threadId.getAndIncrement()));
         long directMemoryBudget = numExecutorThreads * (long) HIGH_WRITE_BUFFER_WATER_MARK
                 + numExecutorThreads * config.getMaxResponseChunkSize();
         LOGGER.log(Level.DEBUG,
@@ -156,7 +166,6 @@
                 doStart();
                 setStarted();
             } catch (Throwable e) { // NOSONAR
-                LOGGER.error("Failure starting an Http Server at: {}", defaultAddress, e);
                 setFailed(e);
                 throw e;
             }
@@ -255,14 +264,19 @@
         return servlets.getServlets();
     }
 
-    protected void doStart() throws InterruptedException, IOException {
+    protected void doStart() throws Exception {
         for (IServlet servlet : servlets.getServlets()) {
-            servlet.init();
+            try {
+                servlet.init();
+            } catch (IOException e) {
+                LOGGER.error("Failure initializing servlet {} on http server {}", servlet, addresses, e);
+                throw e;
+            }
         }
         bind();
     }
 
-    private void bind() throws InterruptedException {
+    private void bind() throws Exception {
         ServerBootstrap b = new ServerBootstrap();
         b.group(bossGroup, workerGroup).channel(NioServerSocketChannel.class)
                 .childOption(ChannelOption.RCVBUF_ALLOCATOR, new FixedRecvByteBufAllocator(RECEIVE_BUFFER_SIZE))
@@ -270,17 +284,28 @@
                 .childOption(ChannelOption.ALLOCATOR, PooledByteBufAllocator.DEFAULT)
                 .childOption(ChannelOption.WRITE_BUFFER_WATER_MARK, WRITE_BUFFER_WATER_MARK)
                 .handler(new LoggingHandler(LogLevel.DEBUG)).childHandler(getChannelInitializer());
-        List<ChannelFuture> channelFutures = new ArrayList<>();
+        List<Pair<InetSocketAddress, ChannelFuture>> channelFutures = new ArrayList<>();
         for (InetSocketAddress address : addresses) {
-            channelFutures.add(b.bind(address));
+            channelFutures.add(org.apache.commons.lang3.tuple.Pair.of(address, b.bind(address)));
         }
-        for (ChannelFuture future : channelFutures) {
-            Channel channel = future.sync().channel();
-            channel.closeFuture().addListener(channelCloseListener);
-            synchronized (lock) {
-                channels.add(channel);
+        Exception failure = null;
+        for (Pair<InetSocketAddress, ChannelFuture> addressFuture : channelFutures) {
+            try {
+                Channel channel = addressFuture.getRight().sync().channel();
+                channel.closeFuture().addListener(channelCloseListener);
+                synchronized (lock) {
+                    channels.add(channel);
+                }
+            } catch (InterruptedException e) {
+                throw e;
+            } catch (Exception e) {
+                LOGGER.error("Bind failure starting http server at {}", addressFuture.getLeft(), e);
+                failure = ExceptionUtils.suppress(failure, e);
             }
         }
+        if (failure != null) {
+            throw failure;
+        }
     }
 
     private void triggerRecovery() throws InterruptedException {
@@ -395,7 +420,7 @@
 
     @Override
     public String toString() {
-        return "{\"class\":\"" + getClass().getSimpleName() + "\",\"address\":" + defaultAddress + ",\"state\":\""
+        return "{\"class\":\"" + getClass().getSimpleName() + "\",\"addresses\":" + addresses + ",\"state\":\""
                 + getState() + "\"}";
     }
 
@@ -403,8 +428,9 @@
         return config;
     }
 
+    @Deprecated // this returns an arbitrary (the first supplied if collection is ordered) address
     public InetSocketAddress getAddress() {
-        return defaultAddress;
+        return addresses.iterator().next();
     }
 
     private void closeChannels() throws InterruptedException {