add a bunch of tests with array typed columns to CalciteArraysQueryTest (#15101)

* add a bunch of tests with array typed columns to CalciteArraysQueryTest
* fix a bug with unnest filter pushdown when filtering on unnested array columns
This commit is contained in:
Clint Wylie 2023-10-09 06:16:06 -07:00 committed by GitHub
parent 549ef56288
commit 1fc8fb1b20
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
5 changed files with 1962 additions and 77 deletions

View File

@ -27,6 +27,7 @@ import com.google.inject.Module;
import com.google.inject.TypeLiteral;
import org.apache.druid.collections.ReferenceCountingResourceHolder;
import org.apache.druid.collections.ResourceHolder;
import org.apache.druid.data.input.ResourceInputSource;
import org.apache.druid.data.input.impl.DimensionsSpec;
import org.apache.druid.data.input.impl.LongDimensionSchema;
import org.apache.druid.data.input.impl.StringDimensionSchema;
@ -47,6 +48,7 @@ import org.apache.druid.msq.guice.MSQIndexingModule;
import org.apache.druid.msq.querykit.DataSegmentProvider;
import org.apache.druid.query.DruidProcessingConfig;
import org.apache.druid.query.ForwardingQueryProcessingPool;
import org.apache.druid.query.NestedDataTestUtils;
import org.apache.druid.query.QueryProcessingPool;
import org.apache.druid.query.aggregation.CountAggregatorFactory;
import org.apache.druid.query.aggregation.DoubleSumAggregatorFactory;
@ -74,7 +76,9 @@ import org.apache.druid.segment.writeout.OffHeapMemorySegmentWriteOutMediumFacto
import org.apache.druid.server.SegmentManager;
import org.apache.druid.server.coordination.DataSegmentAnnouncer;
import org.apache.druid.server.coordination.NoopDataSegmentAnnouncer;
import org.apache.druid.sql.calcite.CalciteArraysQueryTest;
import org.apache.druid.sql.calcite.util.CalciteTests;
import org.apache.druid.sql.calcite.util.TestDataBuilder;
import org.apache.druid.timeline.DataSegment;
import org.apache.druid.timeline.SegmentId;
import org.easymock.EasyMock;
@ -83,7 +87,6 @@ import org.junit.rules.TemporaryFolder;
import org.mockito.Mockito;
import javax.annotation.Nullable;
import java.io.File;
import java.io.IOException;
import java.util.List;
import java.util.Set;
@ -232,7 +235,7 @@ public class CalciteMSQTestsHelper
.build();
index = IndexBuilder
.create()
.tmpDir(new File(temporaryFolder.newFolder(), "1"))
.tmpDir(temporaryFolder.newFolder())
.segmentWriteOutMediumFactory(OffHeapMemorySegmentWriteOutMediumFactory.instance())
.schema(foo1Schema)
.rows(ROWS1)
@ -259,7 +262,7 @@ public class CalciteMSQTestsHelper
.build();
index = IndexBuilder
.create()
.tmpDir(new File(temporaryFolder.newFolder(), "2"))
.tmpDir(temporaryFolder.newFolder())
.segmentWriteOutMediumFactory(OffHeapMemorySegmentWriteOutMediumFactory.instance())
.schema(indexSchemaDifferentDim3M1Types)
.rows(ROWS2)
@ -269,7 +272,7 @@ public class CalciteMSQTestsHelper
case CalciteTests.BROADCAST_DATASOURCE:
index = IndexBuilder
.create()
.tmpDir(new File(temporaryFolder.newFolder(), "3"))
.tmpDir(temporaryFolder.newFolder())
.segmentWriteOutMediumFactory(OffHeapMemorySegmentWriteOutMediumFactory.instance())
.schema(INDEX_SCHEMA_NUMERIC_DIMS)
.rows(ROWS1_WITH_NUMERIC_DIMS)
@ -278,12 +281,36 @@ public class CalciteMSQTestsHelper
case DATASOURCE5:
index = IndexBuilder
.create()
.tmpDir(new File(temporaryFolder.newFolder(), "5"))
.tmpDir(temporaryFolder.newFolder())
.segmentWriteOutMediumFactory(OffHeapMemorySegmentWriteOutMediumFactory.instance())
.schema(INDEX_SCHEMA_LOTS_O_COLUMNS)
.rows(ROWS_LOTS_OF_COLUMNS)
.buildMMappedIndex();
break;
case CalciteArraysQueryTest.DATA_SOURCE_ARRAYS:
index = IndexBuilder.create()
.tmpDir(temporaryFolder.newFolder())
.segmentWriteOutMediumFactory(OffHeapMemorySegmentWriteOutMediumFactory.instance())
.schema(
new IncrementalIndexSchema.Builder()
.withTimestampSpec(NestedDataTestUtils.AUTO_SCHEMA.getTimestampSpec())
.withDimensionsSpec(NestedDataTestUtils.AUTO_SCHEMA.getDimensionsSpec())
.withMetrics(
new CountAggregatorFactory("cnt")
)
.withRollup(false)
.build()
)
.inputSource(
ResourceInputSource.of(
NestedDataTestUtils.class.getClassLoader(),
NestedDataTestUtils.ARRAY_TYPES_DATA_FILE
)
)
.inputFormat(TestDataBuilder.DEFAULT_JSON_INPUT_FORMAT)
.inputTmpDir(temporaryFolder.newFolder())
.buildMMappedIndex();
break;
default:
throw new ISE("Cannot query segment %s in test runner", segmentId);

View File

@ -322,19 +322,17 @@ public class UnnestStorageAdapter implements StorageAdapter
// outside filter contains unnested column
// requires check for OR and And filters, disqualify rewrite for non-unnest filters
if (queryFilter instanceof BooleanFilter) {
boolean isTopLevelAndFilter = queryFilter instanceof AndFilter;
List<Filter> preFilterList = recursiveRewriteOnUnnestFilters(
(BooleanFilter) queryFilter,
inputColumn,
inputColumnCapabilites,
filterSplitter,
isTopLevelAndFilter
filterSplitter
);
// If rewite on entire query filter is successful then add entire filter to preFilter else skip and only add to post filter.
if (filterSplitter.getPreFilterCount() == filterSplitter.getOriginalFilterCount()) {
if (!preFilterList.isEmpty()) {
if (queryFilter instanceof AndFilter) {
filterSplitter.addPreFilter(new AndFilter(preFilterList));
} else if (queryFilter instanceof OrFilter) {
} else if (queryFilter instanceof OrFilter && filterSplitter.getPreFilterCount() == filterSplitter.getOriginalFilterCount()) {
filterSplitter.addPreFilter(new OrFilter(preFilterList));
}
}
@ -470,8 +468,7 @@ public class UnnestStorageAdapter implements StorageAdapter
BooleanFilter queryFilter,
final String inputColumn,
final ColumnCapabilities inputColumnCapabilites,
final FilterSplitter filterSplitter,
final boolean isTopLevelAndFilter
final FilterSplitter filterSplitter
)
{
final List<Filter> preFilterList = new ArrayList<>();
@ -482,25 +479,26 @@ public class UnnestStorageAdapter implements StorageAdapter
(BooleanFilter) filter,
inputColumn,
inputColumnCapabilites,
filterSplitter,
isTopLevelAndFilter
filterSplitter
);
if (!andChildFilters.isEmpty()) {
preFilterList.add(new AndFilter(andChildFilters));
}
} else if (filter instanceof OrFilter) {
// in case of Or Fiters, we set isTopLevelAndFilter to false that prevents pushing down any child filters to base
List<Filter> orChildFilters = recursiveRewriteOnUnnestFilters(
(BooleanFilter) filter,
inputColumn,
inputColumnCapabilites,
filterSplitter,
false
filterSplitter
);
preFilterList.add(new OrFilter(orChildFilters));
if (orChildFilters.size() == ((OrFilter) filter).getFilters().size()) {
preFilterList.add(new OrFilter(orChildFilters));
}
} else if (filter instanceof NotFilter) {
// nothing to do here...
continue;
} else {
// can we rewrite
final Filter newFilter = rewriteFilterOnUnnestColumnIfPossible(
filter,
inputColumn,
@ -511,13 +509,6 @@ public class UnnestStorageAdapter implements StorageAdapter
preFilterList.add(newFilter);
filterSplitter.addToPreFilterCount(1);
}
/*
Push down the filters to base only if top level is And Filter
we can not push down if top level filter is OR or unnestColumn is derived expression like arrays
*/
if (isTopLevelAndFilter && getUnnestInputIfDirectAccess(unnestColumn) != null) {
filterSplitter.addPreFilter(newFilter != null ? newFilter : filter);
}
filterSplitter.addToOriginalFilterCount(1);
}
} else {

View File

@ -20,6 +20,8 @@
package org.apache.druid.segment;
import com.google.common.collect.ImmutableList;
import org.apache.druid.data.input.InputSource;
import org.apache.druid.data.input.ResourceInputSource;
import org.apache.druid.java.util.common.DateTimes;
import org.apache.druid.java.util.common.Pair;
import org.apache.druid.java.util.common.granularity.Granularities;
@ -27,11 +29,14 @@ import org.apache.druid.java.util.common.granularity.Granularity;
import org.apache.druid.java.util.common.guava.Sequence;
import org.apache.druid.java.util.common.io.Closer;
import org.apache.druid.math.expr.ExprMacroTable;
import org.apache.druid.query.NestedDataTestUtils;
import org.apache.druid.query.QueryMetrics;
import org.apache.druid.query.dimension.DefaultDimensionSpec;
import org.apache.druid.query.filter.EqualityFilter;
import org.apache.druid.query.filter.Filter;
import org.apache.druid.query.filter.SelectorDimFilter;
import org.apache.druid.segment.column.ColumnCapabilities;
import org.apache.druid.segment.column.ColumnType;
import org.apache.druid.segment.column.ValueType;
import org.apache.druid.segment.filter.AndFilter;
import org.apache.druid.segment.filter.OrFilter;
@ -40,8 +45,10 @@ import org.apache.druid.segment.generator.GeneratorBasicSchemas;
import org.apache.druid.segment.generator.GeneratorSchemaInfo;
import org.apache.druid.segment.generator.SegmentGenerator;
import org.apache.druid.segment.incremental.IncrementalIndex;
import org.apache.druid.segment.incremental.IncrementalIndexSchema;
import org.apache.druid.segment.incremental.IncrementalIndexStorageAdapter;
import org.apache.druid.segment.join.PostJoinCursor;
import org.apache.druid.segment.transform.TransformSpec;
import org.apache.druid.segment.virtual.ExpressionVirtualColumn;
import org.apache.druid.testing.InitializedNullHandlingTest;
import org.apache.druid.timeline.DataSegment;
@ -53,9 +60,12 @@ import org.joda.time.Interval;
import org.junit.AfterClass;
import org.junit.Assert;
import org.junit.BeforeClass;
import org.junit.ClassRule;
import org.junit.Test;
import org.junit.rules.TemporaryFolder;
import javax.annotation.Nullable;
import java.io.IOException;
import java.util.Arrays;
import java.util.List;
@ -66,18 +76,23 @@ import static org.apache.druid.segment.filter.Filters.or;
public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
{
@ClassRule
public static TemporaryFolder tmp = new TemporaryFolder();
private static Closer CLOSER;
private static IncrementalIndex INCREMENTAL_INDEX;
private static IncrementalIndexStorageAdapter INCREMENTAL_INDEX_STORAGE_ADAPTER;
private static QueryableIndex QUERYABLE_INDEX;
private static UnnestStorageAdapter UNNEST_STORAGE_ADAPTER;
private static UnnestStorageAdapter UNNEST_STORAGE_ADAPTER1;
private static UnnestStorageAdapter UNNEST_ARRAYS;
private static List<StorageAdapter> ADAPTERS;
private static String COLUMNNAME = "multi-string1";
private static String INPUT_COLUMN_NAME = "multi-string1";
private static String OUTPUT_COLUMN_NAME = "unnested-multi-string1";
private static String OUTPUT_COLUMN_NAME1 = "unnested-multi-string1-again";
@BeforeClass
public static void setup()
public static void setup() throws IOException
{
CLOSER = Closer.create();
final GeneratorSchemaInfo schemaInfo = GeneratorBasicSchemas.SCHEMA_MAP.get("expression-testbench");
@ -98,13 +113,40 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
INCREMENTAL_INDEX_STORAGE_ADAPTER = new IncrementalIndexStorageAdapter(INCREMENTAL_INDEX);
UNNEST_STORAGE_ADAPTER = new UnnestStorageAdapter(
INCREMENTAL_INDEX_STORAGE_ADAPTER,
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + COLUMNNAME + "\"", null, ExprMacroTable.nil()),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + INPUT_COLUMN_NAME + "\"", null, ExprMacroTable.nil()),
null
);
UNNEST_STORAGE_ADAPTER1 = new UnnestStorageAdapter(
UNNEST_STORAGE_ADAPTER,
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME1, "\"" + COLUMNNAME + "\"", null, ExprMacroTable.nil()),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME1, "\"" + INPUT_COLUMN_NAME + "\"", null, ExprMacroTable.nil()),
null
);
final InputSource inputSource = ResourceInputSource.of(
UnnestStorageAdapterTest.class.getClassLoader(),
NestedDataTestUtils.ALL_TYPES_TEST_DATA_FILE
);
IndexBuilder bob = IndexBuilder.create()
.tmpDir(tmp.newFolder())
.schema(
IncrementalIndexSchema.builder()
.withTimestampSpec(NestedDataTestUtils.TIMESTAMP_SPEC)
.withDimensionsSpec(NestedDataTestUtils.AUTO_DISCOVERY)
.withQueryGranularity(Granularities.DAY)
.withRollup(false)
.withMinTimestamp(0)
.build()
)
.indexSpec(IndexSpec.DEFAULT)
.inputSource(inputSource)
.inputFormat(NestedDataTestUtils.DEFAULT_JSON_INPUT_FORMAT)
.transform(TransformSpec.NONE)
.inputTmpDir(tmp.newFolder());
QUERYABLE_INDEX = CLOSER.register(bob.buildMMappedIndex());
UNNEST_ARRAYS = new UnnestStorageAdapter(
new QueryableIndexStorageAdapter(QUERYABLE_INDEX),
new ExpressionVirtualColumn("u", "\"arrayLongNulls\"", ColumnType.LONG, ExprMacroTable.nil()),
null
);
@ -269,7 +311,7 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
{
final UnnestStorageAdapter unnestStorageAdapter = new UnnestStorageAdapter(
new TestStorageAdapter(INCREMENTAL_INDEX),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + COLUMNNAME + "\"", null, ExprMacroTable.nil()),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + INPUT_COLUMN_NAME + "\"", null, ExprMacroTable.nil()),
null
);
@ -313,7 +355,7 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
{
final UnnestStorageAdapter unnestStorageAdapter = new UnnestStorageAdapter(
new TestStorageAdapter(INCREMENTAL_INDEX),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + COLUMNNAME + "\"", null, ExprMacroTable.nil()),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + INPUT_COLUMN_NAME + "\"", null, ExprMacroTable.nil()),
null
);
@ -365,7 +407,7 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
selector(OUTPUT_COLUMN_NAME, "3"),
or(ImmutableList.of(
selector("newcol", "2"),
selector(COLUMNNAME, "2"),
selector(INPUT_COLUMN_NAME, "2"),
selector(OUTPUT_COLUMN_NAME, "1")
))
));
@ -383,10 +425,10 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
or(ImmutableList.of(
or(ImmutableList.of(
selector("newcol", "2"),
selector(COLUMNNAME, "2"),
selector(INPUT_COLUMN_NAME, "2"),
and(ImmutableList.of(
selector("newcol", "3"),
selector(COLUMNNAME, "7")
selector(INPUT_COLUMN_NAME, "7")
))
)),
selector(OUTPUT_COLUMN_NAME, "1")
@ -406,11 +448,11 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
or(ImmutableList.of(
or(ImmutableList.of(
selector("newcol", "2"),
selector(COLUMNNAME, "2"),
selector(INPUT_COLUMN_NAME, "2"),
and(ImmutableList.of(
selector("newcol", "3"),
and(ImmutableList.of(
selector(COLUMNNAME, "7"),
selector(INPUT_COLUMN_NAME, "7"),
selector("newcol_1", "10")
))
))
@ -431,7 +473,7 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
selector(OUTPUT_COLUMN_NAME, "3"),
and(ImmutableList.of(
selector("newcol", "2"),
selector(COLUMNNAME, "2"),
selector(INPUT_COLUMN_NAME, "2"),
selector(OUTPUT_COLUMN_NAME, "1")
))
));
@ -449,11 +491,11 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
selector(OUTPUT_COLUMN_NAME, "3"),
or(ImmutableList.of(
selector("newcol", "2"),
selector(COLUMNNAME, "2")
selector(INPUT_COLUMN_NAME, "2")
)),
or(ImmutableList.of(
selector("newcol", "4"),
selector(COLUMNNAME, "8"),
selector(INPUT_COLUMN_NAME, "8"),
selector(OUTPUT_COLUMN_NAME, "6")
))
));
@ -469,7 +511,7 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
{
final Filter testQueryFilter = and(ImmutableList.of(
not(selector(OUTPUT_COLUMN_NAME, "3")),
selector(COLUMNNAME, "2")
selector(INPUT_COLUMN_NAME, "2")
));
testComputeBaseAndPostUnnestFilters(
testQueryFilter,
@ -483,7 +525,7 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
{
final Filter testQueryFilter = or(ImmutableList.of(
not(selector(OUTPUT_COLUMN_NAME, "3")),
selector(COLUMNNAME, "2")
selector(INPUT_COLUMN_NAME, "2")
));
testComputeBaseAndPostUnnestFilters(
testQueryFilter,
@ -500,10 +542,10 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
or(ImmutableList.of(
or(ImmutableList.of(
selector("newcol", "2"),
selector(COLUMNNAME, "2"),
selector(INPUT_COLUMN_NAME, "2"),
and(ImmutableList.of(
selector("newcol", "3"),
selector(COLUMNNAME, "7")
selector(INPUT_COLUMN_NAME, "7")
))
)),
selector(OUTPUT_COLUMN_NAME, "1")
@ -516,12 +558,97 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
);
}
@Test
public void testPartialArrayPushdown()
{
final Filter testQueryFilter = and(
ImmutableList.of(
new EqualityFilter("u", ColumnType.LONG, 1L, null),
new EqualityFilter("str", ColumnType.STRING, "a", null),
new EqualityFilter("long", ColumnType.LONG, 1L, null)
)
);
testComputeBaseAndPostUnnestFilters(
UNNEST_ARRAYS,
testQueryFilter,
"(str = a && long = 1 (LONG))",
"(u = 1 (LONG) && str = a && long = 1 (LONG))"
);
}
@Test
public void testPartialArrayPushdownNested()
{
final Filter testQueryFilter = and(
ImmutableList.of(
and(
ImmutableList.of(
new EqualityFilter("u", ColumnType.LONG, 1L, null),
new EqualityFilter("str", ColumnType.STRING, "a", null)
)
),
new EqualityFilter("long", ColumnType.LONG, 1L, null)
)
);
// this seems wrong since we should be able to push down str = a and long = 1
testComputeBaseAndPostUnnestFilters(
UNNEST_ARRAYS,
testQueryFilter,
"(str = a && long = 1 (LONG))",
"(u = 1 (LONG) && str = a && long = 1 (LONG))"
);
}
@Test
public void testPartialArrayPushdown2()
{
final Filter testQueryFilter = and(
ImmutableList.of(
or(
ImmutableList.of(
new EqualityFilter("u", ColumnType.LONG, 1L, null),
new EqualityFilter("str", ColumnType.STRING, "a", null)
)
),
new EqualityFilter("long", ColumnType.LONG, 1L, null)
)
);
testComputeBaseAndPostUnnestFilters(
UNNEST_ARRAYS,
testQueryFilter,
"long = 1 (LONG)",
"((u = 1 (LONG) || str = a) && long = 1 (LONG))"
);
}
@Test
public void testArrayCannotPushdown2()
{
final Filter testQueryFilter = or(
ImmutableList.of(
or(
ImmutableList.of(
new EqualityFilter("u", ColumnType.LONG, 1L, null),
new EqualityFilter("str", ColumnType.STRING, "a", null)
)
),
new EqualityFilter("long", ColumnType.LONG, 1L, null)
)
);
testComputeBaseAndPostUnnestFilters(
UNNEST_ARRAYS,
testQueryFilter,
"",
"(u = 1 (LONG) || str = a || long = 1 (LONG))"
);
}
@Test
public void test_pushdown_filters_unnested_dimension_with_unnest_adapters()
{
final UnnestStorageAdapter unnestStorageAdapter = new UnnestStorageAdapter(
new TestStorageAdapter(INCREMENTAL_INDEX),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + COLUMNNAME + "\"", null, ExprMacroTable.nil()),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + INPUT_COLUMN_NAME + "\"", null, ExprMacroTable.nil()),
new SelectorDimFilter(OUTPUT_COLUMN_NAME, "1", null)
);
@ -567,7 +694,7 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
{
final UnnestStorageAdapter unnestStorageAdapter = new UnnestStorageAdapter(
new TestStorageAdapter(INCREMENTAL_INDEX),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + COLUMNNAME + "\"", null, ExprMacroTable.nil()),
new ExpressionVirtualColumn(OUTPUT_COLUMN_NAME, "\"" + INPUT_COLUMN_NAME + "\"", null, ExprMacroTable.nil()),
null
);
@ -613,14 +740,29 @@ public class UnnestStorageAdapterTest extends InitializedNullHandlingTest
String expectedPostUnnest
)
{
final String inputColumn = UNNEST_STORAGE_ADAPTER.getUnnestInputIfDirectAccess(UNNEST_STORAGE_ADAPTER.getUnnestColumn());
final VirtualColumn vc = UNNEST_STORAGE_ADAPTER.getUnnestColumn();
Pair<Filter, Filter> filterPair = UNNEST_STORAGE_ADAPTER.computeBaseAndPostUnnestFilters(
testComputeBaseAndPostUnnestFilters(
UNNEST_STORAGE_ADAPTER,
testQueryFilter,
expectedBasePushDown,
expectedPostUnnest
);
}
public void testComputeBaseAndPostUnnestFilters(
UnnestStorageAdapter adapter,
Filter testQueryFilter,
String expectedBasePushDown,
String expectedPostUnnest
)
{
final String inputColumn = adapter.getUnnestInputIfDirectAccess(adapter.getUnnestColumn());
final VirtualColumn vc = adapter.getUnnestColumn();
Pair<Filter, Filter> filterPair = adapter.computeBaseAndPostUnnestFilters(
testQueryFilter,
null,
VirtualColumns.EMPTY,
inputColumn,
vc.capabilities(UNNEST_STORAGE_ADAPTER, inputColumn)
vc.capabilities(adapter, inputColumn)
);
Filter actualPushDownFilter = filterPair.lhs;
Filter actualPostUnnestFilter = filterPair.rhs;

View File

@ -172,7 +172,7 @@ public class TestDataBuilder
);
private static final IncrementalIndexSchema INDEX_SCHEMA = new IncrementalIndexSchema.Builder()
public static final IncrementalIndexSchema INDEX_SCHEMA = new IncrementalIndexSchema.Builder()
.withMetrics(
new CountAggregatorFactory("cnt"),
new FloatSumAggregatorFactory("m1", "m1"),