You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@calcite.apache.org by jc...@apache.org on 2018/06/22 18:49:13 UTC

[2/3] calcite git commit: [CALCITE-2286] Support timestamp type for Druid adapter

http://git-wip-us.apache.org/repos/asf/calcite/blob/b29397d9/druid/src/test/java/org/apache/calcite/test/DruidAdapterIT2.java
----------------------------------------------------------------------
diff --git a/druid/src/test/java/org/apache/calcite/test/DruidAdapterIT2.java b/druid/src/test/java/org/apache/calcite/test/DruidAdapterIT2.java
new file mode 100644
index 0000000..92e2322
--- /dev/null
+++ b/druid/src/test/java/org/apache/calcite/test/DruidAdapterIT2.java
@@ -0,0 +1,3979 @@
+/*
+ * 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.calcite.test;
+
+import org.apache.calcite.adapter.druid.DruidQuery;
+import org.apache.calcite.adapter.druid.DruidSchema;
+import org.apache.calcite.config.CalciteConnectionConfig;
+import org.apache.calcite.config.CalciteConnectionProperty;
+import org.apache.calcite.rel.RelNode;
+import org.apache.calcite.rel.type.RelDataType;
+import org.apache.calcite.rex.RexNode;
+import org.apache.calcite.schema.impl.AbstractSchema;
+import org.apache.calcite.sql.fun.SqlStdOperatorTable;
+import org.apache.calcite.sql.type.SqlTypeName;
+import org.apache.calcite.tools.RelBuilder;
+import org.apache.calcite.util.Util;
+
+import com.google.common.base.Function;
+import com.google.common.collect.ArrayListMultimap;
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.ImmutableMap;
+import com.google.common.collect.Multimap;
+
+import org.junit.Test;
+
+import java.net.URL;
+import java.sql.Connection;
+import java.sql.DatabaseMetaData;
+import java.sql.ResultSet;
+import java.sql.SQLException;
+import java.util.List;
+
+import static org.hamcrest.CoreMatchers.containsString;
+import static org.hamcrest.CoreMatchers.is;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertSame;
+import static org.junit.Assert.assertThat;
+import static org.junit.Assert.assertTrue;
+
+/**
+ * Tests for the {@code org.apache.calcite.adapter.druid} package.
+ *
+ * <p>Before calling this test, you need to populate Druid, as follows:
+ *
+ * <blockquote><code>
+ * git clone https://github.com/vlsi/calcite-test-dataset<br>
+ * cd calcite-test-dataset<br>
+ * mvn install
+ * </code></blockquote>
+ *
+ * <p>This will create a virtual machine with Druid and test data set.
+ *
+ * <p>Features not yet implemented:
+ * <ul>
+ *   <li>push LIMIT into "select" query</li>
+ *   <li>push SORT and/or LIMIT into "groupBy" query</li>
+ *   <li>push HAVING into "groupBy" query</li>
+ * </ul>
+ *
+ * These tests use "timestamp" type for the Druid timestamp column, instead
+ * of "timestamp with local time zone" type as {@link DruidAdapterIT}.
+ */
+public class DruidAdapterIT2 {
+  /** URL of the "druid-foodmart" model. */
+  public static final URL FOODMART =
+      DruidAdapterIT2.class.getResource("/druid-foodmart-model-timestamp.json");
+
+  /** Whether to run Druid tests. Enabled by default, however test is only
+   * included if "it" profile is activated ({@code -Pit}). To disable,
+   * specify {@code -Dcalcite.test.druid=false} on the Java command line. */
+  public static final boolean ENABLED =
+      Util.getBooleanProperty("calcite.test.druid", true);
+
+  private static final String VARCHAR_TYPE =
+      "VARCHAR CHARACTER SET \"ISO-8859-1\" COLLATE \"ISO-8859-1$en_US$primary\"";
+
+  private static final String FOODMART_TABLE = "\"foodmart\"";
+
+  /** Whether to run this test. */
+  protected boolean enabled() {
+    return ENABLED;
+  }
+
+  /** Returns a function that checks that a particular Druid query is
+   * generated to implement a query. */
+  private static Function<List, Void> druidChecker(final String... lines) {
+    return new Function<List, Void>() {
+      public Void apply(List list) {
+        assertThat(list.size(), is(1));
+        DruidQuery.QuerySpec querySpec = (DruidQuery.QuerySpec) list.get(0);
+        for (String line : lines) {
+          final String s = line.replace('\'', '"');
+          assertThat(querySpec.getQueryString(null, -1), containsString(s));
+        }
+        return null;
+      }
+    };
+  }
+
+  /**
+   * Creates a query against FOODMART with approximate parameters
+   * */
+  private CalciteAssert.AssertQuery foodmartApprox(String sql) {
+    return approxQuery(FOODMART, sql);
+  }
+
+  private CalciteAssert.AssertQuery approxQuery(URL url, String sql) {
+    return CalciteAssert.that()
+        .enable(enabled())
+        .with(ImmutableMap.of("model", url.getPath()))
+        .with(CalciteConnectionProperty.APPROXIMATE_DISTINCT_COUNT.camelName(), true)
+        .with(CalciteConnectionProperty.APPROXIMATE_TOP_N.camelName(), true)
+        .with(CalciteConnectionProperty.APPROXIMATE_DECIMAL.camelName(), true)
+        .query(sql);
+  }
+
+  /** Creates a query against a data set given by a map. */
+  private CalciteAssert.AssertQuery sql(String sql, URL url) {
+    return CalciteAssert.that()
+        .enable(enabled())
+        .with(ImmutableMap.of("model", url.getPath()))
+        .query(sql);
+  }
+
+  /** Creates a query against the {@link #FOODMART} data set. */
+  private CalciteAssert.AssertQuery sql(String sql) {
+    return sql(sql, FOODMART);
+  }
+
+  @Test public void testMetadataColumns() throws Exception {
+    sql("values 1")
+        .withConnection(
+            new Function<Connection, Void>() {
+              public Void apply(Connection c) {
+                try {
+                  final DatabaseMetaData metaData = c.getMetaData();
+                  final ResultSet r =
+                      metaData.getColumns(null, null, "foodmart", null);
+                  Multimap<String, Boolean> map = ArrayListMultimap.create();
+                  while (r.next()) {
+                    map.put(r.getString("TYPE_NAME"), true);
+                  }
+                  System.out.println(map);
+                  // 1 timestamp, 2 float measure, 1 int measure, 88 dimensions
+                  assertThat(map.keySet().size(), is(4));
+                  assertThat(map.values().size(), is(92));
+                  assertThat(map.get("TIMESTAMP(0) NOT NULL").size(), is(1));
+                  assertThat(map.get("DOUBLE").size(), is(2));
+                  assertThat(map.get("BIGINT").size(), is(1));
+                  assertThat(map.get(VARCHAR_TYPE).size(), is(88));
+                } catch (SQLException e) {
+                  throw new RuntimeException(e);
+                }
+                return null;
+              }
+            });
+  }
+
+  @Test public void testSelectDistinct() {
+    final String explain = "PLAN="
+        + "EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], groups=[{30}], aggs=[[]])";
+    final String sql = "select distinct \"state_province\" from \"foodmart\"";
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart','granularity':'all',"
+        + "'dimensions':[{'type':'default','dimension':'state_province','outputName':'state_province'"
+        + ",'outputType':'STRING'}],'limitSpec':{'type':'default'},"
+        + "'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sql)
+        .returnsUnordered("state_province=CA",
+            "state_province=OR",
+            "state_province=WA")
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testSelectGroupBySum() {
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], "
+        + "projects=[[$30, CAST($89):INTEGER]], groups=[{0}], aggs=[[SUM($1)]])";
+    final String sql = "select \"state_province\", sum(cast(\"unit_sales\" as integer)) as u\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\"";
+    sql(sql)
+        .returnsUnordered("state_province=CA; U=74748",
+            "state_province=OR; U=67659",
+            "state_province=WA; U=124366")
+        .explainContains(explain);
+  }
+
+  @Test public void testGroupbyMetric() {
+    final String sql = "select  \"store_sales\" ,\"product_id\" from \"foodmart\" "
+        + "where \"product_id\" = 1020" + "group by \"store_sales\" ,\"product_id\" ";
+    final String plan = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], filter=[=($1, 1020)],"
+        + " projects=[[$90, $1]], groups=[{0, 1}], aggs=[[]])";
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart','granularity':'all',"
+        + "'dimensions':[{'type':'default','dimension':'store_sales',\"outputName\":\"store_sales\","
+        + "'outputType':'DOUBLE'},{'type':'default','dimension':'product_id','outputName':"
+        + "'product_id','outputType':'STRING'}],'limitSpec':{'type':'default'},"
+        + "'filter':{'type':'bound','dimension':'product_id','lower':'1020','lowerStrict':false,"
+        + "'upper':'1020','upperStrict':false,'ordering':'numeric'},'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sql)
+        .explainContains(plan)
+        .queryContains(druidChecker(druidQuery))
+        .returnsUnordered("store_sales=0.51; product_id=1020",
+            "store_sales=1.02; product_id=1020",
+            "store_sales=1.53; product_id=1020",
+            "store_sales=2.04; product_id=1020",
+            "store_sales=2.55; product_id=1020");
+  }
+
+  @Test public void testPushSimpleGroupBy() {
+    final String sql = "select \"product_id\" from \"foodmart\" where "
+        + "\"product_id\" = 1020 group by \"product_id\"";
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'default',"
+        + "'dimension':'product_id','outputName':'product_id','outputType':'STRING'}],"
+        + "'limitSpec':{'type':'default'},'filter':{'type':'bound','dimension':'product_id',"
+        + "'lower':'1020','lowerStrict':false,'upper':'1020','upperStrict':false,"
+        + "'ordering':'numeric'},'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sql).returnsUnordered("product_id=1020").queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testComplexPushGroupBy() {
+    final String innerQuery = "select \"product_id\" as \"id\" from \"foodmart\" where "
+        + "\"product_id\" = 1020";
+    final String sql = "select \"id\" from (" + innerQuery + ") group by \"id\"";
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all',"
+        + "'dimensions':[{'type':'default','dimension':'product_id','outputName':'product_id',"
+        + "'outputType':'STRING'}],'limitSpec':{'type':'default'},"
+        + "'filter':{'type':'bound','dimension':'product_id','lower':'1020','lowerStrict':false,"
+        + "'upper':'1020','upperStrict':false,'ordering':'numeric'},'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sql)
+        .returnsUnordered("id=1020")
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1281">[CALCITE-1281]
+   * Druid adapter wrongly returns all numeric values as int or float</a>. */
+  @Test public void testSelectCount() {
+    final String sql = "select count(*) as c from \"foodmart\"";
+    sql(sql)
+        .returns(new Function<ResultSet, Void>() {
+          public Void apply(ResultSet input) {
+            try {
+              assertThat(input.next(), is(true));
+              assertThat(input.getInt(1), is(86829));
+              assertThat(input.getLong(1), is(86829L));
+              assertThat(input.getString(1), is("86829"));
+              assertThat(input.wasNull(), is(false));
+              assertThat(input.next(), is(false));
+              return null;
+            } catch (SQLException e) {
+              throw new RuntimeException(e);
+            }
+          }
+        });
+  }
+
+  @Test public void testSort() {
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], projects=[[$39, $30]], "
+        + "groups=[{0, 1}], aggs=[[]], sort0=[1], sort1=[0], dir0=[ASC], dir1=[DESC])";
+    final String sql = "select distinct \"gender\", \"state_province\"\n"
+        + "from \"foodmart\" order by 2, 1 desc";
+    sql(sql)
+        .returnsOrdered("gender=M; state_province=CA",
+            "gender=F; state_province=CA",
+            "gender=M; state_province=OR",
+            "gender=F; state_province=OR",
+            "gender=M; state_province=WA",
+            "gender=F; state_province=WA")
+        .queryContains(
+            druidChecker("{'queryType':'groupBy','dataSource':'foodmart','granularity':'all',"
+                + "'dimensions':[{'type':'default','dimension':'gender','outputName':'gender',"
+                + "'outputType':'STRING'},{'type':'default','dimension':'state_province',"
+                + "'outputName':'state_province','outputType':'STRING'}],'limitSpec':"
+                + "{'type':'default','columns':[{'dimension':'state_province','direction':'ascending'"
+                + ",'dimensionOrder':'lexicographic'},{'dimension':'gender','direction':'descending',"
+                + "'dimensionOrder':'lexicographic'}]},'aggregations':[],"
+                + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}"))
+        .explainContains(explain);
+  }
+
+  @Test public void testSortLimit() {
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  BindableSort(sort0=[$1], sort1=[$0], dir0=[ASC], dir1=[DESC], offset=[2], fetch=[3])\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], projects=[[$39, $30]], "
+        + "groups=[{0, 1}], aggs=[[]], sort0=[1], sort1=[0], dir0=[ASC], dir1=[DESC])";
+    final String sql = "select distinct \"gender\", \"state_province\"\n"
+        + "from \"foodmart\"\n"
+        + "order by 2, 1 desc offset 2 rows fetch next 3 rows only";
+    sql(sql)
+        .returnsOrdered("gender=M; state_province=OR",
+            "gender=F; state_province=OR",
+            "gender=M; state_province=WA")
+        .explainContains(explain);
+  }
+
+  @Test public void testOffsetLimit() {
+    // We do not yet push LIMIT into a Druid "select" query as a "threshold".
+    // It is not possible to push OFFSET into Druid "select" query.
+    final String sql = "select \"state_province\", \"product_name\"\n"
+        + "from \"foodmart\"\n"
+        + "offset 2 fetch next 3 rows only";
+    final String druidQuery = "{'queryType':'scan','dataSource':'foodmart',"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'columns':['state_province','product_name'],"
+        + "'resultFormat':'compactedList'}";
+    sql(sql)
+        .runs()
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testLimit() {
+    final String sql = "select \"gender\", \"state_province\"\n"
+        + "from \"foodmart\" fetch next 3 rows only";
+    final String druidQuery = "{'queryType':'scan','dataSource':'foodmart',"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'columns':['gender','state_province'],"
+        + "'resultFormat':'compactedList','limit':3";
+    sql(sql)
+        .runs()
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testDistinctLimit() {
+    final String sql = "select distinct \"gender\", \"state_province\"\n"
+        + "from \"foodmart\" fetch next 3 rows only";
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'default','dimension':'gender',"
+        + "'outputName':'gender','outputType':'STRING'},"
+        + "{'type':'default','dimension':'state_province','outputName':'state_province',"
+        + "'outputType':'STRING'}],'limitSpec':{'type':'default',"
+        + "'limit':3,'columns':[]},"
+        + "'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], projects=[[$39, $30]], "
+        + "groups=[{0, 1}], aggs=[[]], fetch=[3])";
+    sql(sql)
+        .runs()
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQuery))
+        .returnsUnordered("gender=F; state_province=CA", "gender=F; state_province=OR",
+            "gender=F; state_province=WA");
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1578">[CALCITE-1578]
+   * Druid adapter: wrong semantics of topN query limit with granularity</a>. */
+  @Test public void testGroupBySortLimit() {
+    final String sql = "select \"brand_name\", \"gender\", sum(\"unit_sales\") as s\n"
+        + "from \"foodmart\"\n"
+        + "group by \"brand_name\", \"gender\"\n"
+        + "order by s desc limit 3";
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'default',"
+        + "'dimension':'brand_name','outputName':'brand_name','outputType':'STRING'},"
+        + "{'type':'default','dimension':'gender','outputName':'gender','outputType':'STRING'}],"
+        + "'limitSpec':{'type':'default','limit':3,'columns':[{'dimension':'S',"
+        + "'direction':'descending','dimensionOrder':'numeric'}]},"
+        + "'aggregations':[{'type':'longSum','name':'S','fieldName':'unit_sales'}],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[$2, $39, $89]], groups=[{0, 1}], "
+        + "aggs=[[SUM($2)]], sort0=[2], dir0=[DESC], fetch=[3])";
+    sql(sql)
+        .runs()
+        .returnsOrdered("brand_name=Hermanos; gender=M; S=4286",
+            "brand_name=Hermanos; gender=F; S=4183",
+            "brand_name=Tell Tale; gender=F; S=4033")
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1587">[CALCITE-1587]
+   * Druid adapter: topN returns approximate results</a>. */
+  @Test public void testGroupBySingleSortLimit() {
+    checkGroupBySingleSortLimit(false);
+  }
+
+  /** As {@link #testGroupBySingleSortLimit}, but allowing approximate results
+   * due to {@link CalciteConnectionConfig#approximateDistinctCount()}.
+   * Therefore we send a "topN" query to Druid. */
+  @Test public void testGroupBySingleSortLimitApprox() {
+    checkGroupBySingleSortLimit(true);
+  }
+
+  private void checkGroupBySingleSortLimit(boolean approx) {
+    final String sql = "select \"brand_name\", sum(\"unit_sales\") as s\n"
+        + "from \"foodmart\"\n"
+        + "group by \"brand_name\"\n"
+        + "order by s desc limit 3";
+    final String approxDruid = "{'queryType':'topN','dataSource':'foodmart','granularity':'all',"
+        + "'dimension':{'type':'default','dimension':'brand_name','outputName':'brand_name','outputType':'STRING'},'metric':'S',"
+        + "'aggregations':[{'type':'longSum','name':'S','fieldName':'unit_sales'}],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'threshold':3}";
+    final String exactDruid = "{'queryType':'groupBy','dataSource':'foodmart','granularity':'all',"
+        + "'dimensions':[{'type':'default','dimension':'brand_name','outputName':'brand_name',"
+        + "'outputType':'STRING'}],'limitSpec':{'type':'default','limit':3,'columns':"
+        + "[{'dimension':'S','direction':'descending','dimensionOrder':'numeric'}]},'aggregations':"
+        + "[{'type':'longSum','name':'S','fieldName':'unit_sales'}],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    final String druidQuery = approx ? approxDruid : exactDruid;
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[$2, $89]], groups=[{0}], "
+        + "aggs=[[SUM($1)]], sort0=[1], dir0=[DESC], fetch=[3])";
+    CalciteAssert.that()
+        .enable(enabled())
+        .with(ImmutableMap.of("model", FOODMART.getPath()))
+        .with(CalciteConnectionProperty.APPROXIMATE_TOP_N.name(), approx)
+        .query(sql)
+        .runs()
+        .returnsOrdered("brand_name=Hermanos; S=8469",
+            "brand_name=Tell Tale; S=7877",
+            "brand_name=Ebony; S=7438")
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1578">[CALCITE-1578]
+   * Druid adapter: wrong semantics of groupBy query limit with granularity</a>.
+   *
+   * <p>Before CALCITE-1578 was fixed, this would use a "topN" query but return
+   * the wrong results. */
+  @Test public void testGroupByDaySortDescLimit() {
+    final String sql = "select \"brand_name\","
+        + " floor(\"timestamp\" to DAY) as d,"
+        + " sum(\"unit_sales\") as s\n"
+        + "from \"foodmart\"\n"
+        + "group by \"brand_name\", floor(\"timestamp\" to DAY)\n"
+        + "order by s desc limit 30";
+    final String explain =
+        "PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+            + "2992-01-10T00:00:00.000Z]], projects=[[$2, FLOOR($0, FLAG(DAY)), $89]], "
+            + "groups=[{0, 1}], aggs=[[SUM($2)]], sort0=[2], dir0=[DESC], fetch=[30])";
+    sql(sql)
+        .runs()
+        .returnsStartingWith("brand_name=Ebony; D=1997-07-27 00:00:00; S=135",
+            "brand_name=Tri-State; D=1997-05-09 00:00:00; S=120",
+            "brand_name=Hermanos; D=1997-05-09 00:00:00; S=115")
+        .explainContains(explain)
+        .queryContains(
+            druidChecker("'queryType':'groupBy'", "'granularity':'all'", "'limitSpec"
+                + "':{'type':'default','limit':30,'columns':[{'dimension':'S',"
+                + "'direction':'descending','dimensionOrder':'numeric'}]}"));
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1579">[CALCITE-1579]
+   * Druid adapter: wrong semantics of groupBy query limit with
+   * granularity</a>.
+   *
+   * <p>Before CALCITE-1579 was fixed, this would use a "groupBy" query but
+   * wrongly try to use a {@code limitSpec} to sort and filter. (A "topN" query
+   * was not possible because the sort was {@code ASC}.) */
+  @Test public void testGroupByDaySortLimit() {
+    final String sql = "select \"brand_name\","
+        + " floor(\"timestamp\" to DAY) as d,"
+        + " sum(\"unit_sales\") as s\n"
+        + "from \"foodmart\"\n"
+        + "group by \"brand_name\", floor(\"timestamp\" to DAY)\n"
+        + "order by s desc limit 30";
+    final String druidQueryPart1 = "{'queryType':'groupBy','dataSource':'foodmart'";
+    final String druidQueryPart2 = "'limitSpec':{'type':'default','limit':30,"
+        + "'columns':[{'dimension':'S','direction':'descending',"
+        + "'dimensionOrder':'numeric'}]},'aggregations':[{'type':'longSum',"
+        + "'name':'S','fieldName':'unit_sales'}],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[$2, FLOOR($0, FLAG(DAY)), $89]], groups=[{0, 1}], "
+        + "aggs=[[SUM($2)]], sort0=[2], dir0=[DESC], fetch=[30])";
+    sql(sql)
+        .runs()
+        .returnsStartingWith("brand_name=Ebony; D=1997-07-27 00:00:00; S=135",
+            "brand_name=Tri-State; D=1997-05-09 00:00:00; S=120",
+            "brand_name=Hermanos; D=1997-05-09 00:00:00; S=115")
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQueryPart1, druidQueryPart2));
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1580">[CALCITE-1580]
+   * Druid adapter: Wrong semantics for ordering within groupBy queries</a>. */
+  @Test public void testGroupByDaySortDimension() {
+    final String sql =
+        "select \"brand_name\", floor(\"timestamp\" to DAY) as d,"
+            + " sum(\"unit_sales\") as s\n"
+            + "from \"foodmart\"\n"
+            + "group by \"brand_name\", floor(\"timestamp\" to DAY)\n"
+            + "order by \"brand_name\"";
+    final String subDruidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'default',"
+        + "'dimension':'brand_name','outputName':'brand_name','outputType':'STRING'},"
+        + "{'type':'extraction','dimension':'__time',"
+        + "'outputName':'floor_day','extractionFn':{'type':'timeFormat'";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[$2, FLOOR($0, FLAG(DAY)), $89]], groups=[{0, 1}],"
+        + " aggs=[[SUM($2)]], sort0=[0], dir0=[ASC])";
+    sql(sql)
+        .runs()
+        .returnsStartingWith("brand_name=ADJ; D=1997-01-11 00:00:00; S=2",
+            "brand_name=ADJ; D=1997-01-12 00:00:00; S=3",
+            "brand_name=ADJ; D=1997-01-17 00:00:00; S=3")
+        .explainContains(explain)
+        .queryContains(druidChecker(subDruidQuery));
+  }
+
+  /** Tests a query that contains no GROUP BY and is therefore executed as a
+   * Druid "select" query. */
+  @Test public void testFilterSortDesc() {
+    final String sql = "select \"product_name\" from \"foodmart\"\n"
+        + "where \"product_id\" BETWEEN '1500' AND '1502'\n"
+        + "order by \"state_province\" desc, \"product_id\"";
+    final String druidQuery = "{'queryType':'scan','dataSource':'foodmart',"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'filter':{'type':'and','fields':["
+        + "{'type':'bound','dimension':'product_id','lower':'1500','lowerStrict':false,'ordering':'lexicographic'},"
+        + "{'type':'bound','dimension':'product_id','upper':'1502','upperStrict':false,'ordering':'lexicographic'}]},"
+        + "'columns':['product_name','state_province','product_id'],"
+        + "'resultFormat':'compactedList'";
+    sql(sql)
+        .limit(4)
+        .returns(
+            new Function<ResultSet, Void>() {
+              public Void apply(ResultSet resultSet) {
+                try {
+                  for (int i = 0; i < 4; i++) {
+                    assertTrue(resultSet.next());
+                    assertThat(resultSet.getString("product_name"),
+                        is("Fort West Dried Apricots"));
+                  }
+                  assertFalse(resultSet.next());
+                  return null;
+                } catch (SQLException e) {
+                  throw new RuntimeException(e);
+                }
+              }
+            })
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  /** As {@link #testFilterSortDesc()} but the bounds are numeric. */
+  @Test public void testFilterSortDescNumeric() {
+    final String sql = "select \"product_name\" from \"foodmart\"\n"
+        + "where \"product_id\" BETWEEN 1500 AND 1502\n"
+        + "order by \"state_province\" desc, \"product_id\"";
+    final String druidQuery = "{'queryType':'scan','dataSource':'foodmart',"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'filter':{'type':'and','fields':["
+        + "{'type':'bound','dimension':'product_id','lower':'1500','lowerStrict':false,'ordering':'numeric'},"
+        + "{'type':'bound','dimension':'product_id','upper':'1502','upperStrict':false,'ordering':'numeric'}]},"
+        + "'columns':['product_name','state_province','product_id'],"
+        + "'resultFormat':'compactedList'";
+    sql(sql)
+        .limit(4)
+        .returns(
+            new Function<ResultSet, Void>() {
+              public Void apply(ResultSet resultSet) {
+                try {
+                  for (int i = 0; i < 4; i++) {
+                    assertTrue(resultSet.next());
+                    assertThat(resultSet.getString("product_name"),
+                        is("Fort West Dried Apricots"));
+                  }
+                  assertFalse(resultSet.next());
+                  return null;
+                } catch (SQLException e) {
+                  throw new RuntimeException(e);
+                }
+              }
+            })
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  /** Tests a query whose filter removes all rows. */
+  @Test public void testFilterOutEverything() {
+    final String sql = "select \"product_name\" from \"foodmart\"\n"
+        + "where \"product_id\" = -1";
+    final String druidQuery = "{'queryType':'scan','dataSource':'foodmart',"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'filter':{'type':'bound','dimension':'product_id','lower':'-1','lowerStrict':false,"
+        + "'upper':'-1','upperStrict':false,'ordering':'numeric'},"
+        + "'columns':['product_name'],"
+        + "'resultFormat':'compactedList'}";
+    sql(sql)
+        .limit(4)
+        .returnsUnordered()
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  /** As {@link #testFilterSortDescNumeric()} but with a filter that cannot
+   * be pushed down to Druid. */
+  @Test public void testNonPushableFilterSortDesc() {
+    final String sql = "select \"product_name\" from \"foodmart\"\n"
+        + "where cast(\"product_id\" as integer) - 1500 BETWEEN 0 AND 2\n"
+        + "order by \"state_province\" desc, \"product_id\"";
+    final String druidQuery = "{'queryType':'scan','dataSource':'foodmart',"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],";
+    final String druidFilter = "\"filter\":{\"type\":\"and\","
+        + "\"fields\":[{\"type\":\"expression\",\"expression\":\"((CAST(\\\"product_id\\\"";
+    final String druidQuery2 = "'columns':['product_name','state_province','product_id'],"
+        + "'resultFormat':'compactedList'}";
+
+    sql(sql)
+        .limit(4)
+        .returns(
+            new Function<ResultSet, Void>() {
+              public Void apply(ResultSet resultSet) {
+                try {
+                  for (int i = 0; i < 4; i++) {
+                    assertTrue(resultSet.next());
+                    assertThat(resultSet.getString("product_name"),
+                        is("Fort West Dried Apricots"));
+                  }
+                  assertFalse(resultSet.next());
+                  return null;
+                } catch (SQLException e) {
+                  throw new RuntimeException(e);
+                }
+              }
+            })
+        .queryContains(druidChecker(druidQuery, druidFilter, druidQuery2));
+  }
+
+  @Test public void testUnionPlan() {
+    final String sql = "select distinct \"gender\" from \"foodmart\"\n"
+        + "union all\n"
+        + "select distinct \"marital_status\" from \"foodmart\"";
+    final String explain = "PLAN="
+        + "EnumerableInterpreter\n"
+        + "  BindableUnion(all=[true])\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], groups=[{39}], aggs=[[]])\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], groups=[{37}], aggs=[[]])";
+    sql(sql)
+        .explainContains(explain)
+        .returnsUnordered(
+            "gender=F",
+            "gender=M",
+            "gender=M",
+            "gender=S");
+  }
+
+  @Test public void testFilterUnionPlan() {
+    final String sql = "select * from (\n"
+        + "  select distinct \"gender\" from \"foodmart\"\n"
+        + "  union all\n"
+        + "  select distinct \"marital_status\" from \"foodmart\")\n"
+        + "where \"gender\" = 'M'";
+    final String explain = "PLAN="
+        + "EnumerableInterpreter\n"
+        + "  BindableFilter(condition=[=($0, 'M')])\n"
+        + "    BindableUnion(all=[true])\n"
+        + "      DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], groups=[{39}], aggs=[[]])\n"
+        + "      DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], groups=[{37}], aggs=[[]])";
+    sql(sql)
+        .explainContains(explain)
+        .returnsUnordered("gender=M",
+            "gender=M");
+  }
+
+  @Test public void testCountGroupByEmpty() {
+    final String druidQuery = "{'queryType':'timeseries','dataSource':'foodmart',"
+        + "'descending':false,'granularity':'all',"
+        + "'aggregations':[{'type':'count','name':'EXPR$0'}],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'context':{'skipEmptyBuckets':false}}";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[0]], groups=[{}], aggs=[[COUNT()]])";
+    final String sql = "select count(*) from \"foodmart\"";
+    sql(sql)
+        .returns("EXPR$0=86829\n")
+        .queryContains(druidChecker(druidQuery))
+        .explainContains(explain);
+  }
+
+  @Test public void testGroupByOneColumnNotProjected() {
+    final String sql = "select count(*) as c from \"foodmart\"\n"
+        + "group by \"state_province\" order by 1";
+    sql(sql)
+        .returnsOrdered("C=21610",
+            "C=24441",
+            "C=40778");
+  }
+
+  /** Unlike {@link #testGroupByTimeAndOneColumnNotProjected()}, we cannot use
+   * "topN" because we have a global limit, and that requires
+   * {@code granularity: all}. */
+  @Test public void testGroupByTimeAndOneColumnNotProjectedWithLimit() {
+    final String sql = "select count(*) as \"c\","
+        + " floor(\"timestamp\" to MONTH) as \"month\"\n"
+        + "from \"foodmart\"\n"
+        + "group by floor(\"timestamp\" to MONTH), \"state_province\"\n"
+        + "order by \"c\" desc limit 3";
+    sql(sql)
+        .returnsOrdered("c=4070; month=1997-12-01 00:00:00",
+            "c=4033; month=1997-11-01 00:00:00",
+            "c=3511; month=1997-07-01 00:00:00")
+        .queryContains(druidChecker("'queryType':'groupBy'"));
+  }
+
+  @Test public void testGroupByTimeAndOneMetricNotProjected() {
+    final String sql =
+        "select count(*) as \"c\", floor(\"timestamp\" to MONTH) as \"month\", floor"
+            + "(\"store_sales\") as sales\n"
+            + "from \"foodmart\"\n"
+            + "group by floor(\"timestamp\" to MONTH), \"state_province\", floor"
+            + "(\"store_sales\")\n"
+            + "order by \"c\" desc limit 3";
+    sql(sql).returnsOrdered("c=494; month=1997-11-01 00:00:00; SALES=5.0",
+        "c=475; month=1997-12-01 00:00:00; SALES=5.0",
+        "c=468; month=1997-03-01 00:00:00; SALES=5.0").queryContains(druidChecker("'queryType':'groupBy'"));
+  }
+
+  @Test public void testGroupByTimeAndOneColumnNotProjected() {
+    final String sql = "select count(*) as \"c\",\n"
+        + "  floor(\"timestamp\" to MONTH) as \"month\"\n"
+        + "from \"foodmart\"\n"
+        + "group by floor(\"timestamp\" to MONTH), \"state_province\"\n"
+        + "having count(*) > 3500";
+    sql(sql)
+        .returnsUnordered("c=3511; month=1997-07-01 00:00:00",
+            "c=4033; month=1997-11-01 00:00:00",
+            "c=4070; month=1997-12-01 00:00:00")
+        .queryContains(druidChecker("'queryType':'groupBy'"));
+  }
+
+  @Test public void testOrderByOneColumnNotProjected() {
+    // Result including state: CA=24441, OR=21610, WA=40778
+    final String sql = "select count(*) as c from \"foodmart\"\n"
+        + "group by \"state_province\" order by \"state_province\"";
+    sql(sql)
+        .returnsOrdered("C=24441",
+            "C=21610",
+            "C=40778");
+  }
+
+  @Test public void testGroupByOneColumn() {
+    final String sql = "select \"state_province\", count(*) as c\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\"\n"
+        + "order by \"state_province\"";
+    String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[$30]], groups=[{0}], "
+        + "aggs=[[COUNT()]], sort0=[0], dir0=[ASC])";
+    sql(sql)
+        .limit(2)
+        .returnsOrdered("state_province=CA; C=24441",
+            "state_province=OR; C=21610")
+        .explainContains(explain);
+  }
+
+  @Test public void testGroupByOneColumnReversed() {
+    final String sql = "select count(*) as c, \"state_province\"\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\"\n"
+        + "order by \"state_province\"";
+    sql(sql)
+        .limit(2)
+        .returnsOrdered("C=24441; state_province=CA",
+            "C=21610; state_province=OR");
+  }
+
+  @Test public void testGroupByAvgSumCount() {
+    final String sql = "select \"state_province\",\n"
+        + " avg(\"unit_sales\") as a,\n"
+        + " sum(\"unit_sales\") as s,\n"
+        + " count(\"store_sqft\") as c,\n"
+        + " count(*) as c0\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\"\n"
+        + "order by 1";
+    sql(sql)
+        .limit(2)
+        .returnsUnordered("state_province=CA; A=3; S=74748; C=16347; C0=24441",
+            "state_province=OR; A=3; S=67659; C=21610; C0=21610")
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  BindableProject(state_province=[$0], A=[CAST(/(CASE(=($2, 0), null, $1), $2)):BIGINT],"
+            + " S=[CASE(=($2, 0), null, $1)], C=[$3], C0=[$4])\n"
+            + "    DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+            + "2992-01-10T00:00:00.000Z]], projects=[[$30, $89, $71]], groups=[{0}], "
+            + "aggs=[[$SUM0($1), COUNT($1), COUNT($2), COUNT()]], sort0=[0], dir0=[ASC])")
+        .queryContains(
+            druidChecker("{'queryType':'groupBy','dataSource':'foodmart','granularity':'all'"
+                + ",'dimensions':[{'type':'default','dimension':'state_province','outputName':'state_province'"
+                + ",'outputType':'STRING'}],'limitSpec':"
+                + "{'type':'default','columns':[{'dimension':'state_province',"
+                + "'direction':'ascending','dimensionOrder':'lexicographic'}]},'aggregations':"
+                + "[{'type':'longSum','name':'$f1','fieldName':'unit_sales'},{'type':'filtered',"
+                + "'filter':{'type':'not','field':{'type':'selector','dimension':'unit_sales',"
+                + "'value':null}},'aggregator':{'type':'count','name':'$f2','fieldName':'unit_sales'}}"
+                + ",{'type':'filtered','filter':{'type':'not','field':{'type':'selector',"
+                + "'dimension':'store_sqft','value':null}},'aggregator':{'type':'count','name':'C',"
+                + "'fieldName':'store_sqft'}},{'type':'count','name':'C0'}],"
+                + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}"));
+  }
+
+  @Test public void testGroupByMonthGranularity() {
+    final String sql = "select sum(\"unit_sales\") as s,\n"
+        + " count(\"store_sqft\") as c\n"
+        + "from \"foodmart\"\n"
+        + "group by floor(\"timestamp\" to MONTH) order by s";
+    String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart'";
+    sql(sql)
+        .limit(3)
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  BindableProject(S=[$1], C=[$2])\n"
+            + "    DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+            + "2992-01-10T00:00:00.000Z]], projects=[[FLOOR($0, FLAG(MONTH)), $89, $71]], "
+            + "groups=[{0}], aggs=[[SUM($1), COUNT($2)]], sort0=[1], dir0=[ASC])")
+        .returnsOrdered("S=19958; C=5606", "S=20179; C=5523", "S=20388; C=5591")
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1577">[CALCITE-1577]
+   * Druid adapter: Incorrect result - limit on timestamp disappears</a>. */
+  @Test public void testGroupByMonthGranularitySort() {
+    final String sql = "select sum(\"unit_sales\") as s,\n"
+        + " count(\"store_sqft\") as c\n"
+        + "from \"foodmart\"\n"
+        + "group by floor(\"timestamp\" to MONTH)\n"
+        + "order by floor(\"timestamp\" to MONTH) ASC";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  BindableProject(S=[$1], C=[$2], EXPR$2=[$0])\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], projects=[[FLOOR($0, "
+        + "FLAG(MONTH)), $89, $71]], groups=[{0}], aggs=[[SUM($1), COUNT($2)]], sort0=[0], "
+        + "dir0=[ASC])";
+    sql(sql)
+        .explainContains(explain)
+        .returnsOrdered("S=21628; C=5957",
+            "S=20957; C=5842",
+            "S=23706; C=6528",
+            "S=20179; C=5523",
+            "S=21081; C=5793",
+            "S=21350; C=5863",
+            "S=23763; C=6762",
+            "S=21697; C=5915",
+            "S=20388; C=5591",
+            "S=19958; C=5606",
+            "S=25270; C=7026",
+            "S=26796; C=7338");
+  }
+
+  @Test public void testGroupByMonthGranularitySortLimit() {
+    final String sql = "select floor(\"timestamp\" to MONTH) as m,\n"
+        + " sum(\"unit_sales\") as s,\n"
+        + " count(\"store_sqft\") as c\n"
+        + "from \"foodmart\"\n"
+        + "group by floor(\"timestamp\" to MONTH)\n"
+        + "order by floor(\"timestamp\" to MONTH) limit 3";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[FLOOR($0, FLAG(MONTH)), $89, $71]], groups=[{0}], "
+        + "aggs=[[SUM($1), COUNT($2)]], sort0=[0], dir0=[ASC], fetch=[3])";
+    sql(sql)
+        .returnsOrdered("M=1997-01-01 00:00:00; S=21628; C=5957",
+            "M=1997-02-01 00:00:00; S=20957; C=5842",
+            "M=1997-03-01 00:00:00; S=23706; C=6528")
+        .explainContains(explain);
+  }
+
+  @Test public void testGroupByDayGranularity() {
+    final String sql = "select sum(\"unit_sales\") as s,\n"
+        + " count(\"store_sqft\") as c\n"
+        + "from \"foodmart\"\n"
+        + "group by floor(\"timestamp\" to DAY) order by c desc";
+    String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart'";
+    sql(sql)
+        .limit(3)
+        .queryContains(druidChecker(druidQuery))
+        .returnsOrdered("S=3850; C=1230", "S=3342; C=1071", "S=3219; C=1024");
+  }
+
+  @Test public void testGroupByMonthGranularityFiltered() {
+    final String sql = "select sum(\"unit_sales\") as s,\n"
+        + " count(\"store_sqft\") as c\n"
+        + "from \"foodmart\"\n"
+        + "where \"timestamp\" >= '1996-01-01 00:00:00' and "
+        + " \"timestamp\" < '1998-01-01 00:00:00'\n"
+        + "group by floor(\"timestamp\" to MONTH) order by s asc";
+    String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart'";
+
+    sql(sql)
+        .limit(3)
+        .returnsOrdered("S=19958; C=5606", "S=20179; C=5523", "S=20388; C=5591")
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testTopNMonthGranularity() {
+    final String sql = "select sum(\"unit_sales\") as s,\n"
+        + "max(\"unit_sales\") as m,\n"
+        + "\"state_province\" as p\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\", floor(\"timestamp\" to MONTH)\n"
+        + "order by s desc limit 3";
+    // Cannot use a Druid "topN" query, granularity != "all";
+    // have to use "groupBy" query followed by external Sort and fetch.
+    final String explain = "PLAN="
+        + "EnumerableCalc(expr#0..3=[{inputs}], S=[$t2], M=[$t3], P=[$t0])\n"
+        + "  EnumerableInterpreter\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], projects=[[$30, FLOOR"
+        + "($0, FLAG(MONTH)), $89]], groups=[{0, 1}], aggs=[[SUM($2), MAX($2)]], sort0=[2], "
+        + "dir0=[DESC], fetch=[3])";
+    final String druidQueryPart1 = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'default',"
+        + "'dimension':'state_province',\"outputName\":\"state_province\",\"outputType\":\"STRING\"},"
+        + "{'type':'extraction','dimension':'__time',"
+        + "'outputName':'floor_month','extractionFn':{'type':'timeFormat','format'";
+    final String druidQueryPart2 = "'limitSpec':{'type':'default','limit':3,"
+        + "'columns':[{'dimension':'S','direction':'descending',"
+        + "'dimensionOrder':'numeric'}]},'aggregations':[{'type':'longSum',"
+        + "'name':'S','fieldName':'unit_sales'},{'type':'longMax','name':'M',"
+        + "'fieldName':'unit_sales'}],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sql)
+        .returnsUnordered("S=12399; M=6; P=WA",
+            "S=12297; M=7; P=WA",
+            "S=10640; M=6; P=WA")
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQueryPart1, druidQueryPart2));
+  }
+
+  @Test public void testTopNDayGranularityFiltered() {
+    final String sql = "select sum(\"unit_sales\") as s,\n"
+        + "max(\"unit_sales\") as m,\n"
+        + "\"state_province\" as p\n"
+        + "from \"foodmart\"\n"
+        + "where \"timestamp\" >= '1997-01-01 00:00:00' and "
+        + " \"timestamp\" < '1997-09-01 00:00:00'\n"
+        + "group by \"state_province\", floor(\"timestamp\" to DAY)\n"
+        + "order by s desc limit 6";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  BindableProject(S=[$2], M=[$3], P=[$0])\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1997-01-01T00:00:00.000Z/1997-09-01T00:00:00.000Z]], projects=[[$30, FLOOR"
+        + "($0, FLAG(DAY)), $89]], groups=[{0, 1}], aggs=[[SUM($2), MAX($2)]], sort0=[2], "
+        + "dir0=[DESC], fetch=[6])";
+    final String druidQueryType = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions'";
+    final String limitSpec = "'limitSpec':{'type':'default','limit':6,"
+        + "'columns':[{'dimension':'S','direction':'descending','dimensionOrder':'numeric'}]}";
+    sql(sql)
+        .returnsOrdered("S=2527; M=5; P=OR",
+            "S=2525; M=6; P=OR",
+            "S=2238; M=6; P=OR",
+            "S=1715; M=5; P=OR",
+            "S=1691; M=5; P=OR",
+            "S=1629; M=5; P=WA")
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQueryType, limitSpec));
+  }
+
+  @Test public void testGroupByHaving() {
+    final String sql = "select \"state_province\" as s, count(*) as c\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\" having count(*) > 23000 order by 1";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]], projects=[[$30]], groups=[{0}], aggs=[[COUNT()]], "
+        + "filter=[>($1, 23000)], sort0=[0], dir0=[ASC])";
+    sql(sql)
+        .returnsOrdered("S=CA; C=24441",
+            "S=WA; C=40778")
+        .explainContains(explain);
+  }
+
+  @Test public void testGroupComposite() {
+    // Note: We don't push down SORT-LIMIT yet
+    final String sql = "select count(*) as c, \"state_province\", \"city\"\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\", \"city\"\n"
+        + "order by c desc limit 2";
+    final String explain = "BindableProject(C=[$2], state_province=[$0], city=[$1])\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], projects=[[$30, $29]], groups=[{0, 1}], aggs=[[COUNT()]], sort0=[2], dir0=[DESC], fetch=[2])";
+    sql(sql)
+        .returnsOrdered("C=7394; state_province=WA; city=Spokane",
+            "C=3958; state_province=WA; city=Olympia")
+        .explainContains(explain);
+  }
+
+  /** Tests that distinct-count is pushed down to Druid and evaluated using
+   * "cardinality". The result is approximate, but gives the correct result in
+   * this example when rounded down using FLOOR. */
+  @Test public void testDistinctCount() {
+    final String sql = "select \"state_province\",\n"
+        + " floor(count(distinct \"city\")) as cdc\n"
+        + "from \"foodmart\"\n"
+        + "group by \"state_province\"\n"
+        + "order by 2 desc limit 2";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  BindableSort(sort0=[$1], dir0=[DESC], fetch=[2])\n"
+        + "    BindableProject(state_province=[$0], CDC=[FLOOR($1)])\n"
+        + "      BindableAggregate(group=[{1}], agg#0=[COUNT($0)])\n"
+        + "        DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], groups=[{29, 30}], "
+        + "aggs=[[]])";
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'default','dimension':'city','outputName':'city'"
+        + ",'outputType':'STRING'},"
+        + "{'type':'default','dimension':'state_province','outputName':'state_province','outputType':'STRING'}],"
+        + "'limitSpec':{'type':'default'},'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sql)
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQuery))
+        .returnsUnordered("state_province=CA; CDC=45",
+            "state_province=WA; CDC=22");
+  }
+
+  /** Tests that projections of columns are pushed into the DruidQuery, and
+   * projections of expressions that Druid cannot handle (in this case, a
+   * literal 0) stay up. */
+  @Test public void testProject() {
+    final String sql = "select \"product_name\", 0 as zero\n"
+        + "from \"foodmart\"\n"
+        + "order by \"product_name\"";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  BindableSort(sort0=[$0], dir0=[ASC])\n"
+        + "    DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], projects=[[$3, 0]])";
+    sql(sql)
+        .limit(2)
+        .returnsUnordered("product_name=ADJ Rosy Sunglasses; ZERO=0",
+            "product_name=ADJ Rosy Sunglasses; ZERO=0")
+        .explainContains(explain);
+  }
+
+  @Test public void testFilterDistinct() {
+    final String sql = "select distinct \"state_province\", \"city\",\n"
+        + "  \"product_name\"\n"
+        + "from \"foodmart\"\n"
+        + "where \"product_name\" = 'High Top Dried Mushrooms'\n"
+        + "and \"quarter\" in ('Q2', 'Q3')\n"
+        + "and \"state_province\" = 'WA'";
+    final String druidQuery1 = "{'queryType':'groupBy','dataSource':'foodmart','granularity':'all'";
+    final String druidQuery2 = "'filter':{'type':'and','fields':[{'type':'selector','dimension':"
+        + "'product_name','value':'High Top Dried Mushrooms'},{'type':'or','fields':[{'type':'selector',"
+        + "'dimension':'quarter','value':'Q2'},{'type':'selector','dimension':'quarter',"
+        + "'value':'Q3'}]},{'type':'selector','dimension':'state_province','value':'WA'}]},"
+        + "'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+        + "2992-01-10T00:00:00.000Z]],"
+        + " filter=[AND(=($3, 'High Top Dried Mushrooms'),"
+        + " OR(=($87, 'Q2'),"
+        + " =($87, 'Q3')),"
+        + " =($30, 'WA'))],"
+        + " projects=[[$30, $29, $3]], groups=[{0, 1, 2}], aggs=[[]])\n";
+    sql(sql)
+        .queryContains(druidChecker(druidQuery1, druidQuery2))
+        .explainContains(explain)
+        .returnsUnordered(
+            "state_province=WA; city=Bremerton; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Everett; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Kirkland; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Lynnwood; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Olympia; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Port Orchard; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Puyallup; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Spokane; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Tacoma; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Yakima; product_name=High Top Dried Mushrooms");
+  }
+
+  @Test public void testFilter() {
+    final String sql = "select \"state_province\", \"city\",\n"
+        + "  \"product_name\"\n"
+        + "from \"foodmart\"\n"
+        + "where \"product_name\" = 'High Top Dried Mushrooms'\n"
+        + "and \"quarter\" in ('Q2', 'Q3')\n"
+        + "and \"state_province\" = 'WA'";
+    final String druidQuery = "{'queryType':'scan',"
+        + "'dataSource':'foodmart',"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z'],"
+        + "'filter':{'type':'and','fields':["
+        + "{'type':'selector','dimension':'product_name','value':'High Top Dried Mushrooms'},"
+        + "{'type':'or','fields':["
+        + "{'type':'selector','dimension':'quarter','value':'Q2'},"
+        + "{'type':'selector','dimension':'quarter','value':'Q3'}]},"
+        + "{'type':'selector','dimension':'state_province','value':'WA'}]},"
+        + "'columns':['state_province','city','product_name'],"
+        + "'resultFormat':'compactedList'}";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], "
+        + "filter=[AND(=($3, 'High Top Dried Mushrooms'), "
+        + "OR(=($87, 'Q2'), =($87, 'Q3')), =($30, 'WA'))], "
+        + "projects=[[$30, $29, $3]])\n";
+    sql(sql)
+        .queryContains(druidChecker(druidQuery))
+        .explainContains(explain)
+        .returnsUnordered(
+            "state_province=WA; city=Bremerton; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Everett; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Kirkland; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Lynnwood; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Olympia; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Port Orchard; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Puyallup; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Puyallup; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Spokane; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Spokane; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Spokane; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Tacoma; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Yakima; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Yakima; product_name=High Top Dried Mushrooms",
+            "state_province=WA; city=Yakima; product_name=High Top Dried Mushrooms");
+  }
+
+  /** Tests that conditions applied to time units extracted via the EXTRACT
+   * function become ranges on the timestamp column
+   *
+   * <p>Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1334">[CALCITE-1334]
+   * Convert predicates on EXTRACT function calls into date ranges</a>. */
+  @Test public void testFilterTimestamp() {
+    String sql = "select count(*) as c\n"
+        + "from \"foodmart\"\n"
+        + "where extract(year from \"timestamp\") = 1997\n"
+        + "and extract(month from \"timestamp\") in (4, 6)\n";
+    final String explain = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1997-04-01T00:00:00.000Z/"
+        + "1997-05-01T00:00:00.000Z, 1997-06-01T00:00:00.000Z/1997-07-01T00:00:00.000Z]],"
+        + " projects=[[0]], groups=[{}], aggs=[[COUNT()]])";
+    sql(sql)
+        .returnsUnordered("C=13500")
+        .explainContains(explain);
+  }
+
+  @Test public void testFilterSwapped() {
+    String sql = "select \"state_province\"\n"
+        + "from \"foodmart\"\n"
+        + "where 'High Top Dried Mushrooms' = \"product_name\"";
+    final String explain = "EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], filter=[=('High Top Dried Mushrooms', $3)], projects=[[$30]])";
+    final String druidQuery = "'filter':{'type':'selector','dimension':'product_name',"
+        + "'value':'High Top Dried Mushrooms'}";
+    sql(sql)
+        .explainContains(explain)
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testGroupByMetricAndExtractTime() {
+    final String sql =
+        "SELECT count(*), floor(\"timestamp\" to DAY), \"store_sales\" "
+            + "FROM \"foodmart\"\n"
+            + "GROUP BY \"store_sales\", floor(\"timestamp\" to DAY)\n ORDER BY \"store_sales\" DESC\n"
+            + "LIMIT 10\n";
+    sql(sql).queryContains(druidChecker("{\"queryType\":\"groupBy\""));
+  }
+
+  @Test public void testFilterOnDouble() {
+    String sql = "select \"product_id\" from \"foodmart\"\n"
+        + "where cast(\"product_id\" as double) < 0.41024 and \"product_id\" < 12223";
+    sql(sql).queryContains(
+        druidChecker("'type':'bound','dimension':'product_id','upper':'0.41024'",
+            "'upper':'12223'"));
+  }
+
+  @Test public void testPushAggregateOnTime() {
+    String sql = "select \"product_id\", \"timestamp\" as \"time\" "
+        + "from \"foodmart\" "
+        + "where \"product_id\" = 1016 "
+        + "and \"timestamp\" < '1997-01-03 00:00:00' "
+        + "and \"timestamp\" > '1990-01-01 00:00:00' "
+        + "group by \"timestamp\", \"product_id\" ";
+    String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'extraction',"
+        + "'dimension':'__time','outputName':'extract',"
+        + "'extractionFn':{'type':'timeFormat','format':'yyyy-MM-dd";
+    sql(sql)
+        .returnsUnordered("product_id=1016; time=1997-01-02 00:00:00")
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testPushAggregateOnTimeWithExtractYear() {
+    String sql = "select EXTRACT( year from \"timestamp\") as \"year\",\"product_id\" from "
+        + "\"foodmart\" where \"product_id\" = 1016 and "
+        + "\"timestamp\" < cast('1999-01-02' as timestamp) and \"timestamp\" > cast"
+        + "('1997-01-01' as timestamp)" + " group by "
+        + " EXTRACT( year from \"timestamp\"), \"product_id\" ";
+    sql(sql)
+        .queryContains(
+            druidChecker(
+                ",'granularity':'all'",
+                "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_year',"
+                    + "'extractionFn':{'type':'timeFormat','format':'yyyy',"
+                    + "'timeZone':'UTC','locale':'en-US'}}"))
+        .returnsUnordered("year=1997; product_id=1016");
+  }
+
+  @Test public void testPushAggregateOnTimeWithExtractMonth() {
+    String sql = "select EXTRACT( month from \"timestamp\") as \"month\",\"product_id\" from "
+        + "\"foodmart\" where \"product_id\" = 1016 and "
+        + "\"timestamp\" < cast('1997-06-02' as timestamp) and \"timestamp\" > cast"
+        + "('1997-01-01' as timestamp)" + " group by "
+        + " EXTRACT( month from \"timestamp\"), \"product_id\" ";
+    sql(sql)
+        .queryContains(
+            druidChecker(
+                ",'granularity':'all'",
+                "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_month',"
+                    + "'extractionFn':{'type':'timeFormat','format':'M',"
+                    + "'timeZone':'UTC','locale':'en-US'}}"))
+        .returnsUnordered("month=1; product_id=1016", "month=2; product_id=1016",
+            "month=3; product_id=1016", "month=4; product_id=1016", "month=5; product_id=1016");
+  }
+
+  @Test public void testPushAggregateOnTimeWithExtractDay() {
+    String sql = "select EXTRACT( day from \"timestamp\") as \"day\","
+        + "\"product_id\" from \"foodmart\""
+        + " where \"product_id\" = 1016 and "
+        + "\"timestamp\" < cast('1997-01-20' as timestamp) and \"timestamp\" > cast"
+        + "('1997-01-01' as timestamp)" + " group by "
+        + " EXTRACT( day from \"timestamp\"), \"product_id\" ";
+    sql(sql)
+        .queryContains(
+            druidChecker(
+                ",'granularity':'all'",
+                "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_day',"
+                    + "'extractionFn':{'type':'timeFormat','format':'d',"
+                    + "'timeZone':'UTC','locale':'en-US'}}"))
+        .returnsUnordered("day=2; product_id=1016", "day=10; product_id=1016",
+            "day=13; product_id=1016", "day=16; product_id=1016");
+  }
+
+  @Test
+  public void testPushAggregateOnTimeWithExtractHourOfDay() {
+    String sql =
+        "select EXTRACT( hour from \"timestamp\") as \"hourOfDay\",\"product_id\"  from "
+            + "\"foodmart\" where \"product_id\" = 1016 and "
+            + "\"timestamp\" < cast('1997-06-02' as timestamp) and \"timestamp\" > cast"
+            + "('1997-01-01' as timestamp)" + " group by "
+            + " EXTRACT( hour from \"timestamp\"), \"product_id\" ";
+    sql(sql)
+        .queryContains(druidChecker("'queryType':'groupBy'"))
+        .returnsUnordered("hourOfDay=0; product_id=1016");
+  }
+
+  @Test public void testPushAggregateOnTimeWithExtractYearMonthDay() {
+    String sql = "select EXTRACT( day from \"timestamp\") as \"day\", EXTRACT( month from "
+        + "\"timestamp\") as \"month\",  EXTRACT( year from \"timestamp\") as \"year\",\""
+        + "product_id\"  from \"foodmart\" where \"product_id\" = 1016 and "
+        + "\"timestamp\" < cast('1997-01-20' as timestamp) and \"timestamp\" > cast"
+        + "('1997-01-01' as timestamp)"
+        + " group by "
+        + " EXTRACT( day from \"timestamp\"), EXTRACT( month from \"timestamp\"),"
+        + " EXTRACT( year from \"timestamp\"), \"product_id\" ";
+    sql(sql)
+        .queryContains(
+            druidChecker(
+                ",'granularity':'all'",
+                "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_day',"
+                    + "'extractionFn':{'type':'timeFormat','format':'d',"
+                    + "'timeZone':'UTC','locale':'en-US'}}", "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_month',"
+                    + "'extractionFn':{'type':'timeFormat','format':'M',"
+                    + "'timeZone':'UTC','locale':'en-US'}}", "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_year',"
+                    + "'extractionFn':{'type':'timeFormat','format':'yyyy',"
+                    + "'timeZone':'UTC','locale':'en-US'}}"))
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], "
+            + "intervals=[[1997-01-01T00:00:00.001Z/1997-01-20T00:00:00.000Z]], "
+            + "filter=[=($1, 1016)], projects=[[EXTRACT(FLAG(DAY), $0), EXTRACT(FLAG(MONTH), $0), "
+            + "EXTRACT(FLAG(YEAR), $0), $1]], groups=[{0, 1, 2, 3}], aggs=[[]])\n")
+        .returnsUnordered("day=2; month=1; year=1997; product_id=1016",
+            "day=10; month=1; year=1997; product_id=1016",
+            "day=13; month=1; year=1997; product_id=1016",
+            "day=16; month=1; year=1997; product_id=1016");
+  }
+
+  @Test public void testPushAggregateOnTimeWithExtractYearMonthDayWithOutRenaming() {
+    String sql = "select EXTRACT( day from \"timestamp\"), EXTRACT( month from "
+        + "\"timestamp\"), EXTRACT( year from \"timestamp\"),\""
+        + "product_id\"  from \"foodmart\" where \"product_id\" = 1016 and "
+        + "\"timestamp\" < cast('1997-01-20' as timestamp) and \"timestamp\" > cast"
+        + "('1997-01-01' as timestamp)"
+        + " group by "
+        + " EXTRACT( day from \"timestamp\"), EXTRACT( month from \"timestamp\"),"
+        + " EXTRACT( year from \"timestamp\"), \"product_id\" ";
+    sql(sql)
+        .queryContains(
+            druidChecker(
+                ",'granularity':'all'", "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_day',"
+                    + "'extractionFn':{'type':'timeFormat','format':'d',"
+                    + "'timeZone':'UTC','locale':'en-US'}}", "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_month',"
+                    + "'extractionFn':{'type':'timeFormat','format':'M',"
+                    + "'timeZone':'UTC','locale':'en-US'}}", "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_year',"
+                    + "'extractionFn':{'type':'timeFormat','format':'yyyy',"
+                    + "'timeZone':'UTC','locale':'en-US'}}"))
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], "
+            + "intervals=[[1997-01-01T00:00:00.001Z/1997-01-20T00:00:00.000Z]], "
+            + "filter=[=($1, 1016)], projects=[[EXTRACT(FLAG(DAY), $0), EXTRACT(FLAG(MONTH), $0), "
+            + "EXTRACT(FLAG(YEAR), $0), $1]], groups=[{0, 1, 2, 3}], aggs=[[]])\n")
+        .returnsUnordered("EXPR$0=2; EXPR$1=1; EXPR$2=1997; product_id=1016",
+            "EXPR$0=10; EXPR$1=1; EXPR$2=1997; product_id=1016",
+            "EXPR$0=13; EXPR$1=1; EXPR$2=1997; product_id=1016",
+            "EXPR$0=16; EXPR$1=1; EXPR$2=1997; product_id=1016");
+  }
+
+  @Test public void testPushAggregateOnTimeWithExtractWithOutRenaming() {
+    String sql = "select EXTRACT( day from \"timestamp\"), "
+        + "\"product_id\" as \"dayOfMonth\" from \"foodmart\" "
+        + "where \"product_id\" = 1016 and \"timestamp\" < cast('1997-01-20' as timestamp) "
+        + "and \"timestamp\" > cast('1997-01-01' as timestamp)"
+        + " group by "
+        + " EXTRACT( day from \"timestamp\"), EXTRACT( day from \"timestamp\"),"
+        + " \"product_id\" ";
+    sql(sql)
+        .queryContains(
+            druidChecker(
+                ",'granularity':'all'", "{'type':'extraction',"
+                    + "'dimension':'__time','outputName':'extract_day',"
+                    + "'extractionFn':{'type':'timeFormat','format':'d',"
+                    + "'timeZone':'UTC','locale':'en-US'}}"))
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], "
+            + "intervals=[[1997-01-01T00:00:00.001Z/1997-01-20T00:00:00.000Z]], "
+            + "filter=[=($1, 1016)], projects=[[EXTRACT(FLAG(DAY), $0), $1]], "
+            + "groups=[{0, 1}], aggs=[[]])\n")
+        .returnsUnordered("EXPR$0=2; dayOfMonth=1016", "EXPR$0=10; dayOfMonth=1016",
+            "EXPR$0=13; dayOfMonth=1016", "EXPR$0=16; dayOfMonth=1016");
+  }
+
+  @Test public void testPushComplexFilter() {
+    String sql = "select sum(\"store_sales\") from \"foodmart\" "
+        + "where EXTRACT( year from \"timestamp\") = 1997 and "
+        + "\"cases_per_pallet\" >= 8 and \"cases_per_pallet\" <= 10 and "
+        + "\"units_per_case\" < 15 ";
+    String druidQuery = "{'queryType':'timeseries','dataSource':'foodmart','descending':false,"
+        + "'granularity':'all','filter':{'type':'and','fields':[{'type':'bound','dimension':"
+        + "'cases_per_pallet','lower':'8','lowerStrict':false,'ordering':'numeric'},"
+        + "{'type':'bound','dimension':'cases_per_pallet','upper':'10','upperStrict':false,"
+        + "'ordering':'numeric'},{'type':'bound','dimension':'units_per_case','upper':'15',"
+        + "'upperStrict':true,'ordering':'numeric'}]},'aggregations':[{'type':'doubleSum',"
+        + "'name':'EXPR$0','fieldName':'store_sales'}],'intervals':['1997-01-01T00:00:00.000Z/"
+        + "1998-01-01T00:00:00.000Z'],'context':{'skipEmptyBuckets':true}}";
+    sql(sql)
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], "
+            + "intervals=[[1997-01-01T00:00:00.000Z/1998-01-01T00:00:00.000Z]], "
+            + "filter=[AND(>=(CAST($11):BIGINT, 8), <=(CAST($11):BIGINT, 10), "
+            + "<(CAST($10):BIGINT, 15))], groups=[{}], "
+            + "aggs=[[SUM($90)]])\n")
+        .returnsUnordered("EXPR$0=75364.1")
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testPushOfFilterExtractionOnDayAndMonth() {
+    String sql = "SELECT \"product_id\" , EXTRACT(day from \"timestamp\"), EXTRACT(month from "
+        + "\"timestamp\") from \"foodmart\" WHERE  EXTRACT(day from \"timestamp\") >= 30 AND "
+        + "EXTRACT(month from \"timestamp\") = 11 "
+        + "AND  \"product_id\" >= 1549 group by \"product_id\", EXTRACT(day from "
+        + "\"timestamp\"), EXTRACT(month from \"timestamp\")";
+    sql(sql)
+        .returnsUnordered("product_id=1549; EXPR$1=30; EXPR$2=11",
+            "product_id=1553; EXPR$1=30; EXPR$2=11");
+  }
+
+  @Test public void testPushOfFilterExtractionOnDayAndMonthAndYear() {
+    String sql = "SELECT \"product_id\" , EXTRACT(day from \"timestamp\"), EXTRACT(month from "
+        + "\"timestamp\") , EXTRACT(year from \"timestamp\") from \"foodmart\" "
+        + "WHERE  EXTRACT(day from \"timestamp\") >= 30 AND EXTRACT(month from \"timestamp\") = 11 "
+        + "AND  \"product_id\" >= 1549 AND EXTRACT(year from \"timestamp\") = 1997"
+        + "group by \"product_id\", EXTRACT(day from \"timestamp\"), "
+        + "EXTRACT(month from \"timestamp\"), EXTRACT(year from \"timestamp\")";
+    sql(sql)
+        .returnsUnordered("product_id=1549; EXPR$1=30; EXPR$2=11; EXPR$3=1997",
+            "product_id=1553; EXPR$1=30; EXPR$2=11; EXPR$3=1997")
+        .queryContains(
+            druidChecker("{'queryType':'groupBy','dataSource':'foodmart','granularity':'all'"));
+  }
+
+  @Test public void testFilterExtractionOnMonthWithBetween() {
+    String sqlQuery = "SELECT \"product_id\", EXTRACT(month from \"timestamp\") FROM \"foodmart\""
+        + " WHERE EXTRACT(month from \"timestamp\") BETWEEN 10 AND 11 AND  \"product_id\" >= 1558"
+        + " GROUP BY \"product_id\", EXTRACT(month from \"timestamp\")";
+    String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart'";
+    sql(sqlQuery)
+        .returnsUnordered("product_id=1558; EXPR$1=10", "product_id=1558; EXPR$1=11",
+            "product_id=1559; EXPR$1=11")
+        .queryContains(druidChecker(druidQuery));
+  }
+
+  @Test public void testFilterExtractionOnMonthWithIn() {
+    String sqlQuery = "SELECT \"product_id\", EXTRACT(month from \"timestamp\") FROM \"foodmart\""
+        + " WHERE EXTRACT(month from \"timestamp\") IN (10, 11) AND  \"product_id\" >= 1558"
+        + " GROUP BY \"product_id\", EXTRACT(month from \"timestamp\")";
+    sql(sqlQuery)
+        .returnsUnordered("product_id=1558; EXPR$1=10", "product_id=1558; EXPR$1=11",
+            "product_id=1559; EXPR$1=11")
+        .queryContains(
+            druidChecker("{'queryType':'groupBy',"
+                + "'dataSource':'foodmart','granularity':'all',"
+                + "'dimensions':[{'type':'default','dimension':'product_id','outputName':'product_id','outputType':'STRING'},"
+                + "{'type':'extraction','dimension':'__time','outputName':'extract_month',"
+                + "'extractionFn':{'type':'timeFormat','format':'M','timeZone':'UTC',"
+                + "'locale':'en-US'}}],'limitSpec':{'type':'default'},"
+                + "'filter':{'type':'and','fields':[{'type':'bound',"
+                + "'dimension':'product_id','lower':'1558','lowerStrict':false,"
+                + "'ordering':'numeric'},{'type':'or','fields':[{'type':'bound','dimension':'__time'"
+                + ",'lower':'10','lowerStrict':false,'upper':'10','upperStrict':false,"
+                + "'ordering':'numeric','extractionFn':{'type':'timeFormat',"
+                + "'format':'M','timeZone':'UTC','locale':'en-US'}},{'type':'bound',"
+                + "'dimension':'__time','lower':'11','lowerStrict':false,'upper':'11',"
+                + "'upperStrict':false,'ordering':'numeric','extractionFn':{'type':'timeFormat',"
+                + "'format':'M','timeZone':'UTC','locale':'en-US'}}]}]},"
+                + "'aggregations':[],"
+                + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}"));
+  }
+
+  @Test public void testPushOfOrderByWithMonthExtract() {
+    String sqlQuery = "SELECT  extract(month from \"timestamp\") as m , \"product_id\", SUM"
+        + "(\"unit_sales\") as s FROM \"foodmart\""
+        + " WHERE \"product_id\" >= 1558"
+        + " GROUP BY extract(month from \"timestamp\"), \"product_id\" order by m, s, "
+        + "\"product_id\"";
+    sql(sqlQuery).queryContains(
+        druidChecker("{'queryType':'groupBy','dataSource':'foodmart',"
+            + "'granularity':'all','dimensions':[{'type':'extraction',"
+            + "'dimension':'__time','outputName':'extract_month',"
+            + "'extractionFn':{'type':'timeFormat','format':'M','timeZone':'UTC',"
+            + "'locale':'en-US'}},{'type':'default','dimension':'product_id','outputName':"
+            + "'product_id','outputType':'STRING'}],"
+            + "'limitSpec':{'type':'default','columns':[{'dimension':'extract_month',"
+            + "'direction':'ascending','dimensionOrder':'numeric'},{'dimension':'S',"
+            + "'direction':'ascending','dimensionOrder':'numeric'},"
+            + "{'dimension':'product_id','direction':'ascending',"
+            + "'dimensionOrder':'lexicographic'}]},'filter':{'type':'bound',"
+            + "'dimension':'product_id','lower':'1558','lowerStrict':false,"
+            + "'ordering':'numeric'},'aggregations':[{'type':'longSum','name':'S',"
+            + "'fieldName':'unit_sales'}],"
+            + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}"))
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], "
+            + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], "
+            + "filter=[>=(CAST($1):BIGINT, 1558)], projects=[[EXTRACT(FLAG(MONTH), $0), $1, $89]], "
+            + "groups=[{0, 1}], aggs=[[SUM($2)]], sort0=[0], sort1=[2], sort2=[1], "
+            + "dir0=[ASC], dir1=[ASC], dir2=[ASC])");
+  }
+
+
+  @Test public void testGroupByFloorTimeWithoutLimit() {
+    final String sql = "select floor(\"timestamp\" to MONTH) as \"month\"\n"
+        + "from \"foodmart\"\n"
+        + "group by floor(\"timestamp\" to MONTH)\n"
+        + "order by \"month\" DESC";
+    sql(sql)
+        .queryContains(druidChecker("'queryType':'timeseries'", "'descending':true"))
+        .explainContains("PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z"
+            + "/2992-01-10T00:00:00.000Z]], projects=[[FLOOR($0, FLAG(MONTH))]], groups=[{0}], "
+            + "aggs=[[]], sort0=[0], dir0=[DESC])");
+
+  }
+
+  @Test public void testGroupByFloorTimeWithLimit() {
+    final String sql =
+        "select floor(\"timestamp\" to MONTH) as \"floorOfMonth\"\n"
+            + "from \"foodmart\"\n"
+            + "group by floor(\"timestamp\" to MONTH)\n"
+            + "order by \"floorOfMonth\" DESC LIMIT 3";
+    final String explain =
+        "PLAN=EnumerableInterpreter\n"
+            + "  DruidQuery(table=[[foodmart, foodmart]], intervals=[[1900-01-09T00:00:00.000Z/"
+            + "2992-01-10T00:00:00.000Z]], projects=[[FLOOR($0, FLAG(MONTH))]], groups=[{0}], "
+            + "aggs=[[]], sort0=[0], dir0=[DESC], fetch=[3])";
+    sql(sql)
+        .explainContains(explain)
+        .returnsOrdered("floorOfMonth=1997-12-01 00:00:00", "floorOfMonth=1997-11-01 00:00:00",
+            "floorOfMonth=1997-10-01 00:00:00")
+        .queryContains(druidChecker("'queryType':'groupBy'", "'direction':'descending'"));
+  }
+
+  @Test public void testPushofOrderByYearWithYearMonthExtract() {
+    String sqlQuery = "SELECT year(\"timestamp\") as y, extract(month from \"timestamp\") as m , "
+        + "\"product_id\", SUM"
+        + "(\"unit_sales\") as s FROM \"foodmart\""
+        + " WHERE \"product_id\" >= 1558"
+        + " GROUP BY year(\"timestamp\"), extract(month from \"timestamp\"), \"product_id\" order"
+        + " by y DESC, m ASC, s DESC, \"product_id\" LIMIT 3";
+    final String expectedPlan = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], "
+        + "filter=[>=(CAST($1):BIGINT, 1558)], projects=[[EXTRACT(FLAG(YEAR), $0), "
+        + "EXTRACT(FLAG(MONTH), $0), $1, $89]], groups=[{0, 1, 2}], aggs=[[SUM($3)]], sort0=[0], "
+        + "sort1=[1], sort2=[3], sort3=[2], dir0=[DESC], "
+        + "dir1=[ASC], dir2=[DESC], dir3=[ASC], fetch=[3])";
+    final String expectedDruidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'extraction',"
+        + "'dimension':'__time','outputName':'extract_year',"
+        + "'extractionFn':{'type':'timeFormat','format':'yyyy','timeZone':'UTC',"
+        + "'locale':'en-US'}},{'type':'extraction','dimension':'__time',"
+        + "'outputName':'extract_month','extractionFn':{'type':'timeFormat',"
+        + "'format':'M','timeZone':'UTC','locale':'en-US'}},{'type':'default',"
+        + "'dimension':'product_id','outputName':'product_id','outputType':'STRING'}],"
+        + "'limitSpec':{'type':'default','limit':3,"
+        + "'columns':[{'dimension':'extract_year','direction':'descending',"
+        + "'dimensionOrder':'numeric'},{'dimension':'extract_month',"
+        + "'direction':'ascending','dimensionOrder':'numeric'},{'dimension':'S',"
+        + "'direction':'descending','dimensionOrder':'numeric'},"
+        + "{'dimension':'product_id','direction':'ascending',"
+        + "'dimensionOrder':'lexicographic'}]},'filter':{'type':'bound',"
+        + "'dimension':'product_id','lower':'1558','lowerStrict':false,"
+        + "'ordering':'numeric'},'aggregations':[{'type':'longSum','name':'S',"
+        + "'fieldName':'unit_sales'}],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sqlQuery).explainContains(expectedPlan).queryContains(druidChecker(expectedDruidQuery))
+        .returnsOrdered("Y=1997; M=1; product_id=1558; S=6", "Y=1997; M=1; product_id=1559; S=6",
+            "Y=1997; M=2; product_id=1558; S=24");
+  }
+
+  @Test public void testPushofOrderByMetricWithYearMonthExtract() {
+    String sqlQuery = "SELECT year(\"timestamp\") as y, extract(month from \"timestamp\") as m , "
+        + "\"product_id\", SUM(\"unit_sales\") as s FROM \"foodmart\""
+        + " WHERE \"product_id\" >= 1558"
+        + " GROUP BY year(\"timestamp\"), extract(month from \"timestamp\"), \"product_id\" order"
+        + " by s DESC, m DESC, \"product_id\" LIMIT 3";
+    final String expectedPlan = "PLAN=EnumerableInterpreter\n"
+        + "  DruidQuery(table=[[foodmart, foodmart]], "
+        + "intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]], "
+        + "filter=[>=(CAST($1):BIGINT, 1558)], projects=[[EXTRACT(FLAG(YEAR), $0), "
+        + "EXTRACT(FLAG(MONTH), $0), $1, $89]], groups=[{0, 1, 2}], aggs=[[SUM($3)]], "
+        + "sort0=[3], sort1=[1], sort2=[2], dir0=[DESC], dir1=[DESC], dir2=[ASC], fetch=[3])";
+    final String expectedDruidQueryType = "'queryType':'groupBy'";
+    sql(sqlQuery)
+        .returnsOrdered("Y=1997; M=12; product_id=1558; S=30", "Y=1997; M=3; product_id=1558; S=29",
+            "Y=1997; M=5; product_id=1558; S=27")
+        .explainContains(expectedPlan)
+        .queryContains(druidChecker(expectedDruidQueryType));
+  }
+
+  @Test public void testGroupByTimeSortOverMetrics() {
+    final String sqlQuery = "SELECT count(*) as c , SUM(\"unit_sales\") as s,"
+        + " floor(\"timestamp\" to month)"
+        + " FROM \"foodmart\" group by floor(\"timestamp\" to month) order by s DESC";
+    sql(sqlQuery)
+        .returnsOrdered("C=8716; S=26796; EXPR$2=1997-12-01 00:00:00",
+            "C=8231; S=25270; EXPR$2=1997-11-01 00:00:00",
+            "C=7752; S=23763; EXPR$2=1997-07-01 00:00:00",
+            "C=7710; S=23706; EXPR$2=1997-03-01 00:00:00",
+            "C=7038; S=21697; EXPR$2=1997-08-01 00:00:00",
+            "C=7033; S=21628; EXPR$2=1997-01-01 00:00:00",
+            "C=6912; S=21350; EXPR$2=1997-06-01 00:00:00",
+            "C=6865; S=21081; EXPR$2=1997-05-01 00:00:00",
+            "C=6844; S=20957; EXPR$2=1997-02-01 00:00:00",
+            "C=6662; S=20388; EXPR$2=1997-09-01 00:00:00",
+            "C=6588; S=20179; EXPR$2=1997-04-01 00:00:00",
+            "C=6478; S=19958; EXPR$2=1997-10-01 00:00:00")
+        .queryContains(druidChecker("'queryType':'groupBy'"))
+        .explainContains("DruidQuery(table=[[foodmart, foodmart]],"
+            + " intervals=[[1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z]],"
+            + " projects=[[FLOOR($0, FLAG(MONTH)), $89]], groups=[{0}], "
+            + "aggs=[[COUNT(), SUM($1)]], sort0=[2], dir0=[DESC])");
+  }
+
+  @Test public void testNumericOrderingOfOrderByOperatorFullTime() {
+    final String sqlQuery = "SELECT \"timestamp\" as \"timestamp\","
+        + " count(*) as c, SUM(\"unit_sales\") as s FROM "
+        + "\"foodmart\" group by \"timestamp\" order by \"timestamp\" DESC, c DESC, s LIMIT 5";
+    final String druidSubQuery = "'limitSpec':{'type':'default','limit':5,"
+        + "'columns':[{'dimension':'extract','direction':'descending',"
+        + "'dimensionOrder':'lexicographic'},{'dimension':'C',"
+        + "'direction':'descending','dimensionOrder':'numeric'},{'dimension':'S',"
+        + "'direction':'ascending','dimensionOrder':'numeric'}]},"
+        + "'aggregations':[{'type':'count','name':'C'},{'type':'longSum',"
+        + "'name':'S','fieldName':'unit_sales'}]";
+    sql(sqlQuery).returnsOrdered("timestamp=1997-12-30 00:00:00; C=22; S=36\ntimestamp=1997-12-29"
+        + " 00:00:00; C=321; S=982\ntimestamp=1997-12-28 00:00:00; C=480; "
+        + "S=1496\ntimestamp=1997-12-27 00:00:00; C=363; S=1156\ntimestamp=1997-12-26 00:00:00; "
+        + "C=144; S=420").queryContains(druidChecker(druidSubQuery));
+
+  }
+
+  @Test public void testNumericOrderingOfOrderByOperatorTimeExtract() {
+    final String sqlQuery = "SELECT extract(day from \"timestamp\") as d, extract(month from "
+        + "\"timestamp\") as m,  year(\"timestamp\") as y , count(*) as c, SUM(\"unit_sales\")  "
+        + "as s FROM "
+        + "\"foodmart\" group by  extract(day from \"timestamp\"), extract(month from \"timestamp\"), "
+        + "year(\"timestamp\")  order by d DESC, m ASC, y DESC LIMIT 5";
+    final String druidSubQuery = "'limitSpec':{'type':'default','limit':5,"
+        + "'columns':[{'dimension':'extract_day','direction':'descending',"
+        + "'dimensionOrder':'numeric'},{'dimension':'extract_month',"
+        + "'direction':'ascending','dimensionOrder':'numeric'},"
+        + "{'dimension':'extract_year','direction':'descending',"
+        + "'dimensionOrder':'numeric'}]}";
+    sql(sqlQuery).returnsOrdered("D=30; M=3; Y=1997; C=114; S=351\nD=30; M=5; Y=1997; "
+        + "C=24; S=34\nD=30; M=6; Y=1997; C=73; S=183\nD=30; M=7; Y=1997; C=29; S=54\nD=30; M=8; "
+        + "Y=1997; C=137; S=422").queryContains(druidChecker(druidSubQuery));
+
+  }
+
+  @Test public void testNumericOrderingOfOrderByOperatorStringDims() {
+    final String sqlQuery = "SELECT \"brand_name\", count(*) as c, SUM(\"unit_sales\")  "
+        + "as s FROM "
+        + "\"foodmart\" group by \"brand_name\" order by \"brand_name\"  DESC LIMIT 5";
+    final String druidSubQuery = "'limitSpec':{'type':'default','limit':5,"
+        + "'columns':[{'dimension':'brand_name','direction':'descending',"
+        + "'dimensionOrder':'lexicographic'}]}";
+    sql(sqlQuery).returnsOrdered("brand_name=Washington; C=576; S=1775\nbrand_name=Walrus; C=457;"
+        + " S=1399\nbrand_name=Urban; C=299; S=924\nbrand_name=Tri-State; C=2339; "
+        + "S=7270\nbrand_name=Toucan; C=123; S=380").queryContains(druidChecker(druidSubQuery));
+
+  }
+
+  @Test public void testGroupByWeekExtract() {
+    final String sql = "SELECT extract(week from \"timestamp\") from \"foodmart\" where "
+        + "\"product_id\" = 1558 and extract(week from \"timestamp\") IN (10, 11) group by extract"
+        + "(week from \"timestamp\")";
+
+    final String druidQuery = "{'queryType':'groupBy','dataSource':'foodmart',"
+        + "'granularity':'all','dimensions':[{'type':'extraction',"
+        + "'dimension':'__time','outputName':'extract_week',"
+        + "'extractionFn':{'type':'timeFormat','format':'w','timeZone':'UTC',"
+        + "'locale':'en-US'}}],'limitSpec':{'type':'default'},"
+        + "'filter':{'type':'and','fields':[{'type':'bound','dimension':'product_id',"
+        + "'lower':'1558','lowerStrict':false,'upper':'1558','upperStrict':false,"
+        + "'ordering':'numeric'},{'type':'or',"
+        + "'fields':[{'type':'bound','dimension':'__time','lower':'10','lowerStrict':false,"
+        + "'upper':'10','upperStrict':false,'ordering':'numeric',"
+        + "'extractionFn':{'type':'timeFormat','format':'w','timeZone':'UTC',"
+        + "'locale':'en-US'}},{'type':'bound','dimension':'__time','lower':'11','lowerStrict':false,"
+        + "'upper':'11','upperStrict':false,'ordering':'numeric',"
+        + "'extractionFn':{'type':'timeFormat','format':'w',"
+        + "'timeZone':'UTC','locale':'en-US'}}]}]},"
+        + "'aggregations':[],"
+        + "'intervals':['1900-01-09T00:00:00.000Z/2992-01-10T00:00:00.000Z']}";
+    sql(sql).returnsOrdered("EXPR$0=10\nEXPR$0=11").queryContains(druidChecker(druidQuery));
+  }
+
+  /** Test case for
+   * <a href="https://issues.apache.org/jira/browse/CALCITE-1765">[CALCITE-1765]
+   * Druid adapter: Gracefully handle granularity that cannot be pushed to
+   * extraction function</a>. */
+  @Test public void testTimeExtractThatCannotBePushed() {
+    final String sql = "SELECT extract(

<TRUNCATED>