diff --git a/extensions-core/datasketches/pom.xml b/extensions-core/datasketches/pom.xml index 54e9ceaa5f8..30413806db8 100644 --- a/extensions-core/datasketches/pom.xml +++ b/extensions-core/datasketches/pom.xml @@ -40,6 +40,10 @@ sketches-core 0.10.3 + + org.apache.commons + commons-math3 + io.druid druid-api diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregatorFactory.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregatorFactory.java new file mode 100644 index 00000000000..1caadd7190b --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregatorFactory.java @@ -0,0 +1,311 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.ArrayList; +import java.util.Collections; +import java.util.Comparator; +import java.util.List; +import java.util.Objects; + +import javax.annotation.Nullable; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.google.common.base.Preconditions; + +import io.druid.java.util.common.IAE; +import io.druid.query.aggregation.Aggregator; +import io.druid.query.aggregation.AggregatorFactory; +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.BufferAggregator; +import io.druid.query.cache.CacheKeyBuilder; +import io.druid.query.dimension.DefaultDimensionSpec; +import io.druid.segment.BaseDoubleColumnValueSelector; +import io.druid.segment.BaseObjectColumnValueSelector; +import io.druid.segment.ColumnSelectorFactory; +import io.druid.segment.DimensionSelector; +import io.druid.segment.DimensionSelectorUtils; +import io.druid.segment.NilColumnValueSelector; + +import com.yahoo.sketches.Util; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesUnion; +import com.yahoo.sketches.tuple.ArrayOfDoublesSetOperationBuilder; + +public class ArrayOfDoublesSketchAggregatorFactory extends AggregatorFactory +{ + + public static final Comparator COMPARATOR = + Comparator.nullsFirst(Comparator.comparingDouble(ArrayOfDoublesSketch::getEstimate)); + + private final String name; + private final String fieldName; + private final int nominalEntries; + private final int numberOfValues; + // if specified indicates building sketched from raw data, and also implies the number of values + @Nullable private final List metricColumns; + + @JsonCreator + public ArrayOfDoublesSketchAggregatorFactory( + @JsonProperty("name") final String name, + @JsonProperty("fieldName") final String fieldName, + @JsonProperty("nominalEntries") @Nullable final Integer nominalEntries, + @JsonProperty("metricColumns") @Nullable final List metricColumns, + @JsonProperty("numberOfValues") @Nullable final Integer numberOfValues + ) + { + this.name = Preconditions.checkNotNull(name, "Must have a valid, non-null aggregator name"); + this.fieldName = Preconditions.checkNotNull(fieldName, "Must have a valid, non-null fieldName"); + this.nominalEntries = nominalEntries == null ? Util.DEFAULT_NOMINAL_ENTRIES : nominalEntries; + Util.checkIfPowerOf2(this.nominalEntries, "nominalEntries"); + this.metricColumns = metricColumns; + this.numberOfValues = numberOfValues == null ? (metricColumns == null ? 1 : metricColumns.size()) : numberOfValues; + if (metricColumns != null && metricColumns.size() != this.numberOfValues) { + throw new IAE( + "Number of metricColumns [%d] must agree with numValues [%d]", + metricColumns.size(), + this.numberOfValues + ); + } + } + + @Override + public Aggregator factorize(final ColumnSelectorFactory metricFactory) + { + if (metricColumns == null) { // input is sketches, use merge aggregator + final BaseObjectColumnValueSelector selector = metricFactory + .makeColumnValueSelector(fieldName); + if (selector instanceof NilColumnValueSelector) { + return new ArrayOfDoublesSketchNoOpAggregator(numberOfValues); + } + return new ArrayOfDoublesSketchMergeAggregator(selector, nominalEntries, numberOfValues); + } + // input is raw data (key and array of values), use build aggregator + final DimensionSelector keySelector = metricFactory + .makeDimensionSelector(new DefaultDimensionSpec(fieldName, fieldName)); + if (DimensionSelectorUtils.isNilSelector(keySelector)) { + return new ArrayOfDoublesSketchNoOpAggregator(numberOfValues); + } + final List valueSelectors = new ArrayList<>(); + for (final String column : metricColumns) { + final BaseDoubleColumnValueSelector valueSelector = metricFactory.makeColumnValueSelector(column); + valueSelectors.add(valueSelector); + } + return new ArrayOfDoublesSketchBuildAggregator(keySelector, valueSelectors, nominalEntries); + } + + @Override + public BufferAggregator factorizeBuffered(final ColumnSelectorFactory metricFactory) + { + if (metricColumns == null) { // input is sketches, use merge aggregator + final BaseObjectColumnValueSelector selector = metricFactory + .makeColumnValueSelector(fieldName); + if (selector instanceof NilColumnValueSelector) { + return new ArrayOfDoublesSketchNoOpBufferAggregator(numberOfValues); + } + return new ArrayOfDoublesSketchMergeBufferAggregator( + selector, + nominalEntries, + numberOfValues, + getMaxIntermediateSize() + ); + } + // input is raw data (key and array of values), use build aggregator + final DimensionSelector keySelector = metricFactory + .makeDimensionSelector(new DefaultDimensionSpec(fieldName, fieldName)); + if (DimensionSelectorUtils.isNilSelector(keySelector)) { + return new ArrayOfDoublesSketchNoOpBufferAggregator(numberOfValues); + } + final List valueSelectors = new ArrayList<>(); + for (final String column : metricColumns) { + final BaseDoubleColumnValueSelector valueSelector = metricFactory.makeColumnValueSelector(column); + valueSelectors.add(valueSelector); + } + return new ArrayOfDoublesSketchBuildBufferAggregator( + keySelector, + valueSelectors, + nominalEntries, + getMaxIntermediateSize() + ); + } + + @Override + public Object deserialize(final Object object) + { + return ArrayOfDoublesSketchOperations.deserialize(object); + } + + @Override + public Comparator getComparator() + { + return COMPARATOR; + } + + @Override + public Object combine(@Nullable final Object lhs, @Nullable final Object rhs) + { + final ArrayOfDoublesUnion union = new ArrayOfDoublesSetOperationBuilder().setNominalEntries(nominalEntries) + .setNumberOfValues(numberOfValues).buildUnion(); + if (lhs != null) { + union.update((ArrayOfDoublesSketch) lhs); + } + if (rhs != null) { + union.update((ArrayOfDoublesSketch) rhs); + } + return union.getResult(); + } + + @Override + @JsonProperty + public String getName() + { + return name; + } + + @JsonProperty + public String getFieldName() + { + return fieldName; + } + + @JsonProperty + public int getNominalEntries() + { + return nominalEntries; + } + + @JsonProperty + public List getMetricColumns() + { + return metricColumns; + } + + @JsonProperty + public int getNumberOfValues() + { + return numberOfValues; + } + + @Override + public List requiredFields() + { + return Collections.singletonList(fieldName); + } + + @Override + public byte[] getCacheKey() + { + final CacheKeyBuilder builder = new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_CACHE_TYPE_ID) + .appendString(name) + .appendString(fieldName) + .appendInt(nominalEntries) + .appendInt(numberOfValues); + if (metricColumns != null) { + builder.appendStrings(metricColumns); + } + return builder.build(); + } + + @Override + public int getMaxIntermediateSize() + { + return ArrayOfDoublesUnion.getMaxBytes(nominalEntries, numberOfValues); + } + + @Override + public List getRequiredColumns() + { + return Collections.singletonList( + new ArrayOfDoublesSketchAggregatorFactory( + fieldName, + fieldName, + nominalEntries, + metricColumns, + numberOfValues + ) + ); + } + + @Override + public AggregatorFactory getCombiningFactory() + { + return new ArrayOfDoublesSketchAggregatorFactory(name, name, nominalEntries, null, numberOfValues); + } + + @Override + public Object finalizeComputation(final Object object) + { + return ((ArrayOfDoublesSketch) object).getEstimate(); + } + + @Override + public String getTypeName() + { + if (metricColumns == null) { + return ArrayOfDoublesSketchModule.ARRAY_OF_DOUBLES_SKETCH_MERGE_AGG; + } + return ArrayOfDoublesSketchModule.ARRAY_OF_DOUBLES_SKETCH_BUILD_AGG; + } + + @Override + public boolean equals(final Object o) + { + if (this == o) { + return true; + } + if (!(o instanceof ArrayOfDoublesSketchAggregatorFactory)) { + return false; + } + final ArrayOfDoublesSketchAggregatorFactory that = (ArrayOfDoublesSketchAggregatorFactory) o; + if (!name.equals(that.name)) { + return false; + } + if (!fieldName.equals(that.fieldName)) { + return false; + } + if (nominalEntries != that.nominalEntries) { + return false; + } + if (!Objects.equals(metricColumns, that.metricColumns)) { + return false; + } + return numberOfValues != that.numberOfValues; + } + + @Override + public int hashCode() + { + return Objects.hash(name, fieldName, nominalEntries, metricColumns, numberOfValues); + } + + @Override + public String toString() + { + return this.getClass().getSimpleName() + "{" + + "fieldName=" + fieldName + + ", name=" + name + + ", nominalEntries=" + nominalEntries + + ", metricColumns=" + metricColumns + + ", numberOfValues=" + numberOfValues + + "}"; + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildAggregator.java new file mode 100644 index 00000000000..57b038af811 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildAggregator.java @@ -0,0 +1,110 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketchBuilder; +import io.druid.query.aggregation.Aggregator; +import io.druid.segment.BaseDoubleColumnValueSelector; +import io.druid.segment.DimensionSelector; +import io.druid.segment.data.IndexedInts; + +import java.util.List; + +/** + * This aggregator builds sketches from raw data. + * The input is in the form of a key and array of double values. + * The output is {@link com.yahoo.sketches.tuple.ArrayOfDoublesSketch}. + */ +public class ArrayOfDoublesSketchBuildAggregator implements Aggregator +{ + + private final DimensionSelector keySelector; + private final BaseDoubleColumnValueSelector[] valueSelectors; + private double[] values; // not part of the state, but to reuse in aggregate() method + private ArrayOfDoublesUpdatableSketch sketch; + + public ArrayOfDoublesSketchBuildAggregator( + final DimensionSelector keySelector, + final List valueSelectors, + final int nominalEntries + ) + { + this.keySelector = keySelector; + this.valueSelectors = valueSelectors.toArray(new BaseDoubleColumnValueSelector[0]); + values = new double[valueSelectors.size()]; + sketch = new ArrayOfDoublesUpdatableSketchBuilder().setNominalEntries(nominalEntries) + .setNumberOfValues(valueSelectors.size()).build(); + } + + /** + * This method uses synchronization because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + */ + @Override + public void aggregate() + { + final IndexedInts keys = keySelector.getRow(); + for (int i = 0; i < valueSelectors.length; i++) { + values[i] = valueSelectors[i].getDouble(); + } + synchronized (this) { + for (int i = 0, keysSize = keys.size(); i < keysSize; i++) { + final String key = keySelector.lookupName(keys.get(i)); + sketch.update(key, values); + } + } + } + + /** + * This method uses synchronization because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + * The returned sketch is a separate instance of ArrayOfDoublesCompactSketch + * representing the current state of the aggregation, and is not affected by consequent + * aggregate() calls + */ + @Override + public synchronized Object get() + { + return sketch.compact(); + } + + @Override + public long getLong() + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public float getFloat() + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public void close() + { + sketch = null; + values = null; + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildBufferAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildBufferAggregator.java new file mode 100644 index 00000000000..b3cbcdae8fc --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildBufferAggregator.java @@ -0,0 +1,173 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import com.google.common.util.concurrent.Striped; +import com.yahoo.memory.WritableMemory; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketches; +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketchBuilder; +import io.druid.query.aggregation.BufferAggregator; +import io.druid.query.monomorphicprocessing.RuntimeShapeInspector; +import io.druid.segment.BaseDoubleColumnValueSelector; +import io.druid.segment.DimensionSelector; +import io.druid.segment.data.IndexedInts; + +import java.nio.ByteBuffer; +import java.util.List; +import java.util.concurrent.locks.Lock; +import java.util.concurrent.locks.ReadWriteLock; + +/** + * This aggregator builds sketches from raw data. + * The input is in the form of a key and array of double values. + * The output is {@link com.yahoo.sketches.tuple.ArrayOfDoublesSketch}. + */ +public class ArrayOfDoublesSketchBuildBufferAggregator implements BufferAggregator +{ + + private static final int NUM_STRIPES = 64; // for locking per buffer position (power of 2 to make index computation faster) + + private final DimensionSelector keySelector; + private final BaseDoubleColumnValueSelector[] valueSelectors; + private final int nominalEntries; + private final int maxIntermediateSize; + private double[] values; // not part of the state, but to reuse in aggregate() method + private final Striped stripedLock = Striped.readWriteLock(NUM_STRIPES); + + public ArrayOfDoublesSketchBuildBufferAggregator( + final DimensionSelector keySelector, + final List valueSelectors, + int nominalEntries, + int maxIntermediateSize + ) + { + this.keySelector = keySelector; + this.valueSelectors = valueSelectors.toArray(new BaseDoubleColumnValueSelector[0]); + this.nominalEntries = nominalEntries; + this.maxIntermediateSize = maxIntermediateSize; + values = new double[valueSelectors.size()]; + } + + @Override + public void init(final ByteBuffer buf, final int position) + { + final WritableMemory mem = WritableMemory.wrap(buf); + final WritableMemory region = mem.writableRegion(position, maxIntermediateSize); + new ArrayOfDoublesUpdatableSketchBuilder().setNominalEntries(nominalEntries) + .setNumberOfValues(valueSelectors.length) + .setNumberOfValues(valueSelectors.length).build(region); + } + + /** + * This method uses locks because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + */ + @Override + public void aggregate(final ByteBuffer buf, final int position) + { + for (int i = 0; i < valueSelectors.length; i++) { + values[i] = valueSelectors[i].getDouble(); + } + final IndexedInts keys = keySelector.getRow(); + // Wrapping memory and ArrayOfDoublesSketch is inexpensive compared to sketch operations. + // Maintaining a cache of wrapped objects per buffer position like in Theta sketch aggregator + // might might be considered, but it would increase complexity including relocate() support. + final WritableMemory mem = WritableMemory.wrap(buf); + final WritableMemory region = mem.writableRegion(position, maxIntermediateSize); + final Lock lock = stripedLock.getAt(lockIndex(position)).writeLock(); + lock.lock(); + try { + final ArrayOfDoublesUpdatableSketch sketch = ArrayOfDoublesSketches.wrapUpdatableSketch(region); + for (int i = 0, keysSize = keys.size(); i < keysSize; i++) { + final String key = keySelector.lookupName(keys.get(i)); + sketch.update(key, values); + } + } + finally { + lock.unlock(); + } + } + + /** + * This method uses locks because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + * The returned sketch is a separate instance of ArrayOfDoublesCompactSketch + * representing the current state of the aggregation, and is not affected by consequent + * aggregate() calls + */ + @Override + public Object get(final ByteBuffer buf, final int position) + { + final WritableMemory mem = WritableMemory.wrap(buf); + final WritableMemory region = mem.writableRegion(position, maxIntermediateSize); + final Lock lock = stripedLock.getAt(lockIndex(position)).readLock(); + lock.lock(); + try { + final ArrayOfDoublesUpdatableSketch sketch = (ArrayOfDoublesUpdatableSketch) ArrayOfDoublesSketches + .wrapSketch(region); + return sketch.compact(); + } + finally { + lock.unlock(); + } + } + + @Override + public float getFloat(final ByteBuffer buf, final int position) + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public long getLong(final ByteBuffer buf, final int position) + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public void close() + { + values = null; + } + + @Override + public void inspectRuntimeShape(final RuntimeShapeInspector inspector) + { + inspector.visit("keySelector", keySelector); + inspector.visit("valueSelectors", valueSelectors); + } + + // compute lock index to avoid boxing in Striped.get() call + static int lockIndex(final int position) + { + return smear(position) % NUM_STRIPES; + } + + // from https://github.com/google/guava/blob/master/guava/src/com/google/common/util/concurrent/Striped.java#L536-L548 + private static int smear(int hashCode) + { + hashCode ^= (hashCode >>> 20) ^ (hashCode >>> 12); + return hashCode ^ (hashCode >>> 7) ^ (hashCode >>> 4); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildComplexMetricSerde.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildComplexMetricSerde.java new file mode 100644 index 00000000000..f8af7ad7eca --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildComplexMetricSerde.java @@ -0,0 +1,47 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import io.druid.data.input.InputRow; +import io.druid.segment.serde.ComplexMetricExtractor; + +public class ArrayOfDoublesSketchBuildComplexMetricSerde extends ArrayOfDoublesSketchMergeComplexMetricSerde +{ + + @Override + public ComplexMetricExtractor getExtractor() + { + return new ComplexMetricExtractor() + { + @Override + public Class extractedClass() + { + return Object.class; + } + + @Override + public Object extractValue(final InputRow inputRow, final String metricName) + { + return inputRow.getRaw(metricName); + } + }; + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchJsonSerializer.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchJsonSerializer.java new file mode 100644 index 00000000000..4ee39f79ea4 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchJsonSerializer.java @@ -0,0 +1,43 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.io.IOException; + +import com.fasterxml.jackson.core.JsonGenerator; +import com.fasterxml.jackson.core.JsonProcessingException; +import com.fasterxml.jackson.databind.JsonSerializer; +import com.fasterxml.jackson.databind.SerializerProvider; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +public class ArrayOfDoublesSketchJsonSerializer extends JsonSerializer +{ + + @Override + public void serialize( + final ArrayOfDoublesSketch sketch, + final JsonGenerator generator, + final SerializerProvider provider + ) throws IOException, JsonProcessingException + { + generator.writeBinary(sketch.toByteArray()); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeAggregator.java new file mode 100644 index 00000000000..ed201c5f870 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeAggregator.java @@ -0,0 +1,99 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import com.yahoo.sketches.tuple.ArrayOfDoublesSetOperationBuilder; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesUnion; +import io.druid.query.aggregation.Aggregator; +import io.druid.segment.BaseObjectColumnValueSelector; + +/** + * This aggregator merges existing sketches. + * The input column contains ArrayOfDoublesSketch. + * The output is {@link ArrayOfDoublesSketch} that is a union of the input sketches. + */ +public class ArrayOfDoublesSketchMergeAggregator implements Aggregator +{ + + private final BaseObjectColumnValueSelector selector; + private ArrayOfDoublesUnion union; + + public ArrayOfDoublesSketchMergeAggregator( + final BaseObjectColumnValueSelector selector, + final int nominalEntries, + final int numberOfValues + ) + { + this.selector = selector; + union = new ArrayOfDoublesSetOperationBuilder().setNominalEntries(nominalEntries).setNumberOfValues(numberOfValues) + .buildUnion(); + } + + /** + * This method uses synchronization because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + */ + @Override + public void aggregate() + { + final ArrayOfDoublesSketch update = selector.getObject(); + if (update == null) { + return; + } + synchronized (this) { + union.update(update); + } + } + + /** + * This method uses synchronization because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + * The returned sketch is a separate instance of ArrayOfDoublesCompactSketch + * representing the current state of the aggregation, and is not affected by consequent + * aggregate() calls + */ + @Override + public synchronized Object get() + { + return union.getResult(); + } + + @Override + public float getFloat() + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public long getLong() + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public void close() + { + union = null; + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeBufferAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeBufferAggregator.java new file mode 100644 index 00000000000..95e317ae42b --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeBufferAggregator.java @@ -0,0 +1,149 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import com.google.common.util.concurrent.Striped; +import com.yahoo.memory.WritableMemory; +import com.yahoo.sketches.tuple.ArrayOfDoublesSetOperationBuilder; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketches; +import com.yahoo.sketches.tuple.ArrayOfDoublesUnion; +import io.druid.query.aggregation.BufferAggregator; +import io.druid.query.monomorphicprocessing.RuntimeShapeInspector; +import io.druid.segment.BaseObjectColumnValueSelector; + +import java.nio.ByteBuffer; +import java.util.concurrent.locks.Lock; +import java.util.concurrent.locks.ReadWriteLock; + +/** + * This aggregator merges existing sketches. + * The input column contains ArrayOfDoublesSketch. + * The output is {@link ArrayOfDoublesSketch} that is a union of the input sketches. + */ +public class ArrayOfDoublesSketchMergeBufferAggregator implements BufferAggregator +{ + + private static final int NUM_STRIPES = 64; // for locking per buffer position + + private final BaseObjectColumnValueSelector selector; + private final int nominalEntries; + private final int numberOfValues; + private final int maxIntermediateSize; + private final Striped stripedLock = Striped.readWriteLock(NUM_STRIPES); + + public ArrayOfDoublesSketchMergeBufferAggregator( + final BaseObjectColumnValueSelector selector, + final int nominalEntries, + final int numberOfValues, + final int maxIntermediateSize + ) + { + this.selector = selector; + this.nominalEntries = nominalEntries; + this.numberOfValues = numberOfValues; + this.maxIntermediateSize = maxIntermediateSize; + } + + @Override + public void init(final ByteBuffer buf, final int position) + { + final WritableMemory mem = WritableMemory.wrap(buf); + final WritableMemory region = mem.writableRegion(position, maxIntermediateSize); + new ArrayOfDoublesSetOperationBuilder().setNominalEntries(nominalEntries) + .setNumberOfValues(numberOfValues).buildUnion(region); + } + + /** + * This method uses locks because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + */ + @Override + public void aggregate(final ByteBuffer buf, final int position) + { + final ArrayOfDoublesSketch update = selector.getObject(); + if (update == null) { + return; + } + // Wrapping memory and ArrayOfDoublesUnion is inexpensive compared to union operations. + // Maintaining a cache of wrapped objects per buffer position like in Theta sketch aggregator + // might might be considered, but it would increase complexity including relocate() support. + final WritableMemory mem = WritableMemory.wrap(buf); + final WritableMemory region = mem.writableRegion(position, maxIntermediateSize); + final Lock lock = stripedLock.getAt(ArrayOfDoublesSketchBuildBufferAggregator.lockIndex(position)).writeLock(); + lock.lock(); + try { + final ArrayOfDoublesUnion union = ArrayOfDoublesSketches.wrapUnion(region); + union.update(update); + } + finally { + lock.unlock(); + } + } + + /** + * This method uses locks because it can be used during indexing, + * and Druid can call aggregate() and get() concurrently + * https://github.com/druid-io/druid/pull/3956 + * The returned sketch is a separate instance of ArrayOfDoublesCompactSketch + * representing the current state of the aggregation, and is not affected by consequent + * aggregate() calls + */ + @Override + public Object get(final ByteBuffer buf, final int position) + { + final WritableMemory mem = WritableMemory.wrap(buf); + final WritableMemory region = mem.writableRegion(position, maxIntermediateSize); + final Lock lock = stripedLock.getAt(ArrayOfDoublesSketchBuildBufferAggregator.lockIndex(position)).readLock(); + lock.lock(); + try { + final ArrayOfDoublesUnion union = ArrayOfDoublesSketches.wrapUnion(region); + return union.getResult(); + } + finally { + lock.unlock(); + } + } + + @Override + public float getFloat(final ByteBuffer buf, final int position) + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public long getLong(final ByteBuffer buf, final int position) + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public void close() + { + } + + @Override + public void inspectRuntimeShape(final RuntimeShapeInspector inspector) + { + inspector.visit("selector", selector); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeComplexMetricSerde.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeComplexMetricSerde.java new file mode 100644 index 00000000000..06fcedf4405 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMergeComplexMetricSerde.java @@ -0,0 +1,89 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.nio.ByteBuffer; + +import io.druid.data.input.InputRow; +import io.druid.segment.GenericColumnSerializer; +import io.druid.segment.column.ColumnBuilder; +import io.druid.segment.data.GenericIndexed; +import io.druid.segment.data.ObjectStrategy; +import io.druid.segment.serde.ComplexColumnPartSupplier; +import io.druid.segment.serde.ComplexMetricExtractor; +import io.druid.segment.serde.ComplexMetricSerde; +import io.druid.segment.serde.LargeColumnSupportedComplexColumnSerializer; +import io.druid.segment.writeout.SegmentWriteOutMedium; + +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +public class ArrayOfDoublesSketchMergeComplexMetricSerde extends ComplexMetricSerde +{ + + @Override + public String getTypeName() + { + return ArrayOfDoublesSketchModule.ARRAY_OF_DOUBLES_SKETCH; + } + + @Override + public ComplexMetricExtractor getExtractor() + { + return new ComplexMetricExtractor() + { + @Override + public Class extractedClass() + { + return ArrayOfDoublesSketch.class; + } + + @Override + public Object extractValue(final InputRow inputRow, final String metricName) + { + final Object object = inputRow.getRaw(metricName); + if (object == null || object instanceof ArrayOfDoublesSketch) { + return object; + } + return ArrayOfDoublesSketchOperations.deserialize(object); + } + }; + } + + @Override + public void deserializeColumn(final ByteBuffer buffer, final ColumnBuilder builder) + { + final GenericIndexed ge = GenericIndexed.read(buffer, ArrayOfDoublesSketchObjectStrategy.STRATEGY); + builder.setComplexColumn(new ComplexColumnPartSupplier(getTypeName(), ge)); + } + + @Override + public ObjectStrategy getObjectStrategy() + { + return ArrayOfDoublesSketchObjectStrategy.STRATEGY; + } + + // support large columns + @Override + public GenericColumnSerializer getSerializer(final SegmentWriteOutMedium segmentWriteOutMedium, final String column) + { + return LargeColumnSupportedComplexColumnSerializer.create(segmentWriteOutMedium, column, this.getObjectStrategy()); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchModule.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchModule.java new file mode 100644 index 00000000000..a57868a44ef --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchModule.java @@ -0,0 +1,120 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Arrays; +import java.util.List; + +import com.fasterxml.jackson.databind.Module; +import com.fasterxml.jackson.databind.jsontype.NamedType; +import com.fasterxml.jackson.databind.module.SimpleModule; +import com.google.inject.Binder; + +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +import io.druid.initialization.DruidModule; +import io.druid.segment.serde.ComplexMetrics; + +/** + * This module is to support numeric Tuple sketches, which extend the functionality of the count-distinct + * Theta sketches by adding arrays of double values associated with unique keys. + * + * See Tuple Sketch Overview + */ +public class ArrayOfDoublesSketchModule implements DruidModule +{ + + public static final String ARRAY_OF_DOUBLES_SKETCH = "arrayOfDoublesSketch"; + + public static final String ARRAY_OF_DOUBLES_SKETCH_MERGE_AGG = "arrayOfDoublesSketchMerge"; + public static final String ARRAY_OF_DOUBLES_SKETCH_BUILD_AGG = "arrayOfDoublesSketchBuild"; + + @Override + public void configure(final Binder binder) + { + if (ComplexMetrics.getSerdeForType(ARRAY_OF_DOUBLES_SKETCH) == null) { + ComplexMetrics.registerSerde(ARRAY_OF_DOUBLES_SKETCH, new ArrayOfDoublesSketchMergeComplexMetricSerde()); + } + + if (ComplexMetrics.getSerdeForType(ARRAY_OF_DOUBLES_SKETCH_MERGE_AGG) == null) { + ComplexMetrics.registerSerde( + ARRAY_OF_DOUBLES_SKETCH_MERGE_AGG, + new ArrayOfDoublesSketchMergeComplexMetricSerde() + ); + } + + if (ComplexMetrics.getSerdeForType(ARRAY_OF_DOUBLES_SKETCH_BUILD_AGG) == null) { + ComplexMetrics.registerSerde( + ARRAY_OF_DOUBLES_SKETCH_BUILD_AGG, + new ArrayOfDoublesSketchBuildComplexMetricSerde() + ); + } + } + + @Override + public List getJacksonModules() + { + return Arrays.asList( + new SimpleModule("ArrayOfDoublesSketchModule").registerSubtypes( + new NamedType( + ArrayOfDoublesSketchAggregatorFactory.class, + ARRAY_OF_DOUBLES_SKETCH + ), + new NamedType( + ArrayOfDoublesSketchToEstimatePostAggregator.class, + "arrayOfDoublesSketchToEstimate" + ), + new NamedType( + ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator.class, + "arrayOfDoublesSketchToEstimateAndBounds" + ), + new NamedType( + ArrayOfDoublesSketchToNumEntriesPostAggregator.class, + "arrayOfDoublesSketchToNumEntries" + ), + new NamedType( + ArrayOfDoublesSketchToMeansPostAggregator.class, + "arrayOfDoublesSketchToMeans" + ), + new NamedType( + ArrayOfDoublesSketchToVariancesPostAggregator.class, + "arrayOfDoublesSketchToVariances" + ), + new NamedType( + ArrayOfDoublesSketchToQuantilesSketchPostAggregator.class, + "arrayOfDoublesSketchToQuantilesSketch" + ), + new NamedType( + ArrayOfDoublesSketchSetOpPostAggregator.class, + "arrayOfDoublesSketchSetOp" + ), + new NamedType( + ArrayOfDoublesSketchTTestPostAggregator.class, + "arrayOfDoublesSketchTTest" + ), + new NamedType( + ArrayOfDoublesSketchToStringPostAggregator.class, + "arrayOfDoublesSketchToString" + ) + ).addSerializer(ArrayOfDoublesSketch.class, new ArrayOfDoublesSketchJsonSerializer()) + ); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMultiPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMultiPostAggregator.java new file mode 100644 index 00000000000..39a736e1937 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchMultiPostAggregator.java @@ -0,0 +1,96 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.List; +import java.util.Objects; +import java.util.Set; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.google.common.collect.Sets; + +import io.druid.query.aggregation.PostAggregator; + +/** + * Base class for post aggs taking multiple sketches as input + */ +public abstract class ArrayOfDoublesSketchMultiPostAggregator extends ArrayOfDoublesSketchPostAggregator +{ + + private final List fields; + private Set dependentFields; + + @JsonCreator + public ArrayOfDoublesSketchMultiPostAggregator(final String name, final List fields) + { + super(name); + this.fields = fields; + } + + @Override + public Set getDependentFields() + { + if (dependentFields == null) { + dependentFields = Sets.newHashSet(super.getDependentFields()); + for (final PostAggregator field : fields) { + dependentFields.addAll(field.getDependentFields()); + } + } + return dependentFields; + } + + @JsonProperty + public List getFields() + { + return fields; + } + + @Override + public String toString() + { + return this.getClass().getSimpleName() + "{" + + "name='" + getName() + '\'' + + ", fields=" + fields + + "}"; + } + + @Override + public boolean equals(final Object o) + { + if (!super.equals(o)) { + return false; + } + // this check is used here instead of instanceof because this is an abstract class + // and subclasses not overriding equals should not be equal to each other + if (o == null || getClass() != o.getClass()) { + return false; + } + final ArrayOfDoublesSketchMultiPostAggregator that = (ArrayOfDoublesSketchMultiPostAggregator) o; + return fields.equals(that.fields); + } + + @Override + public int hashCode() + { + return Objects.hash(super.hashCode(), fields); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchNoOpAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchNoOpAggregator.java new file mode 100644 index 00000000000..593ab163bed --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchNoOpAggregator.java @@ -0,0 +1,65 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketchBuilder; + +import io.druid.query.aggregation.Aggregator; + +public class ArrayOfDoublesSketchNoOpAggregator implements Aggregator +{ + + private final ArrayOfDoublesSketch emptySketch; + + public ArrayOfDoublesSketchNoOpAggregator(final int numberOfValues) + { + emptySketch = new ArrayOfDoublesUpdatableSketchBuilder().setNumberOfValues(numberOfValues).build().compact(); + } + + @Override + public void aggregate() + { + } + + @Override + public Object get() + { + return emptySketch; + } + + @Override + public float getFloat() + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public long getLong() + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public void close() + { + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchNoOpBufferAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchNoOpBufferAggregator.java new file mode 100644 index 00000000000..127481043ea --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchNoOpBufferAggregator.java @@ -0,0 +1,78 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.nio.ByteBuffer; + +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketchBuilder; + +import io.druid.query.aggregation.BufferAggregator; +import io.druid.query.monomorphicprocessing.RuntimeShapeInspector; + +public class ArrayOfDoublesSketchNoOpBufferAggregator implements BufferAggregator +{ + + private final ArrayOfDoublesSketch emptySketch; + + public ArrayOfDoublesSketchNoOpBufferAggregator(final int numberOfValues) + { + emptySketch = new ArrayOfDoublesUpdatableSketchBuilder().setNumberOfValues(numberOfValues).build().compact(); + } + + @Override + public void init(final ByteBuffer buf, final int position) + { + } + + @Override + public void aggregate(final ByteBuffer buf, final int position) + { + } + + @Override + public Object get(final ByteBuffer buf, final int position) + { + return emptySketch; + } + + @Override + public float getFloat(final ByteBuffer buf, final int position) + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public long getLong(final ByteBuffer buf, final int position) + { + throw new UnsupportedOperationException("Not implemented"); + } + + @Override + public void close() + { + } + + @Override + public void inspectRuntimeShape(final RuntimeShapeInspector inspector) + { + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchObjectStrategy.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchObjectStrategy.java new file mode 100644 index 00000000000..4aadba5215a --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchObjectStrategy.java @@ -0,0 +1,65 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import io.druid.segment.data.ObjectStrategy; + +import java.nio.ByteBuffer; + +import javax.annotation.Nullable; + +import com.yahoo.memory.Memory; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketches; + +public class ArrayOfDoublesSketchObjectStrategy implements ObjectStrategy +{ + + static final ArrayOfDoublesSketchObjectStrategy STRATEGY = new ArrayOfDoublesSketchObjectStrategy(); + + @Override + public int compare(final ArrayOfDoublesSketch s1, final ArrayOfDoublesSketch s2) + { + return ArrayOfDoublesSketchAggregatorFactory.COMPARATOR.compare(s1, s2); + } + + @Override + public Class getClazz() + { + return ArrayOfDoublesSketch.class; + } + + @Override + public ArrayOfDoublesSketch fromByteBuffer(final ByteBuffer buffer, final int numBytes) + { + return ArrayOfDoublesSketches.wrapSketch(Memory.wrap(buffer).region(buffer.position(), numBytes)); + } + + @Override + @Nullable + public byte[] toBytes(@Nullable final ArrayOfDoublesSketch sketch) + { + if (sketch == null) { + return null; + } + return sketch.toByteArray(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchOperations.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchOperations.java new file mode 100644 index 00000000000..a6b68c545a6 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchOperations.java @@ -0,0 +1,133 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import com.yahoo.memory.Memory; + +import java.nio.charset.StandardCharsets; + +import org.apache.commons.codec.binary.Base64; + +import com.yahoo.sketches.tuple.ArrayOfDoublesAnotB; +import com.yahoo.sketches.tuple.ArrayOfDoublesCombiner; +import com.yahoo.sketches.tuple.ArrayOfDoublesIntersection; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketches; +import com.yahoo.sketches.tuple.ArrayOfDoublesUnion; + +import io.druid.java.util.common.IAE; +import io.druid.java.util.common.ISE; + +import com.yahoo.sketches.tuple.ArrayOfDoublesSetOperationBuilder; + +public class ArrayOfDoublesSketchOperations +{ + + public enum Operation + { + UNION { + @Override + public ArrayOfDoublesSketch apply(final int nominalEntries, final int numberOfValues, final ArrayOfDoublesSketch[] sketches) + { + final ArrayOfDoublesUnion union = new ArrayOfDoublesSetOperationBuilder().setNominalEntries(nominalEntries) + .setNumberOfValues(numberOfValues).buildUnion(); + for (final ArrayOfDoublesSketch sketch : sketches) { + union.update(sketch); + } + return union.getResult(); + } + }, + INTERSECT { + @Override + public ArrayOfDoublesSketch apply(final int nominalEntries, final int numberOfValues, final ArrayOfDoublesSketch[] sketches) + { + final ArrayOfDoublesIntersection intersection = new ArrayOfDoublesSetOperationBuilder() + .setNominalEntries(nominalEntries).setNumberOfValues(numberOfValues).buildIntersection(); + for (final ArrayOfDoublesSketch sketch : sketches) { + intersection.update(sketch, COMBINER); + } + return intersection.getResult(); + } + }, + NOT { + @Override + public ArrayOfDoublesSketch apply(final int nominalEntries, final int numberOfValues, final ArrayOfDoublesSketch[] sketches) + { + if (sketches.length < 1) { + throw new IAE("A-Not-B requires at least 1 sketch"); + } + + if (sketches.length == 1) { + return sketches[0]; + } + + ArrayOfDoublesSketch result = sketches[0]; + for (int i = 1; i < sketches.length; i++) { + final ArrayOfDoublesAnotB aNotB = new ArrayOfDoublesSetOperationBuilder().setNumberOfValues(numberOfValues) + .buildAnotB(); + aNotB.update(result, sketches[i]); + result = aNotB.getResult(); + } + return result; + } + }; + + public abstract ArrayOfDoublesSketch apply(int nominalEntries, int numberOfValues, ArrayOfDoublesSketch[] sketches); + } + + // This is how to combine values for sketch intersection. + // Might not fit all use cases. + private static ArrayOfDoublesCombiner COMBINER = new ArrayOfDoublesCombiner() + { + @Override + public double[] combine(final double[] a, final double[] b) + { + final double[] result = new double[a.length]; + for (int i = 0; i < a.length; i++) { + result[i] = a[i] + b[i]; + } + return result; + } + }; + + public static ArrayOfDoublesSketch deserialize(final Object serializedSketch) + { + if (serializedSketch instanceof String) { + return deserializeFromBase64EncodedString((String) serializedSketch); + } else if (serializedSketch instanceof byte[]) { + return deserializeFromByteArray((byte[]) serializedSketch); + } else if (serializedSketch instanceof ArrayOfDoublesSketch) { + return (ArrayOfDoublesSketch) serializedSketch; + } + throw new ISE("Object is not of a type that can deserialize to sketch: %s", serializedSketch.getClass()); + } + + public static ArrayOfDoublesSketch deserializeFromBase64EncodedString(final String str) + { + return deserializeFromByteArray(Base64.decodeBase64(str.getBytes(StandardCharsets.UTF_8))); + } + + public static ArrayOfDoublesSketch deserializeFromByteArray(final byte[] data) + { + final Memory mem = Memory.wrap(data); + return ArrayOfDoublesSketches.wrapSketch(mem); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchPostAggregator.java new file mode 100644 index 00000000000..a9bcf495d30 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchPostAggregator.java @@ -0,0 +1,83 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Collections; +import java.util.Map; +import java.util.Set; + +import com.fasterxml.jackson.annotation.JsonProperty; +import com.google.common.base.Preconditions; + +import io.druid.query.aggregation.AggregatorFactory; +import io.druid.query.aggregation.PostAggregator; + +/** + * Base class for all post aggs + */ +public abstract class ArrayOfDoublesSketchPostAggregator implements PostAggregator +{ + + private final String name; + + public ArrayOfDoublesSketchPostAggregator(final String name) + { + this.name = Preconditions.checkNotNull(name, "name is null"); + } + + @Override + @JsonProperty + public String getName() + { + return name; + } + + @Override + public Set getDependentFields() + { + return Collections.emptySet(); + } + + @Override + public boolean equals(final Object o) + { + if (this == o) { + return true; + } + if (!(o instanceof ArrayOfDoublesSketchPostAggregator)) { + return false; + } + final ArrayOfDoublesSketchPostAggregator that = (ArrayOfDoublesSketchPostAggregator) o; + return name.equals(that.getName()); + } + + @Override + public int hashCode() + { + return name.hashCode(); + } + + @Override + public PostAggregator decorate(final Map map) + { + return this; + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchSetOpPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchSetOpPostAggregator.java new file mode 100644 index 00000000000..584c14ba6b8 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchSetOpPostAggregator.java @@ -0,0 +1,152 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Comparator; +import java.util.List; +import java.util.Map; +import java.util.Objects; + +import javax.annotation.Nullable; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.Util; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +import io.druid.java.util.common.IAE; +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Returns a result of a specified set operation on the given array of sketches. Supported operations are: + * union, intersection and set difference (UNION, INTERSECT, NOT). + */ +public class ArrayOfDoublesSketchSetOpPostAggregator extends ArrayOfDoublesSketchMultiPostAggregator +{ + + private final ArrayOfDoublesSketchOperations.Operation operation; + private final int nominalEntries; + private final int numberOfValues; + + @JsonCreator + public ArrayOfDoublesSketchSetOpPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("operation") final String operation, + @JsonProperty("nominalEntries") @Nullable final Integer nominalEntries, + @JsonProperty("numberOfValues") @Nullable final Integer numberOfValues, + @JsonProperty("fields") List fields + ) + { + super(name, fields); + this.operation = ArrayOfDoublesSketchOperations.Operation.valueOf(operation); + this.nominalEntries = nominalEntries == null ? Util.DEFAULT_NOMINAL_ENTRIES : nominalEntries; + this.numberOfValues = numberOfValues == null ? 1 : numberOfValues; + Util.checkIfPowerOf2(this.nominalEntries, "size"); + + if (fields.size() <= 1) { + throw new IAE("Illegal number of fields[%d], must be > 1", fields.size()); + } + } + + @Override + public Comparator getComparator() + { + return ArrayOfDoublesSketchAggregatorFactory.COMPARATOR; + } + + @Override + public ArrayOfDoublesSketch compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch[] sketches = new ArrayOfDoublesSketch[getFields().size()]; + for (int i = 0; i < sketches.length; i++) { + sketches[i] = (ArrayOfDoublesSketch) getFields().get(i).compute(combinedAggregators); + } + return operation.apply(nominalEntries, numberOfValues, sketches); + } + + @JsonProperty + public String getOperation() + { + return operation.toString(); + } + + @JsonProperty + public int getNominalEntries() + { + return nominalEntries; + } + + @JsonProperty + public int getNumberOfValues() + { + return numberOfValues; + } + + @Override + public String toString() + { + return this.getClass().getSimpleName() + "{" + + "name='" + getName() + '\'' + + ", fields=" + getFields() + + ", operation=" + operation + + ", nominalEntries=" + nominalEntries + + ", numberOfValues=" + numberOfValues + + "}"; + } + + @Override + public boolean equals(final Object o) + { + if (!super.equals(o)) { + return false; + } + if (!(o instanceof ArrayOfDoublesSketchSetOpPostAggregator)) { + return false; + } + final ArrayOfDoublesSketchSetOpPostAggregator that = (ArrayOfDoublesSketchSetOpPostAggregator) o; + if (nominalEntries != that.nominalEntries) { + return false; + } + if (numberOfValues != that.numberOfValues) { + return false; + } + return operation.equals(that.operation); + } + + @Override + public int hashCode() + { + return Objects.hash(super.hashCode(), operation, nominalEntries, numberOfValues); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_SET_OP_CACHE_TYPE_ID) + .appendCacheables(getFields()) + .appendInt(nominalEntries) + .appendInt(numberOfValues) + .appendString(operation.toString()) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchTTestPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchTTestPostAggregator.java new file mode 100644 index 00000000000..30f748c9c9f --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchTTestPostAggregator.java @@ -0,0 +1,113 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Arrays; +import java.util.Comparator; +import java.util.List; +import java.util.Map; + +import org.apache.commons.math3.stat.descriptive.SummaryStatistics; +import org.apache.commons.math3.stat.inference.TTest; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketchIterator; + +import io.druid.java.util.common.IAE; +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Performs Student's t-test and returns a list of p-values given two instances of {@link ArrayOfDoublesSketch}. + * The result will be N double values, where N is the number of double values kept in the sketch per key. + * See Student's t-test + */ +public class ArrayOfDoublesSketchTTestPostAggregator extends ArrayOfDoublesSketchMultiPostAggregator +{ + + @JsonCreator + public ArrayOfDoublesSketchTTestPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("fields") List fields + ) + { + super(name, fields); + if (fields.size() != 2) { + throw new IAE("Illegal number of fields[%d], must be 2", fields.size()); + } + } + + @Override + public Comparator getComparator() + { + throw new IAE("Comparing arrays of p values is not supported"); + } + + @Override + public double[] compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch1 = (ArrayOfDoublesSketch) getFields().get(0).compute(combinedAggregators); + final ArrayOfDoublesSketch sketch2 = (ArrayOfDoublesSketch) getFields().get(1).compute(combinedAggregators); + if (sketch1.getNumValues() != sketch2.getNumValues()) { + throw new IAE( + "Sketches have different number of values: %d and %d", + sketch1.getNumValues(), + sketch2.getNumValues() + ); + } + + final SummaryStatistics[] stats1 = getStats(sketch1); + final SummaryStatistics[] stats2 = getStats(sketch2); + + final int numberOfValues = sketch1.getNumValues(); + final double[] pValues = new double[numberOfValues]; + final TTest test = new TTest(); + for (int i = 0; i < pValues.length; i++) { + pValues[i] = test.tTest(stats1[i], stats2[i]); + } + return pValues; + } + + private static SummaryStatistics[] getStats(final ArrayOfDoublesSketch sketch) + { + final SummaryStatistics[] stats = new SummaryStatistics[sketch.getNumValues()]; + Arrays.setAll(stats, i -> new SummaryStatistics()); + final ArrayOfDoublesSketchIterator it = sketch.iterator(); + while (it.next()) { + final double[] values = it.getValues(); + for (int i = 0; i < values.length; i++) { + stats[i].addValue(values[i]); + } + } + return stats; + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_T_TEST_CACHE_TYPE_ID) + .appendCacheables(getFields()) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator.java new file mode 100644 index 00000000000..9d5d29798d2 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator.java @@ -0,0 +1,120 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Comparator; +import java.util.Map; +import java.util.Objects; + +import javax.annotation.Nullable; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +import io.druid.java.util.common.IAE; +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Returns a distinct count estimate and error bounds from a given {@link ArrayOfDoublesSketch}." + * The result will be three double values: estimate of the number of distinct keys, lower bound and upper bound. + * The bounds are provided at the given number of standard deviations (optional, defaults to 1). + * This must be an integer value of 1, 2 or 3 corresponding to approximately 68.3%, 95.4% and 99.7% + * confidence intervals. + */ +public class ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator extends ArrayOfDoublesSketchUnaryPostAggregator +{ + + private final int numStdDevs; + + @JsonCreator + public ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("field") final PostAggregator field, + @JsonProperty("numStdDevs") @Nullable final Integer numStdDevs + ) + { + super(name, field); + this.numStdDevs = numStdDevs == null ? 1 : numStdDevs; + } + + @JsonProperty + public int getNumStdDevs() + { + return numStdDevs; + } + + @Override + public double[] compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch = (ArrayOfDoublesSketch) getField().compute(combinedAggregators); + return new double[] {sketch.getEstimate(), sketch.getLowerBound(numStdDevs), sketch.getUpperBound(numStdDevs)}; + } + + @Override + public Comparator getComparator() + { + throw new IAE("Comparing arrays of estimates and error bounds is not supported"); + } + + @Override + public String toString() + { + return this.getClass().getSimpleName() + "{" + + "name='" + getName() + '\'' + + ", field=" + getField() + + ", numStdDevs=" + numStdDevs + + "}"; + } + + @Override + public boolean equals(final Object o) + { + if (!super.equals(o)) { + return false; + } + if (!(o instanceof ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator)) { + return false; + } + final ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator that = (ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator) o; + if (numStdDevs != that.numStdDevs) { + return false; + } + return true; + } + + @Override + public int hashCode() + { + return Objects.hash(super.hashCode(), numStdDevs); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_TO_ESTIMATE_AND_BOUNDS_CACHE_TYPE_ID) + .appendCacheable(getField()) + .appendInt(numStdDevs) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimatePostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimatePostAggregator.java new file mode 100644 index 00000000000..fd20c710ad9 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimatePostAggregator.java @@ -0,0 +1,69 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Comparator; +import java.util.Map; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Returns a distinct count estimate from a given {@link ArrayOfDoublesSketch}. + */ +public class ArrayOfDoublesSketchToEstimatePostAggregator extends ArrayOfDoublesSketchUnaryPostAggregator +{ + + @JsonCreator + public ArrayOfDoublesSketchToEstimatePostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("field") final PostAggregator field + ) + { + super(name, field); + } + + @Override + public Double compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch = (ArrayOfDoublesSketch) getField().compute(combinedAggregators); + return sketch.getEstimate(); + } + + @Override + public Comparator getComparator() + { + return Comparator.naturalOrder(); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_TO_ESTIMATE_CACHE_TYPE_ID) + .appendCacheable(getField()) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToMeansPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToMeansPostAggregator.java new file mode 100644 index 00000000000..dc3f778dffc --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToMeansPostAggregator.java @@ -0,0 +1,86 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Arrays; +import java.util.Comparator; +import java.util.Map; + +import org.apache.commons.math3.stat.descriptive.SummaryStatistics; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketchIterator; + +import io.druid.java.util.common.IAE; +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Returns a list of mean values from a given {@link ArrayOfDoublesSketch}. + * The result will be N double values, where N is the number of double values kept in the sketch per key. + */ +public class ArrayOfDoublesSketchToMeansPostAggregator extends ArrayOfDoublesSketchUnaryPostAggregator +{ + + @JsonCreator + public ArrayOfDoublesSketchToMeansPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("field") final PostAggregator field + ) + { + super(name, field); + } + + @Override + public double[] compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch = (ArrayOfDoublesSketch) getField().compute(combinedAggregators); + final SummaryStatistics[] stats = new SummaryStatistics[sketch.getNumValues()]; + Arrays.setAll(stats, i -> new SummaryStatistics()); + final ArrayOfDoublesSketchIterator it = sketch.iterator(); + while (it.next()) { + final double[] values = it.getValues(); + for (int i = 0; i < values.length; i++) { + stats[i].addValue(values[i]); + } + } + final double[] means = new double[sketch.getNumValues()]; + Arrays.setAll(means, i -> stats[i].getMean()); + return means; + } + + @Override + public Comparator getComparator() + { + throw new IAE("Comparing arrays of mean values is not supported"); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_TO_MEANS_CACHE_TYPE_ID) + .appendCacheable(getField()) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToNumEntriesPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToNumEntriesPostAggregator.java new file mode 100644 index 00000000000..0955b44d843 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToNumEntriesPostAggregator.java @@ -0,0 +1,69 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Comparator; +import java.util.Map; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Returns the number of retained entries from a given {@link ArrayOfDoublesSketch}. + */ +public class ArrayOfDoublesSketchToNumEntriesPostAggregator extends ArrayOfDoublesSketchUnaryPostAggregator +{ + + @JsonCreator + public ArrayOfDoublesSketchToNumEntriesPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("field") final PostAggregator field + ) + { + super(name, field); + } + + @Override + public Integer compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch = (ArrayOfDoublesSketch) getField().compute(combinedAggregators); + return sketch.getRetainedEntries(); + } + + @Override + public Comparator getComparator() + { + return Comparator.naturalOrder(); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_TO_NUM_ENTRIES_CACHE_TYPE_ID) + .appendCacheable(getField()) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToQuantilesSketchPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToQuantilesSketchPostAggregator.java new file mode 100644 index 00000000000..8aabbcfbfc3 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToQuantilesSketchPostAggregator.java @@ -0,0 +1,146 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Comparator; +import java.util.Map; +import java.util.Objects; + +import javax.annotation.Nullable; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; + +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.datasketches.quantiles.DoublesSketchAggregatorFactory; +import io.druid.query.cache.CacheKeyBuilder; + +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketchIterator; +import com.yahoo.sketches.quantiles.DoublesSketch; +import com.yahoo.sketches.quantiles.UpdateDoublesSketch; + +/** + * Returns a quanitles {@link DoublesSketch} constructed from a given column of double values from a given + * {@link ArrayOfDoublesSketch} using parameter k that determines the accuracy and size of the quantiles sketch. + * The column number is optional (the default is 1). + * The parameter k is optional (the default is defined in the sketch library). + * The result is a quantiles sketch. + * See Quantiles Sketch Overview + */ +public class ArrayOfDoublesSketchToQuantilesSketchPostAggregator extends ArrayOfDoublesSketchUnaryPostAggregator +{ + + private static final int DEFAULT_QUANTILES_SKETCH_SIZE = 128; + + private final int column; + private final int k; + + @JsonCreator + public ArrayOfDoublesSketchToQuantilesSketchPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("field") final PostAggregator field, + @JsonProperty("column") @Nullable final Integer column, + @JsonProperty("k") @Nullable final Integer k + ) + { + super(name, field); + this.column = column == null ? 1 : column; + this.k = k == null ? DEFAULT_QUANTILES_SKETCH_SIZE : k; + } + + @Override + public Comparator getComparator() + { + return DoublesSketchAggregatorFactory.COMPARATOR; + } + + @Override + public DoublesSketch compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch = (ArrayOfDoublesSketch) getField().compute(combinedAggregators); + final UpdateDoublesSketch qs = UpdateDoublesSketch.builder().setK(k).build(); + final ArrayOfDoublesSketchIterator it = sketch.iterator(); + while (it.next()) { + qs.update(it.getValues()[column - 1]); // convert 1-based column number to zero-based index + } + return qs; + } + + @JsonProperty + public int getColumn() + { + return column; + } + + @JsonProperty + public int getK() + { + return k; + } + + @Override + public String toString() + { + return this.getClass().getSimpleName() + "{" + + "name='" + getName() + '\'' + + ", field=" + getField() + + ", column=" + column + + ", k=" + k + + "}"; + } + + @Override + public boolean equals(final Object o) + { + if (!super.equals(o)) { + return false; + } + if (!(o instanceof ArrayOfDoublesSketchToQuantilesSketchPostAggregator)) { + return false; + } + final ArrayOfDoublesSketchToQuantilesSketchPostAggregator that = (ArrayOfDoublesSketchToQuantilesSketchPostAggregator) o; + if (column != that.column) { + return false; + } + if (k != that.k) { + return false; + } + return true; + } + + @Override + public int hashCode() + { + return Objects.hash(super.hashCode(), column, k); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_TO_QUANTILES_SKETCH_CACHE_TYPE_ID) + .appendCacheable(getField()) + .appendInt(column) + .appendInt(k) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToStringPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToStringPostAggregator.java new file mode 100644 index 00000000000..8350f8e4974 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToStringPostAggregator.java @@ -0,0 +1,72 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Comparator; +import java.util.Map; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; + +import io.druid.java.util.common.IAE; +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Returns a human-readable summary of a given {@link ArrayOfDoublesSketch}. + * This is a string returned by toString() method of the sketch. + * This can be useful for debugging. + */ +public class ArrayOfDoublesSketchToStringPostAggregator extends ArrayOfDoublesSketchUnaryPostAggregator +{ + + @JsonCreator + public ArrayOfDoublesSketchToStringPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("field") final PostAggregator field + ) + { + super(name, field); + } + + @Override + public String compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch = (ArrayOfDoublesSketch) getField().compute(combinedAggregators); + return sketch.toString(); + } + + @Override + public Comparator getComparator() + { + throw new IAE("Comparing sketch summaries is not supported"); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_TO_STRING_CACHE_TYPE_ID) + .appendCacheable(getField()) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToVariancesPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToVariancesPostAggregator.java new file mode 100644 index 00000000000..8b4403542b7 --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToVariancesPostAggregator.java @@ -0,0 +1,86 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Arrays; +import java.util.Comparator; +import java.util.Map; + +import org.apache.commons.math3.stat.descriptive.SummaryStatistics; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesSketchIterator; + +import io.druid.java.util.common.IAE; +import io.druid.query.aggregation.AggregatorUtil; +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.cache.CacheKeyBuilder; + +/** + * Returns a list of variance values from a given {@link ArrayOfDoublesSketch}. + * The result will be N double values, where N is the number of double values kept in the sketch per key. + */ +public class ArrayOfDoublesSketchToVariancesPostAggregator extends ArrayOfDoublesSketchUnaryPostAggregator +{ + + @JsonCreator + public ArrayOfDoublesSketchToVariancesPostAggregator( + @JsonProperty("name") final String name, + @JsonProperty("field") final PostAggregator field + ) + { + super(name, field); + } + + @Override + public double[] compute(final Map combinedAggregators) + { + final ArrayOfDoublesSketch sketch = (ArrayOfDoublesSketch) getField().compute(combinedAggregators); + final SummaryStatistics[] stats = new SummaryStatistics[sketch.getNumValues()]; + Arrays.setAll(stats, i -> new SummaryStatistics()); + final ArrayOfDoublesSketchIterator it = sketch.iterator(); + while (it.next()) { + final double[] values = it.getValues(); + for (int i = 0; i < values.length; i++) { + stats[i].addValue(values[i]); + } + } + final double[] variances = new double[sketch.getNumValues()]; + Arrays.setAll(variances, i -> stats[i].getVariance()); + return variances; + } + + @Override + public Comparator getComparator() + { + throw new IAE("Comparing arrays of variance values is not supported"); + } + + @Override + public byte[] getCacheKey() + { + return new CacheKeyBuilder(AggregatorUtil.ARRAY_OF_DOUBLES_SKETCH_TO_VARIANCES_CACHE_TYPE_ID) + .appendCacheable(getField()) + .build(); + } + +} diff --git a/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchUnaryPostAggregator.java b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchUnaryPostAggregator.java new file mode 100644 index 00000000000..3db041feb9a --- /dev/null +++ b/extensions-core/datasketches/src/main/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchUnaryPostAggregator.java @@ -0,0 +1,97 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Objects; +import java.util.Set; + +import com.fasterxml.jackson.annotation.JsonCreator; +import com.fasterxml.jackson.annotation.JsonProperty; +import com.google.common.base.Preconditions; +import com.google.common.collect.Sets; + +import io.druid.query.aggregation.PostAggregator; + +/** + * Base class for post aggs taking one sketch as input + */ +public abstract class ArrayOfDoublesSketchUnaryPostAggregator extends ArrayOfDoublesSketchPostAggregator +{ + + private final PostAggregator field; + private Set dependentFields; + + @JsonCreator + public ArrayOfDoublesSketchUnaryPostAggregator( + final String name, + final PostAggregator field + ) + { + super(name); + this.field = Preconditions.checkNotNull(field, "field is null"); + } + + @JsonProperty + public PostAggregator getField() + { + return field; + } + + @Override + public Set getDependentFields() + { + if (dependentFields == null) { + dependentFields = Sets.newHashSet(super.getDependentFields()); + dependentFields.addAll(field.getDependentFields()); + } + return dependentFields; + } + + @Override + public boolean equals(final Object o) + { + if (!super.equals(o)) { + return false; + } + // this check is used here instead of instanceof because this is an abstract class + // and subclasses not overriding equals should not be equal to each other + if (o == null || getClass() != o.getClass()) { + return false; + } + final ArrayOfDoublesSketchUnaryPostAggregator that = (ArrayOfDoublesSketchUnaryPostAggregator) o; + return field.equals(that.getField()); + } + + @Override + public int hashCode() + { + return Objects.hash(super.hashCode(), field); + } + + @Override + public String toString() + { + return this.getClass().getSimpleName() + "{" + + "name='" + getName() + '\'' + + ", field=" + getField() + + "}"; + } + +} diff --git a/extensions-core/datasketches/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions-core/datasketches/src/main/resources/META-INF/services/io.druid.initialization.DruidModule index dfc5b69e00f..319e10e0b88 100644 --- a/extensions-core/datasketches/src/main/resources/META-INF/services/io.druid.initialization.DruidModule +++ b/extensions-core/datasketches/src/main/resources/META-INF/services/io.druid.initialization.DruidModule @@ -1,3 +1,4 @@ io.druid.query.aggregation.datasketches.theta.SketchModule io.druid.query.aggregation.datasketches.theta.oldapi.OldApiSketchModule io.druid.query.aggregation.datasketches.quantiles.DoublesSketchModule +io.druid.query.aggregation.datasketches.tuple.ArrayOfDoublesSketchModule diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregationTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregationTest.java new file mode 100644 index 00000000000..838f5d428f9 --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregationTest.java @@ -0,0 +1,565 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import com.google.common.collect.Lists; +import com.yahoo.sketches.quantiles.DoublesSketch; +import io.druid.data.input.Row; +import io.druid.initialization.DruidModule; +import io.druid.java.util.common.granularity.Granularities; +import io.druid.java.util.common.guava.Sequence; +import io.druid.query.aggregation.AggregationTestHelper; +import io.druid.query.groupby.GroupByQueryConfig; +import io.druid.query.groupby.GroupByQueryRunnerTest; +import org.junit.Assert; +import org.junit.Rule; +import org.junit.Test; +import org.junit.rules.TemporaryFolder; +import org.junit.runner.RunWith; +import org.junit.runners.Parameterized; + +import java.io.File; +import java.util.Collection; +import java.util.List; + +@RunWith(Parameterized.class) +public class ArrayOfDoublesSketchAggregationTest +{ + + private final AggregationTestHelper helper; + + @Rule + public final TemporaryFolder tempFolder = new TemporaryFolder(); + + public ArrayOfDoublesSketchAggregationTest(final GroupByQueryConfig config) + { + DruidModule module = new ArrayOfDoublesSketchModule(); + module.configure(null); + helper = AggregationTestHelper.createGroupByQueryAggregationTestHelper( + module.getJacksonModules(), config, tempFolder); + } + + @Parameterized.Parameters(name = "{0}") + public static Collection constructorFeeder() + { + final List constructors = Lists.newArrayList(); + for (GroupByQueryConfig config : GroupByQueryRunnerTest.testConfigs()) { + constructors.add(new Object[] {config}); + } + return constructors; + } + + @Test + public void ingestingSketches() throws Exception + { + Sequence seq = helper.createIndexAndRunQueryOnSegment( + new File(this.getClass().getClassLoader().getResource("tuple/array_of_doubles_sketch_data.tsv").getFile()), + String.join("\n", + "{", + " \"type\": \"string\",", + " \"parseSpec\": {", + " \"format\": \"tsv\",", + " \"timestampSpec\": {\"column\": \"timestamp\", \"format\": \"yyyyMMddHH\"},", + " \"dimensionsSpec\": {", + " \"dimensions\": [\"product\"],", + " \"dimensionExclusions\": [],", + " \"spatialDimensions\": []", + " },", + " \"columns\": [\"timestamp\", \"product\", \"sketch\"]", + " }", + "}"), + String.join("\n", + "[", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"sketch\", \"nominalEntries\": 1024},", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"non_existing_sketch\", \"fieldName\": \"non_existing_sketch\"}", + "]"), + 0, // minTimestamp + Granularities.NONE, + 10, // maxRowCount + String.join("\n", + "{", + " \"queryType\": \"groupBy\",", + " \"dataSource\": \"test_datasource\",", + " \"granularity\": \"ALL\",", + " \"dimensions\": [],", + " \"aggregations\": [", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"sketch\", \"nominalEntries\": 1024},", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"non_existing_sketch\", \"fieldName\": \"non_existing_sketch\"}", + " ],", + " \"postAggregations\": [", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"estimate\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToQuantilesSketch\", \"name\": \"quantiles-sketch\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"union\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"union\",", + " \"operation\": \"UNION\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"intersection\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"intersection\",", + " \"operation\": \"INTERSECT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"anotb\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"anotb\",", + " \"operation\": \"NOT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }}", + " ],", + " \"intervals\": [\"2015-01-01T00:00:00.000Z/2015-01-31T00:00:00.000Z\"]", + "}")); + List results = seq.toList(); + Assert.assertEquals(1, results.size()); + Row row = results.get(0); + Assert.assertEquals(0, (double) row.getMetric("non_existing_sketch"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("sketch"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("estimate"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("union"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("intersection"), 0); + Assert.assertEquals(0, (double) row.getRaw("anotb"), 0); + + Object obj = row.getRaw("quantiles-sketch"); + Assert.assertTrue(obj instanceof DoublesSketch); + DoublesSketch ds = (DoublesSketch) obj; + Assert.assertEquals(40, ds.getN()); + Assert.assertEquals(1.0, ds.getMinValue(), 0); + Assert.assertEquals(1.0, ds.getMaxValue(), 0); + } + + @Test + public void ingestingSketchesTwoValues() throws Exception + { + Sequence seq = helper.createIndexAndRunQueryOnSegment( + new File(this.getClass().getClassLoader().getResource("tuple/array_of_doubles_sketch_data_two_values.tsv") + .getFile()), + String.join("\n", + "{", + " \"type\": \"string\",", + " \"parseSpec\": {", + " \"format\": \"tsv\",", + " \"timestampSpec\": {\"column\": \"timestamp\", \"format\": \"yyyyMMddHH\"},", + " \"dimensionsSpec\": {", + " \"dimensions\": [\"product\"],", + " \"dimensionExclusions\": [],", + " \"spatialDimensions\": []", + " },", + " \"columns\": [\"timestamp\", \"product\", \"sketch\"]", + " }", + "}"), + String.join("\n", + "[", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"sketch\", \"nominalEntries\": 1024, \"numberOfValues\": 2}", + "]"), + 0, // minTimestamp + Granularities.NONE, + 10, // maxRowCount + String.join("\n", + "{", + " \"queryType\": \"groupBy\",", + " \"dataSource\": \"test_datasource\",", + " \"granularity\": \"ALL\",", + " \"dimensions\": [],", + " \"aggregations\": [", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"sketch\", \"nominalEntries\": 1024, \"numberOfValues\": 2}", + " ],", + " \"postAggregations\": [", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"estimate\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToQuantilesSketch\", \"name\": \"quantiles-sketch\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"union\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"union\",", + " \"operation\": \"UNION\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"intersection\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"intersection\",", + " \"operation\": \"INTERSECT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"anotb\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"anotb\",", + " \"operation\": \"NOT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {", + " \"type\": \"arrayOfDoublesSketchToMeans\",", + " \"name\": \"means\",", + " \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}", + " }", + " ],", + " \"intervals\": [\"2015-01-01T00:00:00.000Z/2015-01-31T00:00:00.000Z\"]", + "}")); + List results = seq.toList(); + Assert.assertEquals(1, results.size()); + Row row = results.get(0); + Assert.assertEquals(40.0, (double) row.getRaw("sketch"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("estimate"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("union"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("intersection"), 0); + Assert.assertEquals(0, (double) row.getRaw("anotb"), 0); + + Object meansObj = row.getRaw("means"); + Assert.assertTrue(meansObj instanceof double[]); + double[] means = (double[]) meansObj; + Assert.assertEquals(2, means.length); + Assert.assertEquals(1.0, means[0], 0); + Assert.assertEquals(2.0, means[1], 0); + + Object quantilesObj = row.getRaw("quantiles-sketch"); + Assert.assertTrue(quantilesObj instanceof DoublesSketch); + DoublesSketch ds = (DoublesSketch) quantilesObj; + Assert.assertEquals(40, ds.getN()); + Assert.assertEquals(1.0, ds.getMinValue(), 0); + Assert.assertEquals(1.0, ds.getMaxValue(), 0); + } + + @Test + public void buildingSketchesAtIngestionTime() throws Exception + { + Sequence seq = helper.createIndexAndRunQueryOnSegment( + new File(this.getClass().getClassLoader().getResource("tuple/array_of_doubles_build_data.tsv").getFile()), + String.join("\n", + "{", + " \"type\": \"string\",", + " \"parseSpec\": {", + " \"format\": \"tsv\",", + " \"timestampSpec\": {\"column\": \"timestamp\", \"format\": \"yyyyMMddHH\"},", + " \"dimensionsSpec\": {", + " \"dimensions\": [\"product\"],", + " \"dimensionExclusions\": [],", + " \"spatialDimensions\": []", + " },", + " \"columns\": [\"timestamp\", \"product\", \"key\", \"value\"]", + " }", + "}"), + String.join("\n", + "[", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"key\", \"metricColumns\": [\"value\"], \"nominalEntries\": 1024}", + "]"), + 0, // minTimestamp + Granularities.NONE, + 10, // maxRowCount + String.join("\n", + "{", + " \"queryType\": \"groupBy\",", + " \"dataSource\": \"test_datasource\",", + " \"granularity\": \"ALL\",", + " \"dimensions\": [],", + " \"aggregations\": [", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"sketch\", \"size\": 1024}", + " ],", + " \"postAggregations\": [", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"estimate\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToQuantilesSketch\", \"name\": \"quantiles-sketch\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"union\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"union\",", + " \"operation\": \"UNION\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"intersection\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"intersection\",", + " \"operation\": \"INTERSECT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"anotb\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"anotb\",", + " \"operation\": \"NOT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }}", + " ],", + " \"intervals\": [\"2015-01-01T00:00:00.000Z/2015-01-31T00:00:00.000Z\"]", + "}")); + List results = seq.toList(); + Assert.assertEquals(1, results.size()); + Row row = results.get(0); + Assert.assertEquals(40.0, (double) row.getRaw("sketch"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("estimate"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("union"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("intersection"), 0); + Assert.assertEquals(0, (double) row.getRaw("anotb"), 0); + + Object obj = row.getRaw("quantiles-sketch"); + Assert.assertTrue(obj instanceof DoublesSketch); + DoublesSketch ds = (DoublesSketch) obj; + Assert.assertEquals(40, ds.getN()); + Assert.assertEquals(1.0, ds.getMinValue(), 0); + Assert.assertEquals(1.0, ds.getMaxValue(), 0); + } + + @Test + public void buildingSketchesAtIngestionTimeTwoValues() throws Exception + { + Sequence seq = helper.createIndexAndRunQueryOnSegment( + new File( + this.getClass().getClassLoader().getResource("tuple/array_of_doubles_build_data_two_values.tsv").getFile()), + String.join("\n", + "{", + " \"type\": \"string\",", + " \"parseSpec\": {", + " \"format\": \"tsv\",", + " \"timestampSpec\": {\"column\": \"timestamp\", \"format\": \"yyyyMMddHH\"},", + " \"dimensionsSpec\": {", + " \"dimensions\": [\"product\"],", + " \"dimensionExclusions\": [],", + " \"spatialDimensions\": []", + " },", + " \"columns\": [\"timestamp\", \"product\", \"key\", \"value1\", \"value2\"]", + " }", + "}"), + String.join("\n", + "[", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"key\", \"metricColumns\": [ \"value1\", \"value2\" ], \"nominalEntries\": 1024}", + "]"), + 0, // minTimestamp + Granularities.NONE, + 10, // maxRowCount + String.join("\n", + "{", + " \"queryType\": \"groupBy\",", + " \"dataSource\": \"test_datasource\",", + " \"granularity\": \"ALL\",", + " \"dimensions\": [],", + " \"aggregations\": [", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"sketch\", \"nominalEntries\": 1024, \"numberOfValues\": 2}", + " ],", + " \"postAggregations\": [", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"estimate\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToQuantilesSketch\", \"name\": \"quantiles-sketch\", \"column\": 2, \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"union\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"union\",", + " \"operation\": \"UNION\",", + " \"nominalEntries\": 1024,", + " \"numberOfValues\": 2,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"intersection\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"intersection\",", + " \"operation\": \"INTERSECT\",", + " \"nominalEntries\": 1024,", + " \"numberOfValues\": 2,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"anotb\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"anotb\",", + " \"operation\": \"NOT\",", + " \"nominalEntries\": 1024,", + " \"numberOfValues\": 2,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {", + " \"type\": \"arrayOfDoublesSketchToMeans\",", + " \"name\": \"means\",", + " \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}", + " }", + " ],", + " \"intervals\": [\"2015-01-01T00:00:00.000Z/2015-01-31T00:00:00.000Z\"]", + "}")); + List results = seq.toList(); + Assert.assertEquals(1, results.size()); + Row row = results.get(0); + Assert.assertEquals(40.0, (double) row.getRaw("sketch"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("estimate"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("union"), 0); + Assert.assertEquals(40.0, (double) row.getRaw("intersection"), 0); + Assert.assertEquals(0, (double) row.getRaw("anotb"), 0); + + Object meansObj = row.getRaw("means"); + Assert.assertTrue(meansObj instanceof double[]); + double[] means = (double[]) meansObj; + Assert.assertEquals(2, means.length); + Assert.assertEquals(1.0, means[0], 0); + Assert.assertEquals(2.0, means[1], 0); + + Object obj = row.getRaw("quantiles-sketch"); + Assert.assertTrue(obj instanceof DoublesSketch); + DoublesSketch ds = (DoublesSketch) obj; + Assert.assertEquals(40, ds.getN()); + Assert.assertEquals(2.0, ds.getMinValue(), 0); + Assert.assertEquals(2.0, ds.getMaxValue(), 0); + } + + @Test + public void buildingSketchesAtQueryTime() throws Exception + { + Sequence seq = helper.createIndexAndRunQueryOnSegment( + new File(this.getClass().getClassLoader().getResource("tuple/array_of_doubles_build_data.tsv").getFile()), + String.join("\n", + "{", + " \"type\": \"string\",", + " \"parseSpec\": {", + " \"format\": \"tsv\",", + " \"timestampSpec\": {\"column\": \"timestamp\", \"format\": \"yyyyMMddHH\"},", + " \"dimensionsSpec\": {", + " \"dimensions\": [\"product\", \"key\"],", + " \"dimensionExclusions\": [],", + " \"spatialDimensions\": []", + " },", + " \"columns\": [\"timestamp\", \"product\", \"key\", \"value\"]", + " }", + "}"), + String.join("\n", + "[", + " {\"type\": \"doubleSum\", \"name\": \"value\", \"fieldName\": \"value\"}", + "]"), + 0, // minTimestamp + Granularities.NONE, + 40, // maxRowCount + String.join("\n", + "{", + " \"queryType\": \"groupBy\",", + " \"dataSource\": \"test_datasource\",", + " \"granularity\": \"ALL\",", + " \"dimensions\": [],", + " \"aggregations\": [", + " {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch\", \"fieldName\": \"key\", \"metricColumns\": [\"value\"], \"nominalEntries\": 1024},", + " {\"type\": \"count\", \"name\":\"cnt\"}", + " ],", + " \"postAggregations\": [", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"estimate\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToQuantilesSketch\", \"name\": \"quantiles-sketch\", \"field\": {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"union\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"union\",", + " \"operation\": \"UNION\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"intersection\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"intersection\",", + " \"operation\": \"INTERSECT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }},", + " {\"type\": \"arrayOfDoublesSketchToEstimate\", \"name\": \"anotb\", \"field\": {", + " \"type\": \"arrayOfDoublesSketchSetOp\",", + " \"name\": \"anotb\",", + " \"operation\": \"NOT\",", + " \"nominalEntries\": 1024,", + " \"fields\": [{\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}, {\"type\": \"fieldAccess\", \"fieldName\": \"sketch\"}]", + " }}", + " ],", + " \"intervals\": [\"2015-01-01T00:00:00.000Z/2015-01-31T00:00:00.000Z\"]", + "}")); + List results = seq.toList(); + Assert.assertEquals(1, results.size()); + Row row = results.get(0); + Assert.assertEquals(40.0, new Double(row.getRaw("cnt").toString()), 0); + Assert.assertEquals(40.0, (double) row.getRaw("sketch"), 0); + Assert.assertEquals(40.0, new Double(row.getRaw("estimate").toString()), 0); + Assert.assertEquals(40.0, new Double(row.getRaw("union").toString()), 0); + Assert.assertEquals(40.0, new Double(row.getRaw("intersection").toString()), 0); + Assert.assertEquals(0, new Double(row.getRaw("anotb").toString()), 0); + + Object obj = row.getRaw("quantiles-sketch"); + Assert.assertTrue(obj instanceof DoublesSketch); + DoublesSketch ds = (DoublesSketch) obj; + Assert.assertEquals(40, ds.getN()); + Assert.assertEquals(1.0, ds.getMinValue(), 0); + Assert.assertEquals(1.0, ds.getMaxValue(), 0); + } + + // Two buckets with statistically significant difference. + // See GenerateTestData class for details. + @Test + public void buildingSketchesAtQueryTimeAndTTest() throws Exception + { + Sequence seq = helper.createIndexAndRunQueryOnSegment( + new File(this.getClass().getClassLoader().getResource("tuple/bucket_test_data.tsv").getFile()), + String.join("\n", + "{", + " \"type\": \"string\",", + " \"parseSpec\": {", + " \"format\": \"tsv\",", + " \"timestampSpec\": {\"column\": \"timestamp\", \"format\": \"yyyyMMdd\"},", + " \"dimensionsSpec\": {", + " \"dimensions\": [\"label\", \"userid\"]", + " },", + " \"columns\": [\"timestamp\", \"label\", \"userid\", \"parameter\"]", + " }", + "}"), + String.join("\n", + "[", + " {\"type\": \"doubleSum\", \"name\": \"parameter\", \"fieldName\": \"parameter\"}", + "]"), + 0, // minTimestamp + Granularities.NONE, + 2000, // maxRowCount + String.join("\n", + "{", + " \"queryType\": \"groupBy\",", + " \"dataSource\": \"test_datasource\",", + " \"granularity\": \"ALL\",", + " \"dimensions\": [],", + " \"aggregations\": [", + " {", + " \"type\": \"filtered\",", + " \"filter\": {\"type\": \"selector\", \"dimension\": \"label\", \"value\": \"test\"},", + " \"aggregator\": {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch-test\", \"fieldName\": \"userid\", \"metricColumns\": [\"parameter\"]}", + " },", + " {", + " \"type\": \"filtered\",", + " \"filter\": {\"type\": \"selector\", \"dimension\": \"label\", \"value\": \"control\"},", + " \"aggregator\": {\"type\": \"arrayOfDoublesSketch\", \"name\": \"sketch-control\", \"fieldName\": \"userid\", \"metricColumns\": [\"parameter\"]}", + " }", + " ],", + " \"postAggregations\": [", + " {\"type\": \"arrayOfDoublesSketchTTest\",", + " \"name\": \"p-value\", \"fields\": [", + " {\"type\": \"fieldAccess\", \"fieldName\": \"sketch-test\"},", + " {\"type\": \"fieldAccess\", \"fieldName\": \"sketch-control\"}", + " ]", + " }", + " ],", + " \"intervals\": [\"2017-01-01T00:00:00.000Z/2017-01-31T00:00:00.000Z\"]", + "}")); + List results = seq.toList(); + Assert.assertEquals(1, results.size()); + Row row = results.get(0); + Object obj = row.getRaw("p-value"); + Assert.assertTrue(obj instanceof double[]); + double[] array = (double[]) obj; + Assert.assertEquals(1, array.length); + double pValue = array[0]; + // Test and control buckets were constructed to have different means, so we + // expect very low p value + Assert.assertEquals(0, pValue, 0.001); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchSetOpPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchSetOpPostAggregatorTest.java new file mode 100644 index 00000000000..72d239826b4 --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchSetOpPostAggregatorTest.java @@ -0,0 +1,90 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Arrays; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchSetOpPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchSetOpPostAggregator( + "a", + "UNION", + null, + null, + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchSetOpPostAggregator( + "a", + "UNION", + null, + null, + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different operation + final PostAggregator postAgg3 = new ArrayOfDoublesSketchSetOpPostAggregator( + "a", + "INTERSECT", + null, + null, + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different field + final PostAggregator postAgg4 = new ArrayOfDoublesSketchSetOpPostAggregator( + "a", + "UNION", + null, + null, + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 1)) + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent, not overriding equals and hashCode + final PostAggregator postAgg5 = new ArrayOfDoublesSketchTTestPostAggregator( + "a", + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchTTestPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchTTestPostAggregatorTest.java new file mode 100644 index 00000000000..3c6090f4350 --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchTTestPostAggregatorTest.java @@ -0,0 +1,81 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.util.Arrays; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchTTestPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchTTestPostAggregator( + "a", + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchTTestPostAggregator( + "a", + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchTTestPostAggregator( + "a", + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 1)) + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different name + final PostAggregator postAgg4 = new ArrayOfDoublesSketchTTestPostAggregator( + "b", + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent + final PostAggregator postAgg5 = new ArrayOfDoublesSketchSetOpPostAggregator( + "a", + "UNION", + null, + null, + Arrays.asList(new ConstantPostAggregator("", 0), new ConstantPostAggregator("", 0)) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimateAndBoundsPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimateAndBoundsPostAggregatorTest.java new file mode 100644 index 00000000000..821aad3606d --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimateAndBoundsPostAggregatorTest.java @@ -0,0 +1,80 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchToEstimateAndBoundsPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator( + "a", + new ConstantPostAggregator("", 0), + null + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator( + "a", + new ConstantPostAggregator("", 0), + null + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator( + "a", + new ConstantPostAggregator("", 1), + null + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different numStdDevs + final PostAggregator postAgg4 = new ArrayOfDoublesSketchToEstimateAndBoundsPostAggregator( + "a", + new ConstantPostAggregator("", 0), + 2 + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent + final PostAggregator postAgg5 = new ArrayOfDoublesSketchToEstimatePostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimatePostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimatePostAggregatorTest.java new file mode 100644 index 00000000000..4c665514ad1 --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToEstimatePostAggregatorTest.java @@ -0,0 +1,76 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchToEstimatePostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchToEstimatePostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchToEstimatePostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchToEstimatePostAggregator( + "a", + new ConstantPostAggregator("", 1) + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different name + final PostAggregator postAgg4 = new ArrayOfDoublesSketchToEstimatePostAggregator( + "b", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent, also not overriding equals and hashCode + final PostAggregator postAgg5 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToMeansPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToMeansPostAggregatorTest.java new file mode 100644 index 00000000000..12d1fdb7762 --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToMeansPostAggregatorTest.java @@ -0,0 +1,76 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchToMeansPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchToMeansPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchToMeansPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchToMeansPostAggregator( + "a", + new ConstantPostAggregator("", 1) + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different name + final PostAggregator postAgg4 = new ArrayOfDoublesSketchToMeansPostAggregator( + "b", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent, also not overriding equals and hashCode + final PostAggregator postAgg5 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToNumEntriesPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToNumEntriesPostAggregatorTest.java new file mode 100644 index 00000000000..ee2cbb55464 --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToNumEntriesPostAggregatorTest.java @@ -0,0 +1,76 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchToNumEntriesPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchToNumEntriesPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchToNumEntriesPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchToNumEntriesPostAggregator( + "a", + new ConstantPostAggregator("", 1) + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different name + final PostAggregator postAgg4 = new ArrayOfDoublesSketchToNumEntriesPostAggregator( + "b", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent, also not overriding equals and hashCode + final PostAggregator postAgg5 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToQuantilesSketchPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToQuantilesSketchPostAggregatorTest.java new file mode 100644 index 00000000000..c38c88f8baa --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToQuantilesSketchPostAggregatorTest.java @@ -0,0 +1,84 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchToQuantilesSketchPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchToQuantilesSketchPostAggregator( + "a", + new ConstantPostAggregator("", 0), + null, + null + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchToQuantilesSketchPostAggregator( + "a", + new ConstantPostAggregator("", 0), + null, + null + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchToQuantilesSketchPostAggregator( + "a", + new ConstantPostAggregator("", 1), + null, + null + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different column + final PostAggregator postAgg4 = new ArrayOfDoublesSketchToQuantilesSketchPostAggregator( + "a", + new ConstantPostAggregator("", 0), + 2, + null + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent + final PostAggregator postAgg5 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToStringPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToStringPostAggregatorTest.java new file mode 100644 index 00000000000..41a4858b44f --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToStringPostAggregatorTest.java @@ -0,0 +1,76 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchToStringPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 1) + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different name + final PostAggregator postAgg4 = new ArrayOfDoublesSketchToStringPostAggregator( + "b", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent, also not overriding equals and hashCode + final PostAggregator postAgg5 = new ArrayOfDoublesSketchToNumEntriesPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToVariancesPostAggregatorTest.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToVariancesPostAggregatorTest.java new file mode 100644 index 00000000000..453b2be1cf9 --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchToVariancesPostAggregatorTest.java @@ -0,0 +1,76 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import org.junit.Assert; +import org.junit.Test; + +import io.druid.query.aggregation.PostAggregator; +import io.druid.query.aggregation.post.ConstantPostAggregator; + +public class ArrayOfDoublesSketchToVariancesPostAggregatorTest +{ + + @Test + public void equalsAndHashCode() + { + final PostAggregator postAgg1 = new ArrayOfDoublesSketchToVariancesPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + @SuppressWarnings("ObjectEqualsNull") + final boolean equalsNull = postAgg1.equals(null); + Assert.assertFalse(equalsNull); + @SuppressWarnings({"EqualsWithItself", "SelfEquals"}) + final boolean equalsSelf = postAgg1.equals(postAgg1); + Assert.assertTrue(equalsSelf); + Assert.assertEquals(postAgg1.hashCode(), postAgg1.hashCode()); + + // equals + final PostAggregator postAgg2 = new ArrayOfDoublesSketchToVariancesPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertTrue(postAgg1.equals(postAgg2)); + Assert.assertEquals(postAgg1.hashCode(), postAgg2.hashCode()); + + // same class, different field + final PostAggregator postAgg3 = new ArrayOfDoublesSketchToVariancesPostAggregator( + "a", + new ConstantPostAggregator("", 1) + ); + Assert.assertFalse(postAgg1.equals(postAgg3)); + + // same class, different name + final PostAggregator postAgg4 = new ArrayOfDoublesSketchToVariancesPostAggregator( + "b", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg4)); + + // different class, same parent, also not overriding equals and hashCode + final PostAggregator postAgg5 = new ArrayOfDoublesSketchToStringPostAggregator( + "a", + new ConstantPostAggregator("", 0) + ); + Assert.assertFalse(postAgg1.equals(postAgg5)); + } + +} diff --git a/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/GenerateTestData.java b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/GenerateTestData.java new file mode 100644 index 00000000000..9d6ddde727b --- /dev/null +++ b/extensions-core/datasketches/src/test/java/io/druid/query/aggregation/datasketches/tuple/GenerateTestData.java @@ -0,0 +1,95 @@ +/* + * Licensed to Metamarkets Group Inc. (Metamarkets) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. Metamarkets 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 io.druid.query.aggregation.datasketches.tuple; + +import java.io.BufferedWriter; +import java.nio.charset.StandardCharsets; +import java.nio.file.FileSystems; +import java.nio.file.Files; +import java.nio.file.Path; +import java.util.Random; + +import org.apache.commons.codec.binary.Base64; + +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketch; +import com.yahoo.sketches.tuple.ArrayOfDoublesUpdatableSketchBuilder; + +//This is used for generating test data for ArrayOfDoublesSketchAggregationTest +class GenerateTestData +{ + + public static void main(String[] args) throws Exception + { + generateSketches(); + generateBucketTestData(); + } + + private static void generateSketches() throws Exception + { + Path path = FileSystems.getDefault().getPath("array_of_doubles_sketch_data.tsv"); + try (BufferedWriter out = Files.newBufferedWriter(path, StandardCharsets.UTF_8)) { + Random rand = new Random(); + int key = 0; + for (int i = 0; i < 20; i++) { + ArrayOfDoublesUpdatableSketch sketch = new ArrayOfDoublesUpdatableSketchBuilder().setNominalEntries(1024) + .build(); + sketch.update(key++, new double[] {1}); + sketch.update(key++, new double[] {1}); + out.write("2015010101"); + out.write('\t'); + out.write("product_" + (rand.nextInt(10) + 1)); + out.write('\t'); + out.write(Base64.encodeBase64String(sketch.compact().toByteArray())); + out.newLine(); + } + } + } + + // Data for two buckets: test and control. + // Each user ID is associated with a numeric parameter + // randomly drawn from normal distribution. + // Buckets have different means. + private static void generateBucketTestData() throws Exception + { + double meanTest = 10; + double meanControl = 10.2; + Path path = FileSystems.getDefault().getPath("bucket_test_data.tsv"); + try (BufferedWriter out = Files.newBufferedWriter(path, StandardCharsets.UTF_8)) { + Random rand = new Random(); + for (int i = 0; i < 1000; i++) { + writeBucketTestRecord(out, "test", i, rand.nextGaussian() + meanTest); + writeBucketTestRecord(out, "control", i, rand.nextGaussian() + meanControl); + } + } + } + + private static void writeBucketTestRecord(BufferedWriter out, String label, int id, double parameter) throws Exception + { + out.write("20170101"); + out.write("\t"); + out.write(label); + out.write("\t"); + out.write(Integer.toString(id)); + out.write("\t"); + out.write(Double.toString(parameter)); + out.newLine(); + } + +} diff --git a/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_build_data.tsv b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_build_data.tsv new file mode 100644 index 00000000000..3ea6df0697a --- /dev/null +++ b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_build_data.tsv @@ -0,0 +1,40 @@ +2015010101 product_2 key1 1.0 +2015010101 product_3 key2 1.0 +2015010101 product_8 key3 1.0 +2015010101 product_1 key4 1.0 +2015010101 product_1 key5 1.0 +2015010101 product_7 key6 1.0 +2015010101 product_5 key7 1.0 +2015010101 product_4 key8 1.0 +2015010101 product_3 key9 1.0 +2015010101 product_6 key10 1.0 +2015010101 product_5 key11 1.0 +2015010101 product_6 key12 1.0 +2015010101 product_6 key13 1.0 +2015010101 product_6 key14 1.0 +2015010101 product_6 key15 1.0 +2015010101 product_6 key16 1.0 +2015010101 product_3 key17 1.0 +2015010101 product_1 key18 1.0 +2015010101 product_2 key19 1.0 +2015010101 product_10 key20 1.0 +2015010101 product_2 key21 1.0 +2015010101 product_3 key22 1.0 +2015010101 product_8 key23 1.0 +2015010101 product_1 key24 1.0 +2015010101 product_1 key25 1.0 +2015010101 product_7 key26 1.0 +2015010101 product_5 key27 1.0 +2015010101 product_4 key28 1.0 +2015010101 product_3 key29 1.0 +2015010101 product_6 key30 1.0 +2015010101 product_5 key31 1.0 +2015010101 product_6 key32 1.0 +2015010101 product_6 key33 1.0 +2015010101 product_6 key34 1.0 +2015010101 product_6 key35 1.0 +2015010101 product_6 key36 1.0 +2015010101 product_3 key37 1.0 +2015010101 product_1 key38 1.0 +2015010101 product_2 key39 1.0 +2015010101 product_10 key40 1.0 diff --git a/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_build_data_two_values.tsv b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_build_data_two_values.tsv new file mode 100644 index 00000000000..c5bb451f35d --- /dev/null +++ b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_build_data_two_values.tsv @@ -0,0 +1,40 @@ +2015010101 product_2 key1 1.0 2.0 +2015010101 product_3 key2 1.0 2.0 +2015010101 product_8 key3 1.0 2.0 +2015010101 product_1 key4 1.0 2.0 +2015010101 product_1 key5 1.0 2.0 +2015010101 product_7 key6 1.0 2.0 +2015010101 product_5 key7 1.0 2.0 +2015010101 product_4 key8 1.0 2.0 +2015010101 product_3 key9 1.0 2.0 +2015010101 product_6 key10 1.0 2.0 +2015010101 product_5 key11 1.0 2.0 +2015010101 product_6 key12 1.0 2.0 +2015010101 product_6 key13 1.0 2.0 +2015010101 product_6 key14 1.0 2.0 +2015010101 product_6 key15 1.0 2.0 +2015010101 product_6 key16 1.0 2.0 +2015010101 product_3 key17 1.0 2.0 +2015010101 product_1 key18 1.0 2.0 +2015010101 product_2 key19 1.0 2.0 +2015010101 product_10 key20 1.0 2.0 +2015010101 product_2 key21 1.0 2.0 +2015010101 product_3 key22 1.0 2.0 +2015010101 product_8 key23 1.0 2.0 +2015010101 product_1 key24 1.0 2.0 +2015010101 product_1 key25 1.0 2.0 +2015010101 product_7 key26 1.0 2.0 +2015010101 product_5 key27 1.0 2.0 +2015010101 product_4 key28 1.0 2.0 +2015010101 product_3 key29 1.0 2.0 +2015010101 product_6 key30 1.0 2.0 +2015010101 product_5 key31 1.0 2.0 +2015010101 product_6 key32 1.0 2.0 +2015010101 product_6 key33 1.0 2.0 +2015010101 product_6 key34 1.0 2.0 +2015010101 product_6 key35 1.0 2.0 +2015010101 product_6 key36 1.0 2.0 +2015010101 product_3 key37 1.0 2.0 +2015010101 product_1 key38 1.0 2.0 +2015010101 product_2 key39 1.0 2.0 +2015010101 product_10 key40 1.0 2.0 diff --git a/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_sketch_data.tsv b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_sketch_data.tsv new file mode 100644 index 00000000000..a2380a96ac1 --- /dev/null +++ b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_sketch_data.tsv @@ -0,0 +1,20 @@ +2015010101 product_5 AQEJAwgBzJP/////////fwIAAAAAAAAAFfl9y72GoQXla2HuyIBEIAAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_6 AQEJAwgBzJP/////////fwIAAAAAAAAAukCzwdoGaV3Dl/wSgXCdHgAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_6 AQEJAwgBzJP/////////fwIAAAAAAAAAvTJzckaRzBRA3i7hyds9CAAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_8 AQEJAwgBzJP/////////fwIAAAAAAAAA4PSL6pmDw3z+FiET+5i8EAAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_5 AQEJAwgBzJP/////////fwIAAAAAAAAAGtEwC5mMLyJpi7mRuGhXCAAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_4 AQEJAwgBzJP/////////fwIAAAAAAAAA+zh5iRMkjwHYLSN3S7k1fgAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_5 AQEJAwgBzJP/////////fwIAAAAAAAAAI6VbOBr9dB9XHsScr0GXOgAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_4 AQEJAwgBzJP/////////fwIAAAAAAAAAbakWvEpmYR4+utyjb2+2IAAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_6 AQEJAwgBzJP/////////fwIAAAAAAAAAZ/it9XjZiz+3YhopjQlLSAAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_10 AQEJAwgBzJP/////////fwIAAAAAAAAA4e4o4grYzUTcn/yx1EcobwAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_4 AQEJAwgBzJP/////////fwIAAAAAAAAA4trPn83qvgRkPRJui9c7SQAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_6 AQEJAwgBzJP/////////fwIAAAAAAAAAT/JN4LwMrhybo4QRBt3JGwAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_6 AQEJAwgBzJP/////////fwIAAAAAAAAAXKtUQBM9eT33lvvVCUmqJQAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_3 AQEJAwgBzJP/////////fwIAAAAAAAAA4ww1FskK7k6RnfVc9m+6EQAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_7 AQEJAwgBzJP/////////fwIAAAAAAAAAmtQYKDOmoS2sP+rYMRoKewAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_1 AQEJAwgBzJP/////////fwIAAAAAAAAARuLEMqskVTXLXTAt9sQ8KAAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_4 AQEJAwgBzJP/////////fwIAAAAAAAAAZ9D3YBMITC7PFrUBaFIOcgAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_2 AQEJAwgBzJP/////////fwIAAAAAAAAA1blD14YpNinCNObiWYQdOwAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_5 AQEJAwgBzJP/////////fwIAAAAAAAAAtvKOiU9fciHblwTxetsiPgAAAAAAAPA/AAAAAAAA8D8= +2015010101 product_1 AQEJAwgBzJP/////////fwIAAAAAAAAAmRKxCIUTtzDKA2Gq5zOfFQAAAAAAAPA/AAAAAAAA8D8= diff --git a/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_sketch_data_two_values.tsv b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_sketch_data_two_values.tsv new file mode 100644 index 00000000000..7d46485af29 --- /dev/null +++ b/extensions-core/datasketches/src/test/resources/tuple/array_of_doubles_sketch_data_two_values.tsv @@ -0,0 +1,20 @@ +2015010101 product_5 AQEJAwgCzJP/////////fwIAAAAAAAAA5Wth7siARCAV+X3LvYahBQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAAw5f8EoFwnR66QLPB2gZpXQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAAQN4u4cnbPQi9MnNyRpHMFAAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAA4PSL6pmDw3z+FiET+5i8EAAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAAaYu5kbhoVwga0TALmYwvIgAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_6 AQEJAwgCzJP/////////fwIAAAAAAAAA2C0jd0u5NX77OHmJEySPAQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_1 AQEJAwgCzJP/////////fwIAAAAAAAAAI6VbOBr9dB9XHsScr0GXOgAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_5 AQEJAwgCzJP/////////fwIAAAAAAAAAbakWvEpmYR4+utyjb2+2IAAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_8 AQEJAwgCzJP/////////fwIAAAAAAAAAZ/it9XjZiz+3YhopjQlLSAAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_9 AQEJAwgCzJP/////////fwIAAAAAAAAA4e4o4grYzUTcn/yx1EcobwAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAA4trPn83qvgRkPRJui9c7SQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_9 AQEJAwgCzJP/////////fwIAAAAAAAAAT/JN4LwMrhybo4QRBt3JGwAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_8 AQEJAwgCzJP/////////fwIAAAAAAAAA95b71QlJqiVcq1RAEz15PQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAA4ww1FskK7k6RnfVc9m+6EQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_8 AQEJAwgCzJP/////////fwIAAAAAAAAArD/q2DEaCnua1BgoM6ahLQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_8 AQEJAwgCzJP/////////fwIAAAAAAAAARuLEMqskVTXLXTAt9sQ8KAAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAAZ9D3YBMITC7PFrUBaFIOcgAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_2 AQEJAwgCzJP/////////fwIAAAAAAAAAwjTm4lmEHTvVuUPXhik2KQAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_5 AQEJAwgCzJP/////////fwIAAAAAAAAAtvKOiU9fciHblwTxetsiPgAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA +2015010101 product_5 AQEJAwgCzJP/////////fwIAAAAAAAAAygNhqucznxWZErEIhRO3MAAAAAAAAPA/AAAAAAAAAEAAAAAAAADwPwAAAAAAAABA diff --git a/extensions-core/datasketches/src/test/resources/tuple/bucket_test_data.tsv b/extensions-core/datasketches/src/test/resources/tuple/bucket_test_data.tsv new file mode 100644 index 00000000000..34623315204 --- /dev/null +++ b/extensions-core/datasketches/src/test/resources/tuple/bucket_test_data.tsv @@ -0,0 +1,2000 @@ +20170101 test 0 11.459967936733348 +20170101 control 0 10.549716040773465 +20170101 test 1 8.568120896044702 +20170101 control 1 10.348531286018458 +20170101 test 2 8.447446654247528 +20170101 control 2 10.052647652500545 +20170101 test 3 10.806886261710327 +20170101 control 3 9.198017240479155 +20170101 test 4 11.58755055487326 +20170101 control 4 8.396430351547657 +20170101 test 5 9.800670060967827 +20170101 control 5 10.220574238134374 +20170101 test 6 12.561452599993308 +20170101 control 6 11.644140782415773 +20170101 test 7 11.060822763743449 +20170101 control 7 10.868745226685222 +20170101 test 8 10.481686637435017 +20170101 control 8 10.475003353731175 +20170101 test 9 9.679772992883874 +20170101 control 9 10.755185178097742 +20170101 test 10 10.933065784785576 +20170101 control 10 10.934207365416606 +20170101 test 11 11.185350911461047 +20170101 control 11 9.297109404189236 +20170101 test 12 8.214025393387473 +20170101 control 12 12.282565317251537 +20170101 test 13 9.470182313563338 +20170101 control 13 10.80956842819647 +20170101 test 14 10.385347811508677 +20170101 control 14 10.339201210402884 +20170101 test 15 10.536274990985412 +20170101 control 15 9.645740184872361 +20170101 test 16 9.632752587702953 +20170101 control 16 9.748924285510755 +20170101 test 17 10.103106792108772 +20170101 control 17 10.506077787216055 +20170101 test 18 10.30324807576078 +20170101 control 18 11.037017640666289 +20170101 test 19 10.168764849923868 +20170101 control 19 10.528843872772967 +20170101 test 20 10.537518701047155 +20170101 control 20 9.494309295320543 +20170101 test 21 10.844749053485316 +20170101 control 21 8.655426634583323 +20170101 test 22 10.819194851647977 +20170101 control 22 12.312637553215136 +20170101 test 23 8.653462705209453 +20170101 control 23 10.44676355940558 +20170101 test 24 11.700859187271385 +20170101 control 24 11.635231764337853 +20170101 test 25 11.156256560099218 +20170101 control 25 10.797421116004678 +20170101 test 26 10.467077028730523 +20170101 control 26 9.354061572209629 +20170101 test 27 10.312945584221527 +20170101 control 27 9.660436714510666 +20170101 test 28 10.97582838440979 +20170101 control 28 11.415823504511946 +20170101 test 29 10.571885735408605 +20170101 control 29 11.296044675868208 +20170101 test 30 8.471574489777268 +20170101 control 30 9.4996295314252 +20170101 test 31 8.932955600561668 +20170101 control 31 9.969849267487538 +20170101 test 32 8.485795876868476 +20170101 control 32 9.820910864922151 +20170101 test 33 8.176163445082171 +20170101 control 33 9.637842516161625 +20170101 test 34 11.407581412642449 +20170101 control 34 12.840280991417146 +20170101 test 35 10.267032993297976 +20170101 control 35 10.605655301891973 +20170101 test 36 11.083830164603715 +20170101 control 36 11.017433530050184 +20170101 test 37 9.934682365698562 +20170101 control 37 10.950101648995247 +20170101 test 38 9.725246262570538 +20170101 control 38 11.194009309161984 +20170101 test 39 8.590801991311055 +20170101 control 39 8.781251151132397 +20170101 test 40 10.492039373734158 +20170101 control 40 10.541837878564357 +20170101 test 41 8.716302663186175 +20170101 control 41 9.708953294289195 +20170101 test 42 11.15651884636906 +20170101 control 42 9.927110261599063 +20170101 test 43 9.450043884758614 +20170101 control 43 10.65462184364204 +20170101 test 44 11.391926527922397 +20170101 control 44 10.97406865647974 +20170101 test 45 8.834908754367282 +20170101 control 45 9.68434377332244 +20170101 test 46 10.430461330640869 +20170101 control 46 9.492293889136558 +20170101 test 47 9.273850453308581 +20170101 control 47 10.863188662797315 +20170101 test 48 9.592090478350812 +20170101 control 48 9.219285265270807 +20170101 test 49 11.272526235235507 +20170101 control 49 10.940239776026196 +20170101 test 50 8.96206935054056 +20170101 control 50 9.305416644965456 +20170101 test 51 8.838761615064655 +20170101 control 51 10.432605836740684 +20170101 test 52 11.762807804972264 +20170101 control 52 11.697333299309793 +20170101 test 53 9.893612552543656 +20170101 control 53 10.277001033216983 +20170101 test 54 7.246996540378479 +20170101 control 54 9.491580443615149 +20170101 test 55 10.500207342640351 +20170101 control 55 9.915870790691748 +20170101 test 56 8.826386900251936 +20170101 control 56 9.675363065844953 +20170101 test 57 9.068626578674772 +20170101 control 57 8.359458250417799 +20170101 test 58 9.549027688496334 +20170101 control 58 10.947112022872766 +20170101 test 59 8.886013656266002 +20170101 control 59 9.85382536351549 +20170101 test 60 8.294873268099746 +20170101 control 60 11.29573785751358 +20170101 test 61 10.30706339727673 +20170101 control 61 9.346108348070771 +20170101 test 62 9.899953583559922 +20170101 control 62 10.291357312435785 +20170101 test 63 11.308723579076513 +20170101 control 63 10.904896549309173 +20170101 test 64 9.25301129001663 +20170101 control 64 9.912735915086465 +20170101 test 65 7.180690067638144 +20170101 control 65 10.5908545981214 +20170101 test 66 10.42770277506564 +20170101 control 66 10.26531157791592 +20170101 test 67 7.579379190233095 +20170101 control 67 9.858012391598908 +20170101 test 68 9.758747055739077 +20170101 control 68 9.800437090435999 +20170101 test 69 10.40845628456578 +20170101 control 69 7.741019397574913 +20170101 test 70 10.122953220156116 +20170101 control 70 10.27763861714341 +20170101 test 71 10.503448101655545 +20170101 control 71 8.88864584272527 +20170101 test 72 11.435235998001012 +20170101 control 72 10.045451135718096 +20170101 test 73 9.279912008780435 +20170101 control 73 10.849470193114023 +20170101 test 74 8.960294308271244 +20170101 control 74 11.582510579894283 +20170101 test 75 10.99771182828329 +20170101 control 75 10.930859188977937 +20170101 test 76 10.427890471875319 +20170101 control 76 9.991290914833312 +20170101 test 77 9.155350677903687 +20170101 control 77 10.088617800846592 +20170101 test 78 9.027877736477638 +20170101 control 78 8.94520977170074 +20170101 test 79 9.96686811807249 +20170101 control 79 9.340057279626665 +20170101 test 80 9.178512020588803 +20170101 control 80 10.797882322144691 +20170101 test 81 10.868599644899493 +20170101 control 81 10.379208127460956 +20170101 test 82 7.882264382872262 +20170101 control 82 8.600856311577028 +20170101 test 83 8.915436367130582 +20170101 control 83 10.82843789643216 +20170101 test 84 11.139784495546587 +20170101 control 84 10.774366280741773 +20170101 test 85 9.375004267670345 +20170101 control 85 12.005888136580106 +20170101 test 86 8.4019975344101 +20170101 control 86 11.241303954972622 +20170101 test 87 9.731009759281122 +20170101 control 87 9.318694737909617 +20170101 test 88 11.3488058667274 +20170101 control 88 10.034112450532826 +20170101 test 89 10.107426268730665 +20170101 control 89 9.744781366318275 +20170101 test 90 11.060440680476399 +20170101 control 90 9.291094965243298 +20170101 test 91 10.166158132780605 +20170101 control 91 7.853574419279283 +20170101 test 92 8.885539924235095 +20170101 control 92 8.530250389471401 +20170101 test 93 8.659714711615432 +20170101 control 93 7.394700681850208 +20170101 test 94 7.253068038396911 +20170101 control 94 10.443085809150222 +20170101 test 95 10.782959906456387 +20170101 control 95 10.779852087989903 +20170101 test 96 9.844180195192628 +20170101 control 96 10.830439628767744 +20170101 test 97 10.543515435283735 +20170101 control 97 9.259162811761785 +20170101 test 98 10.414506064612599 +20170101 control 98 9.534503912483391 +20170101 test 99 8.198518708453214 +20170101 control 99 10.803094189833855 +20170101 test 100 11.545631028902777 +20170101 control 100 7.305145969089655 +20170101 test 101 9.062189168140991 +20170101 control 101 9.092446140033767 +20170101 test 102 9.060612885568629 +20170101 control 102 10.871897638681224 +20170101 test 103 8.711589578026134 +20170101 control 103 10.118828121534461 +20170101 test 104 9.799305869066506 +20170101 control 104 10.74773755568313 +20170101 test 105 10.668216855951849 +20170101 control 105 10.652541807037348 +20170101 test 106 11.422931056197887 +20170101 control 106 11.640241060484824 +20170101 test 107 9.249538483017748 +20170101 control 107 11.004521150107097 +20170101 test 108 10.033790858363998 +20170101 control 108 9.975576279721531 +20170101 test 109 9.12277507485658 +20170101 control 109 9.817611083692034 +20170101 test 110 8.233635463220404 +20170101 control 110 8.622793821828324 +20170101 test 111 9.332050057420064 +20170101 control 111 9.810774681223526 +20170101 test 112 9.453139314184375 +20170101 control 112 10.497184489263416 +20170101 test 113 8.259780251487784 +20170101 control 113 9.943009592855711 +20170101 test 114 10.903995765953846 +20170101 control 114 10.85265163232181 +20170101 test 115 9.262963375039417 +20170101 control 115 10.452653439361814 +20170101 test 116 9.584239703368358 +20170101 control 116 7.9287653825326325 +20170101 test 117 11.955179158831296 +20170101 control 117 10.333194459748315 +20170101 test 118 10.786790073314378 +20170101 control 118 11.775328894822234 +20170101 test 119 9.73245398896988 +20170101 control 119 10.75151256735289 +20170101 test 120 10.32529339596644 +20170101 control 120 8.066759170449133 +20170101 test 121 9.66241107033213 +20170101 control 121 9.683462170597702 +20170101 test 122 8.653246889494246 +20170101 control 122 11.58793828248685 +20170101 test 123 9.852712376590304 +20170101 control 123 8.816565133398145 +20170101 test 124 11.47917583746405 +20170101 control 124 9.857691753621866 +20170101 test 125 11.198147483737882 +20170101 control 125 9.034947212213691 +20170101 test 126 10.265504261957295 +20170101 control 126 8.55148494957648 +20170101 test 127 10.850033788961312 +20170101 control 127 11.37824597624751 +20170101 test 128 10.53518678993022 +20170101 control 128 9.677803743242194 +20170101 test 129 11.899990116812647 +20170101 control 129 9.816226716471641 +20170101 test 130 10.462427560720158 +20170101 control 130 9.124667802571693 +20170101 test 131 10.414122177914503 +20170101 control 131 10.12242327783072 +20170101 test 132 11.15078975920477 +20170101 control 132 11.38367533158978 +20170101 test 133 10.895062063269554 +20170101 control 133 11.537406702351781 +20170101 test 134 10.700235551800086 +20170101 control 134 8.778307672280928 +20170101 test 135 9.988886958623086 +20170101 control 135 10.84398224148593 +20170101 test 136 9.286396799420285 +20170101 control 136 11.410227438712035 +20170101 test 137 10.496917427736529 +20170101 control 137 9.25956974442197 +20170101 test 138 8.957491376540593 +20170101 control 138 10.206976223630438 +20170101 test 139 9.293835080539814 +20170101 control 139 10.434381675712824 +20170101 test 140 8.712023470427283 +20170101 control 140 9.11565137832573 +20170101 test 141 11.09191520224789 +20170101 control 141 10.537383458432505 +20170101 test 142 8.940556538113746 +20170101 control 142 9.06778329811347 +20170101 test 143 9.176391371514457 +20170101 control 143 8.959825727138005 +20170101 test 144 9.20230939361822 +20170101 control 144 11.44700377371125 +20170101 test 145 11.857683454353843 +20170101 control 145 11.656291807086857 +20170101 test 146 8.84656160091928 +20170101 control 146 8.626549495335102 +20170101 test 147 10.111765926924932 +20170101 control 147 10.59524507744937 +20170101 test 148 9.95365799474295 +20170101 control 148 9.482944732059243 +20170101 test 149 9.03140334806913 +20170101 control 149 9.614565513052137 +20170101 test 150 8.769436793949165 +20170101 control 150 9.839407895105271 +20170101 test 151 8.954486256296267 +20170101 control 151 10.609882307749231 +20170101 test 152 10.357065194952336 +20170101 control 152 10.12986084690308 +20170101 test 153 9.922768955957077 +20170101 control 153 9.950016573289068 +20170101 test 154 9.459081997642947 +20170101 control 154 11.954836646459386 +20170101 test 155 10.492738725006427 +20170101 control 155 10.388675830030484 +20170101 test 156 11.358807043922743 +20170101 control 156 9.649482593753167 +20170101 test 157 7.528121982455689 +20170101 control 157 10.522042393360902 +20170101 test 158 11.609113628930267 +20170101 control 158 10.234043874968895 +20170101 test 159 9.535803595711254 +20170101 control 159 10.007424232825976 +20170101 test 160 8.488568813676324 +20170101 control 160 11.515385021558082 +20170101 test 161 10.089888692916887 +20170101 control 161 10.688972451362671 +20170101 test 162 9.38791815016897 +20170101 control 162 10.177923778157728 +20170101 test 163 8.984199596319058 +20170101 control 163 10.932928997775381 +20170101 test 164 8.856661003612595 +20170101 control 164 9.43289711267024 +20170101 test 165 9.876685328326332 +20170101 control 165 11.73603008268834 +20170101 test 166 12.442168552003519 +20170101 control 166 10.541187245392834 +20170101 test 167 10.470855082457694 +20170101 control 167 11.829535587836569 +20170101 test 168 8.32263828805461 +20170101 control 168 9.436777431504018 +20170101 test 169 10.159333916985156 +20170101 control 169 8.573929095815018 +20170101 test 170 10.256566850352847 +20170101 control 170 12.769565988624274 +20170101 test 171 10.371433055557194 +20170101 control 171 10.51192726300484 +20170101 test 172 10.737736884014998 +20170101 control 172 10.286193028713845 +20170101 test 173 11.15442621832082 +20170101 control 173 9.37141049094303 +20170101 test 174 9.983764345583822 +20170101 control 174 10.886177562025361 +20170101 test 175 8.153492922968843 +20170101 control 175 10.54686658224763 +20170101 test 176 8.90431503654771 +20170101 control 176 11.679537833797161 +20170101 test 177 9.764211752003453 +20170101 control 177 9.485859240153331 +20170101 test 178 7.666359381772769 +20170101 control 178 9.473491128973897 +20170101 test 179 9.34762344471342 +20170101 control 179 9.745172903853883 +20170101 test 180 7.969864307311342 +20170101 control 180 11.020437828128589 +20170101 test 181 9.318860189348953 +20170101 control 181 11.220070392382565 +20170101 test 182 9.603242042083048 +20170101 control 182 11.221204195745091 +20170101 test 183 9.96974666217208 +20170101 control 183 9.566019843764963 +20170101 test 184 9.724291805559783 +20170101 control 184 10.10797612916699 +20170101 test 185 10.805711087607659 +20170101 control 185 8.594851402642929 +20170101 test 186 9.55155883404929 +20170101 control 186 8.94811455891784 +20170101 test 187 9.656596124170996 +20170101 control 187 8.863535550010969 +20170101 test 188 8.974746572860102 +20170101 control 188 10.896298338304144 +20170101 test 189 9.770618124904594 +20170101 control 189 12.538691182534173 +20170101 test 190 8.27416144285198 +20170101 control 190 10.319809405032336 +20170101 test 191 11.771291667616786 +20170101 control 191 9.324330017406888 +20170101 test 192 11.985997627646816 +20170101 control 192 10.203960793773835 +20170101 test 193 12.277762567593623 +20170101 control 193 11.543836497765898 +20170101 test 194 8.908401352722679 +20170101 control 194 9.301768564585169 +20170101 test 195 11.930950028295364 +20170101 control 195 9.50076751442689 +20170101 test 196 9.884797275402056 +20170101 control 196 9.27957099721153 +20170101 test 197 10.203522530002932 +20170101 control 197 11.040028594262655 +20170101 test 198 10.898344593673315 +20170101 control 198 10.623151482897436 +20170101 test 199 12.437922103844818 +20170101 control 199 12.048918259083528 +20170101 test 200 9.657286651022147 +20170101 control 200 10.844346273798278 +20170101 test 201 10.181412342842485 +20170101 control 201 10.157956960315634 +20170101 test 202 8.978580583888961 +20170101 control 202 9.872327403325658 +20170101 test 203 9.453777849653017 +20170101 control 203 12.866668732293721 +20170101 test 204 9.623582040378375 +20170101 control 204 10.581396643331608 +20170101 test 205 10.768066881762124 +20170101 control 205 11.479269330700223 +20170101 test 206 8.534835966820932 +20170101 control 206 10.367899661190808 +20170101 test 207 10.959474731151355 +20170101 control 207 12.11170720318825 +20170101 test 208 10.298212215754615 +20170101 control 208 10.178651826695537 +20170101 test 209 7.962503281254055 +20170101 control 209 11.036333181999689 +20170101 test 210 11.182146982491423 +20170101 control 210 10.650962980279544 +20170101 test 211 11.485220641154106 +20170101 control 211 9.917878000550738 +20170101 test 212 9.833577726167482 +20170101 control 212 10.761252012235193 +20170101 test 213 9.041395617606884 +20170101 control 213 9.576110215692673 +20170101 test 214 9.813483975435567 +20170101 control 214 9.642470123907065 +20170101 test 215 10.062936145545256 +20170101 control 215 8.827059215894641 +20170101 test 216 9.339401578124862 +20170101 control 216 11.78309963949186 +20170101 test 217 9.864473171060752 +20170101 control 217 11.112660783095118 +20170101 test 218 9.555260063636817 +20170101 control 218 11.105864437227325 +20170101 test 219 10.805470776329324 +20170101 control 219 10.026819593742315 +20170101 test 220 11.149819199514253 +20170101 control 220 10.931932912736524 +20170101 test 221 9.24011008676305 +20170101 control 221 11.704278411007262 +20170101 test 222 10.997775165201743 +20170101 control 222 10.244064734495135 +20170101 test 223 11.229629694390201 +20170101 control 223 11.359068257038166 +20170101 test 224 10.189606135328486 +20170101 control 224 10.937332751331024 +20170101 test 225 10.655170862079135 +20170101 control 225 9.849954142230967 +20170101 test 226 9.108060480206142 +20170101 control 226 9.239741640594444 +20170101 test 227 10.244643983391171 +20170101 control 227 10.794879236739137 +20170101 test 228 9.962329571361556 +20170101 control 228 9.567072180434552 +20170101 test 229 10.298796763192126 +20170101 control 229 9.768057319939167 +20170101 test 230 10.626798425833993 +20170101 control 230 10.253193769848032 +20170101 test 231 12.63912797267015 +20170101 control 231 10.893819646172041 +20170101 test 232 8.747202967422716 +20170101 control 232 10.179452441330776 +20170101 test 233 11.172151917585442 +20170101 control 233 11.298890759179333 +20170101 test 234 9.275166513939729 +20170101 control 234 9.013895306189172 +20170101 test 235 10.758053416757624 +20170101 control 235 11.35572650076121 +20170101 test 236 10.785143943404199 +20170101 control 236 11.371621460542109 +20170101 test 237 10.148068805114477 +20170101 control 237 9.676248899434759 +20170101 test 238 9.651607968481482 +20170101 control 238 9.359549009593927 +20170101 test 239 10.551392195782187 +20170101 control 239 8.532072700246616 +20170101 test 240 9.307386338904236 +20170101 control 240 10.094779378789555 +20170101 test 241 9.720257186473978 +20170101 control 241 7.790896619476037 +20170101 test 242 10.232253171365139 +20170101 control 242 10.51490441969618 +20170101 test 243 10.872286234466436 +20170101 control 243 10.40986199441339 +20170101 test 244 12.34776301034561 +20170101 control 244 11.20208176717607 +20170101 test 245 9.096921594663781 +20170101 control 245 11.71616300046167 +20170101 test 246 10.04296130017132 +20170101 control 246 6.507398487985544 +20170101 test 247 9.267668436595425 +20170101 control 247 10.357229121028677 +20170101 test 248 10.836080272614586 +20170101 control 248 11.778361258846395 +20170101 test 249 10.050649919608789 +20170101 control 249 8.87356745510428 +20170101 test 250 9.903690190368847 +20170101 control 250 10.285451646314563 +20170101 test 251 9.476336506751236 +20170101 control 251 12.208290991417599 +20170101 test 252 11.88690383174119 +20170101 control 252 10.571171247090584 +20170101 test 253 10.05887513637873 +20170101 control 253 11.074347357777752 +20170101 test 254 11.200206151458397 +20170101 control 254 9.445032737615056 +20170101 test 255 10.316319743563222 +20170101 control 255 8.727383519584745 +20170101 test 256 12.63291502583076 +20170101 control 256 10.665643575729817 +20170101 test 257 9.956832268421207 +20170101 control 257 10.035549848424862 +20170101 test 258 10.465966555028793 +20170101 control 258 10.879556004144712 +20170101 test 259 10.196193204796678 +20170101 control 259 12.196819400606916 +20170101 test 260 10.993334698161917 +20170101 control 260 10.936280670136425 +20170101 test 261 9.510934102902205 +20170101 control 261 9.143997341515513 +20170101 test 262 10.091657588508612 +20170101 control 262 11.343867478573562 +20170101 test 263 9.312008675450597 +20170101 control 263 9.953894642495703 +20170101 test 264 10.518544151440688 +20170101 control 264 10.340917720573488 +20170101 test 265 10.56742264698759 +20170101 control 265 10.411338994216042 +20170101 test 266 9.690740788651496 +20170101 control 266 10.983726019873933 +20170101 test 267 9.077014833894452 +20170101 control 267 10.515531617865307 +20170101 test 268 9.63617396006616 +20170101 control 268 9.255781025452967 +20170101 test 269 10.522586101880126 +20170101 control 269 10.538771710606468 +20170101 test 270 10.162031020921956 +20170101 control 270 10.196062348131097 +20170101 test 271 11.7286638737121 +20170101 control 271 10.537663520611941 +20170101 test 272 9.421139946837336 +20170101 control 272 11.364511055219756 +20170101 test 273 10.176977653025585 +20170101 control 273 9.862991870227624 +20170101 test 274 10.629063002921184 +20170101 control 274 10.311919730399197 +20170101 test 275 8.149969436403435 +20170101 control 275 10.70792717478237 +20170101 test 276 10.810401791354533 +20170101 control 276 8.979927156588953 +20170101 test 277 8.872960044534665 +20170101 control 277 10.907896293763606 +20170101 test 278 10.180875554535408 +20170101 control 278 11.964525104328683 +20170101 test 279 10.88983523513461 +20170101 control 279 9.501035153533408 +20170101 test 280 11.30827106773253 +20170101 control 280 10.605329984195842 +20170101 test 281 11.802407178669574 +20170101 control 281 10.422286202668152 +20170101 test 282 9.60271009245262 +20170101 control 282 9.251785543047381 +20170101 test 283 9.356042223927295 +20170101 control 283 9.92763276680497 +20170101 test 284 11.28304694494868 +20170101 control 284 11.18625192912038 +20170101 test 285 9.878020464899347 +20170101 control 285 10.66351856729685 +20170101 test 286 9.770119437959776 +20170101 control 286 10.004483232678895 +20170101 test 287 10.930140692493117 +20170101 control 287 9.96170796981345 +20170101 test 288 9.073641839913016 +20170101 control 288 11.210586295631886 +20170101 test 289 8.305108548968345 +20170101 control 289 9.96941150958355 +20170101 test 290 10.775234923908542 +20170101 control 290 9.631632297504535 +20170101 test 291 9.927801803025922 +20170101 control 291 9.138364134935712 +20170101 test 292 10.674052759849767 +20170101 control 292 10.740571048499982 +20170101 test 293 9.973312777171028 +20170101 control 293 9.723309330748254 +20170101 test 294 11.831127069835052 +20170101 control 294 10.837425568376071 +20170101 test 295 10.767737490068233 +20170101 control 295 10.327865966755454 +20170101 test 296 10.301907987962695 +20170101 control 296 10.28272668431674 +20170101 test 297 9.521518962630575 +20170101 control 297 7.586790186614999 +20170101 test 298 9.552530739254987 +20170101 control 298 10.616282572921037 +20170101 test 299 10.695298112118307 +20170101 control 299 10.248465605786736 +20170101 test 300 9.890850624403544 +20170101 control 300 9.877419533051935 +20170101 test 301 9.290620918989914 +20170101 control 301 10.200677881912465 +20170101 test 302 10.273425493202629 +20170101 control 302 9.181357268515907 +20170101 test 303 9.688359973563802 +20170101 control 303 10.692904212984601 +20170101 test 304 10.023332246544737 +20170101 control 304 10.643249755969132 +20170101 test 305 7.51706444170312 +20170101 control 305 8.8047112119201 +20170101 test 306 11.087649359158858 +20170101 control 306 9.978149246062982 +20170101 test 307 9.406346653908178 +20170101 control 307 11.363129111356479 +20170101 test 308 11.937051403635138 +20170101 control 308 9.705744660219912 +20170101 test 309 10.266476650976372 +20170101 control 309 8.8710834420323 +20170101 test 310 10.528320801512573 +20170101 control 310 9.493580290504473 +20170101 test 311 11.947963657064038 +20170101 control 311 8.865623304409809 +20170101 test 312 8.654149801677613 +20170101 control 312 10.650513445831276 +20170101 test 313 9.389934386606264 +20170101 control 313 7.919126769738352 +20170101 test 314 9.764106374499438 +20170101 control 314 10.6209548442601 +20170101 test 315 10.434069998767601 +20170101 control 315 11.093694798574356 +20170101 test 316 9.735964952715705 +20170101 control 316 9.380465370610477 +20170101 test 317 10.310981836240062 +20170101 control 317 10.682534902643171 +20170101 test 318 9.666920488572487 +20170101 control 318 8.709860357680796 +20170101 test 319 10.152028128565693 +20170101 control 319 9.382130782108424 +20170101 test 320 9.82697526994824 +20170101 control 320 9.498152703266166 +20170101 test 321 9.913252785462625 +20170101 control 321 9.860135347235271 +20170101 test 322 8.527626623716467 +20170101 control 322 10.990955151156447 +20170101 test 323 8.50653560455669 +20170101 control 323 10.934048445627322 +20170101 test 324 10.475759115709394 +20170101 control 324 9.22772657415252 +20170101 test 325 10.69741303868446 +20170101 control 325 10.905154574787467 +20170101 test 326 8.424233050445405 +20170101 control 326 8.661260361484914 +20170101 test 327 9.885393577756417 +20170101 control 327 10.912259361615877 +20170101 test 328 8.669576831690257 +20170101 control 328 11.36155134051953 +20170101 test 329 11.169617185115968 +20170101 control 329 10.485735347237881 +20170101 test 330 10.524599252272242 +20170101 control 330 9.732779301345179 +20170101 test 331 8.910290241134307 +20170101 control 331 11.275506382259131 +20170101 test 332 8.550342565604003 +20170101 control 332 11.25505656973124 +20170101 test 333 10.946116235285626 +20170101 control 333 10.93691553675994 +20170101 test 334 10.828923666752463 +20170101 control 334 9.365442930411188 +20170101 test 335 10.66875073789611 +20170101 control 335 10.494890104069999 +20170101 test 336 8.099852218558718 +20170101 control 336 8.733742112818272 +20170101 test 337 8.75114735089116 +20170101 control 337 10.364294125358956 +20170101 test 338 11.070829351548094 +20170101 control 338 9.205604487639347 +20170101 test 339 10.427456107487929 +20170101 control 339 10.272652471467774 +20170101 test 340 9.915300533450525 +20170101 control 340 10.516012648112033 +20170101 test 341 10.476101691756485 +20170101 control 341 11.795990970553014 +20170101 test 342 10.883084292133237 +20170101 control 342 11.782312725299949 +20170101 test 343 10.898039880620946 +20170101 control 343 10.51742730668986 +20170101 test 344 11.368678974765356 +20170101 control 344 11.0742833523871 +20170101 test 345 8.454167820709758 +20170101 control 345 9.605977061614002 +20170101 test 346 11.288410383512236 +20170101 control 346 11.218794405862413 +20170101 test 347 11.416435617345082 +20170101 control 347 8.903303561734443 +20170101 test 348 10.25053407470686 +20170101 control 348 11.17241568237311 +20170101 test 349 12.062324435884651 +20170101 control 349 9.931162570930224 +20170101 test 350 8.962180521994561 +20170101 control 350 10.587721956231967 +20170101 test 351 9.671402295797494 +20170101 control 351 9.787252072116964 +20170101 test 352 11.32656503802121 +20170101 control 352 10.969278659848898 +20170101 test 353 9.37905545689564 +20170101 control 353 10.437204672198849 +20170101 test 354 8.633191480976699 +20170101 control 354 10.440211779848616 +20170101 test 355 9.067246415853056 +20170101 control 355 10.31017370041462 +20170101 test 356 9.988300436702978 +20170101 control 356 12.15441818463062 +20170101 test 357 9.973520174645834 +20170101 control 357 11.565269044921367 +20170101 test 358 10.024430324781347 +20170101 control 358 8.661618960911383 +20170101 test 359 10.13947053383789 +20170101 control 359 10.408741674218229 +20170101 test 360 8.929301581758866 +20170101 control 360 9.83572186598723 +20170101 test 361 7.854555405021152 +20170101 control 361 10.35604290943272 +20170101 test 362 9.579996511374441 +20170101 control 362 10.438365155708453 +20170101 test 363 10.519000972398718 +20170101 control 363 10.021179962433608 +20170101 test 364 11.776479919578312 +20170101 control 364 8.856691694458481 +20170101 test 365 9.994468809541829 +20170101 control 365 9.6509273228682 +20170101 test 366 10.341932982150961 +20170101 control 366 10.589963470043102 +20170101 test 367 8.948936335710243 +20170101 control 367 10.44332995655543 +20170101 test 368 9.298629062568713 +20170101 control 368 9.671261927536497 +20170101 test 369 9.200865894521398 +20170101 control 369 7.772058060144314 +20170101 test 370 9.153530850829544 +20170101 control 370 10.079638463584072 +20170101 test 371 9.776313381854944 +20170101 control 371 8.034643171160198 +20170101 test 372 9.010707626788072 +20170101 control 372 12.095908547820398 +20170101 test 373 11.188860230519811 +20170101 control 373 10.08508708742979 +20170101 test 374 8.677590101167441 +20170101 control 374 10.360216546206164 +20170101 test 375 9.604326742829608 +20170101 control 375 11.559576572818635 +20170101 test 376 10.469290914228525 +20170101 control 376 12.358734571245414 +20170101 test 377 10.467085838652114 +20170101 control 377 10.411880646096835 +20170101 test 378 10.460153627879091 +20170101 control 378 9.905984181187073 +20170101 test 379 11.159196693869548 +20170101 control 379 11.04450065187599 +20170101 test 380 9.521468314183416 +20170101 control 380 9.295342376118242 +20170101 test 381 10.181162253716142 +20170101 control 381 10.637906311786898 +20170101 test 382 9.39287821965262 +20170101 control 382 10.754058866629359 +20170101 test 383 9.49623534540542 +20170101 control 383 9.661395698430255 +20170101 test 384 11.428044105887748 +20170101 control 384 11.477405579028224 +20170101 test 385 9.676797560014034 +20170101 control 385 10.330410406283695 +20170101 test 386 10.084486085244459 +20170101 control 386 10.963495244263445 +20170101 test 387 10.624888673313349 +20170101 control 387 10.33334382293114 +20170101 test 388 11.00411210838816 +20170101 control 388 9.00734562916567 +20170101 test 389 10.542067202834644 +20170101 control 389 8.290741118142655 +20170101 test 390 8.739854272456698 +20170101 control 390 7.7272633011731235 +20170101 test 391 10.212582795066814 +20170101 control 391 11.553550526358697 +20170101 test 392 9.40285500503088 +20170101 control 392 10.171719772884288 +20170101 test 393 10.446397539193864 +20170101 control 393 9.851153631435102 +20170101 test 394 8.259290895784943 +20170101 control 394 9.360865621070367 +20170101 test 395 9.462067389310004 +20170101 control 395 9.559799200720638 +20170101 test 396 9.823007315106828 +20170101 control 396 7.754806291569447 +20170101 test 397 9.012087200403258 +20170101 control 397 10.457750968114208 +20170101 test 398 9.392086932082469 +20170101 control 398 10.085361153535347 +20170101 test 399 8.10881767848143 +20170101 control 399 10.921620898168364 +20170101 test 400 8.706679759908866 +20170101 control 400 10.93197892636715 +20170101 test 401 9.64602232245326 +20170101 control 401 10.393294614282821 +20170101 test 402 9.891060509889291 +20170101 control 402 9.552555921755499 +20170101 test 403 8.874690758690987 +20170101 control 403 11.201545224690106 +20170101 test 404 7.823235856918295 +20170101 control 404 10.68161027558364 +20170101 test 405 11.692295947682812 +20170101 control 405 9.698723586009255 +20170101 test 406 10.288771319910552 +20170101 control 406 10.57793740675119 +20170101 test 407 9.459667574031103 +20170101 control 407 10.497218249443208 +20170101 test 408 9.246694857101527 +20170101 control 408 10.927270673242958 +20170101 test 409 9.038541895152866 +20170101 control 409 10.43754725254509 +20170101 test 410 9.007633502660841 +20170101 control 410 9.36928924112041 +20170101 test 411 11.286410661148116 +20170101 control 411 10.861023075386719 +20170101 test 412 8.51276651926927 +20170101 control 412 11.649896109742846 +20170101 test 413 8.830649849245477 +20170101 control 413 10.842983246903959 +20170101 test 414 11.558619647788994 +20170101 control 414 10.404901495856732 +20170101 test 415 9.998388373880626 +20170101 control 415 9.623189385813841 +20170101 test 416 10.906122226211558 +20170101 control 416 9.86582221747857 +20170101 test 417 9.703897064675534 +20170101 control 417 9.570177627617007 +20170101 test 418 9.785036829204374 +20170101 control 418 9.334267048506772 +20170101 test 419 9.619438953778571 +20170101 control 419 9.570571347668656 +20170101 test 420 10.731882621896544 +20170101 control 420 11.367387711373961 +20170101 test 421 8.84462370286822 +20170101 control 421 10.445765756985917 +20170101 test 422 10.255661443684556 +20170101 control 422 10.763612429392689 +20170101 test 423 8.829491555046419 +20170101 control 423 10.399855951045007 +20170101 test 424 10.031180212017029 +20170101 control 424 9.491457833812815 +20170101 test 425 9.220813555744689 +20170101 control 425 10.43904504333973 +20170101 test 426 7.579131524791517 +20170101 control 426 9.934582976321053 +20170101 test 427 8.474689997007768 +20170101 control 427 10.93600595274251 +20170101 test 428 9.621865689122691 +20170101 control 428 12.055859543541295 +20170101 test 429 10.375242450028324 +20170101 control 429 8.596431742614339 +20170101 test 430 12.294637149458685 +20170101 control 430 8.826126242688412 +20170101 test 431 9.695434215498372 +20170101 control 431 10.498719179280126 +20170101 test 432 10.126021783590144 +20170101 control 432 10.648019038123753 +20170101 test 433 10.509591448794733 +20170101 control 433 7.27934720732644 +20170101 test 434 11.091788126139196 +20170101 control 434 9.298399690319787 +20170101 test 435 10.198957889497317 +20170101 control 435 10.553401297989636 +20170101 test 436 10.989844326786754 +20170101 control 436 9.00464855778045 +20170101 test 437 9.379485314093122 +20170101 control 437 11.249542501232046 +20170101 test 438 10.258617651801458 +20170101 control 438 10.476147078014863 +20170101 test 439 10.174066220608395 +20170101 control 439 11.069281411052955 +20170101 test 440 11.462676978687144 +20170101 control 440 8.440101832100714 +20170101 test 441 7.88384158670681 +20170101 control 441 12.337372242973412 +20170101 test 442 9.272946224719522 +20170101 control 442 7.239654347467059 +20170101 test 443 10.55531431898964 +20170101 control 443 10.56969568962366 +20170101 test 444 9.279063004070641 +20170101 control 444 8.933530060114137 +20170101 test 445 9.653196295925058 +20170101 control 445 10.371488014271382 +20170101 test 446 11.302973787230346 +20170101 control 446 11.120427765633385 +20170101 test 447 9.124908949719215 +20170101 control 447 11.561308265974219 +20170101 test 448 10.246258716568597 +20170101 control 448 9.58235513966726 +20170101 test 449 11.020909549696858 +20170101 control 449 9.599203831703504 +20170101 test 450 11.05641494122689 +20170101 control 450 11.841139689795208 +20170101 test 451 8.551426493923096 +20170101 control 451 9.511063641039545 +20170101 test 452 9.878320367829044 +20170101 control 452 10.081787618902354 +20170101 test 453 11.463690471940271 +20170101 control 453 10.6044179203205 +20170101 test 454 9.364325561706126 +20170101 control 454 9.248789419838985 +20170101 test 455 8.876778139123367 +20170101 control 455 9.250188919709817 +20170101 test 456 9.81394258929317 +20170101 control 456 10.167246151903868 +20170101 test 457 10.638608395443402 +20170101 control 457 11.859999928972142 +20170101 test 458 10.018980272762457 +20170101 control 458 9.670549947580414 +20170101 test 459 10.24455464239024 +20170101 control 459 10.368810475304779 +20170101 test 460 10.376184884562992 +20170101 control 460 11.393963212072666 +20170101 test 461 11.303967664578103 +20170101 control 461 11.215148094123661 +20170101 test 462 9.073077825370348 +20170101 control 462 9.598291426534164 +20170101 test 463 9.070772351802967 +20170101 control 463 8.578660666153775 +20170101 test 464 8.723092398225331 +20170101 control 464 8.864621147809455 +20170101 test 465 11.267680970444843 +20170101 control 465 10.551324185294828 +20170101 test 466 9.742509851027432 +20170101 control 466 10.676746631263285 +20170101 test 467 9.072488093666006 +20170101 control 467 10.113236141966567 +20170101 test 468 11.778914681983025 +20170101 control 468 11.29715330209169 +20170101 test 469 9.521306704616178 +20170101 control 469 12.477437969457728 +20170101 test 470 10.317225007273821 +20170101 control 470 10.890838302113332 +20170101 test 471 12.122064275200131 +20170101 control 471 10.432491689263001 +20170101 test 472 10.80152357769918 +20170101 control 472 10.0371021704179 +20170101 test 473 9.715461300137607 +20170101 control 473 10.853192075975963 +20170101 test 474 8.475270886355696 +20170101 control 474 10.416679950433194 +20170101 test 475 9.901019141990249 +20170101 control 475 9.384353485361505 +20170101 test 476 9.669157324714076 +20170101 control 476 9.833847055484062 +20170101 test 477 10.90200430550327 +20170101 control 477 10.013476882570329 +20170101 test 478 10.94694263111137 +20170101 control 478 9.349061989677141 +20170101 test 479 10.936614967004784 +20170101 control 479 10.379987602145967 +20170101 test 480 10.344374902433522 +20170101 control 480 10.440397823095415 +20170101 test 481 9.416619233462171 +20170101 control 481 11.077892293423101 +20170101 test 482 8.88788333721517 +20170101 control 482 10.838673921024052 +20170101 test 483 10.343266031908476 +20170101 control 483 9.905848161056332 +20170101 test 484 8.602727697833412 +20170101 control 484 11.074659176454443 +20170101 test 485 9.7255781981743 +20170101 control 485 10.243938367884514 +20170101 test 486 11.357039703143164 +20170101 control 486 10.798707097172342 +20170101 test 487 10.645881971640724 +20170101 control 487 10.558556879834493 +20170101 test 488 11.79293476767306 +20170101 control 488 10.01758998709258 +20170101 test 489 9.258739629555333 +20170101 control 489 11.128332890039568 +20170101 test 490 8.735746877782237 +20170101 control 490 11.783208572318536 +20170101 test 491 7.497720533003743 +20170101 control 491 9.32856825441866 +20170101 test 492 11.80565502236493 +20170101 control 492 9.238220771185354 +20170101 test 493 8.38685795135974 +20170101 control 493 11.06353889319077 +20170101 test 494 10.2784890671281 +20170101 control 494 10.3698210359649 +20170101 test 495 10.947244216043243 +20170101 control 495 9.355277350143544 +20170101 test 496 8.641587896576052 +20170101 control 496 9.94292153799467 +20170101 test 497 9.928516098247417 +20170101 control 497 8.542240949990678 +20170101 test 498 11.667876883807741 +20170101 control 498 9.44550442624052 +20170101 test 499 9.217418121381732 +20170101 control 499 9.16947259506309 +20170101 test 500 11.283638057141303 +20170101 control 500 10.865737474797905 +20170101 test 501 10.066741578199665 +20170101 control 501 9.808260287731718 +20170101 test 502 10.595437411195297 +20170101 control 502 9.604064687339479 +20170101 test 503 10.431265120696892 +20170101 control 503 10.733188691222132 +20170101 test 504 12.047516469209278 +20170101 control 504 8.696858490110467 +20170101 test 505 10.966722378781245 +20170101 control 505 9.681260861944125 +20170101 test 506 9.816909591812822 +20170101 control 506 9.377154222446196 +20170101 test 507 10.266415307428751 +20170101 control 507 11.052372833119923 +20170101 test 508 9.560152381040549 +20170101 control 508 9.413679016936294 +20170101 test 509 9.78995299507474 +20170101 control 509 8.832805321373844 +20170101 test 510 9.65924981742858 +20170101 control 510 11.15207789709064 +20170101 test 511 9.075082667625665 +20170101 control 511 9.926297484053777 +20170101 test 512 10.60389398137615 +20170101 control 512 10.888507998751734 +20170101 test 513 10.278108368476547 +20170101 control 513 8.646278780825211 +20170101 test 514 10.022365754805431 +20170101 control 514 10.262125243000128 +20170101 test 515 11.72265701725018 +20170101 control 515 11.138292988564942 +20170101 test 516 10.79009869213482 +20170101 control 516 9.561357414093216 +20170101 test 517 9.296144577224782 +20170101 control 517 8.967912680149354 +20170101 test 518 11.246710787766457 +20170101 control 518 8.774668978145577 +20170101 test 519 11.038511203065683 +20170101 control 519 10.20010733548542 +20170101 test 520 9.101239216114541 +20170101 control 520 10.044601452704205 +20170101 test 521 9.04773402843838 +20170101 control 521 8.756851024387815 +20170101 test 522 8.7751345744825 +20170101 control 522 9.375139802982437 +20170101 test 523 10.971467289192693 +20170101 control 523 10.22786202013423 +20170101 test 524 10.642780649012634 +20170101 control 524 9.044265345608316 +20170101 test 525 9.649329624046626 +20170101 control 525 10.402833089183424 +20170101 test 526 10.790564447606778 +20170101 control 526 9.065831199726187 +20170101 test 527 9.868638317703669 +20170101 control 527 8.494789900156668 +20170101 test 528 9.106214111669317 +20170101 control 528 11.348685247358063 +20170101 test 529 11.093149193102757 +20170101 control 529 9.236591277726438 +20170101 test 530 8.654877126875633 +20170101 control 530 10.77339753168944 +20170101 test 531 11.695153746142394 +20170101 control 531 10.602558661485894 +20170101 test 532 9.889881464555152 +20170101 control 532 10.439671048368597 +20170101 test 533 10.717539466656145 +20170101 control 533 10.8669406581254 +20170101 test 534 8.368182393686977 +20170101 control 534 10.707814887192404 +20170101 test 535 10.982059997836469 +20170101 control 535 9.03104647450839 +20170101 test 536 10.924020820688431 +20170101 control 536 10.866424763362843 +20170101 test 537 10.40323032682536 +20170101 control 537 10.228345002511887 +20170101 test 538 9.985934536849085 +20170101 control 538 8.90615611438787 +20170101 test 539 10.73515735906549 +20170101 control 539 11.828225005474502 +20170101 test 540 8.534550603931425 +20170101 control 540 9.486375092386856 +20170101 test 541 8.2038840275091 +20170101 control 541 10.450470683750506 +20170101 test 542 10.434479504803997 +20170101 control 542 10.179143354692917 +20170101 test 543 9.955170516179752 +20170101 control 543 8.580080051641295 +20170101 test 544 9.315105387096814 +20170101 control 544 10.124782365629935 +20170101 test 545 9.641130537480594 +20170101 control 545 8.934651892462941 +20170101 test 546 10.258967287412505 +20170101 control 546 9.515718253532983 +20170101 test 547 10.439962048215506 +20170101 control 547 10.275659524220057 +20170101 test 548 10.078246068150552 +20170101 control 548 9.497863717396143 +20170101 test 549 10.020877173544804 +20170101 control 549 11.078607626534366 +20170101 test 550 8.685493156808834 +20170101 control 550 10.672736854188381 +20170101 test 551 10.429667827639214 +20170101 control 551 8.569823267058586 +20170101 test 552 8.804891707596408 +20170101 control 552 10.032059921287305 +20170101 test 553 10.566099497122222 +20170101 control 553 11.350068511875218 +20170101 test 554 10.312708474336675 +20170101 control 554 10.512746506435198 +20170101 test 555 8.95968048782493 +20170101 control 555 10.704798878066434 +20170101 test 556 10.36378319695568 +20170101 control 556 11.403208151984877 +20170101 test 557 9.597668842357985 +20170101 control 557 11.792359748621404 +20170101 test 558 10.24733140515443 +20170101 control 558 8.966927224367886 +20170101 test 559 8.829152747041956 +20170101 control 559 9.620286936230324 +20170101 test 560 9.881712768417447 +20170101 control 560 9.284295026840315 +20170101 test 561 9.79704859356731 +20170101 control 561 10.472887040098822 +20170101 test 562 9.694257954100374 +20170101 control 562 10.996362498211008 +20170101 test 563 8.808195788741699 +20170101 control 563 11.714059052145856 +20170101 test 564 10.21533891121806 +20170101 control 564 8.969264681611927 +20170101 test 565 9.123868347097616 +20170101 control 565 9.223417021379035 +20170101 test 566 10.934758318216824 +20170101 control 566 10.640251417151095 +20170101 test 567 9.547785817089215 +20170101 control 567 10.321965046037736 +20170101 test 568 9.816758443425643 +20170101 control 568 8.389915154158272 +20170101 test 569 8.82709118713981 +20170101 control 569 11.997894511784418 +20170101 test 570 10.623142036132625 +20170101 control 570 13.000998653424475 +20170101 test 571 10.135479567983769 +20170101 control 571 9.488853943769431 +20170101 test 572 9.98776347152708 +20170101 control 572 10.2954207153277 +20170101 test 573 9.782218591793981 +20170101 control 573 10.967903592637043 +20170101 test 574 9.445955445164962 +20170101 control 574 8.904229184453996 +20170101 test 575 9.770354616993009 +20170101 control 575 11.579645725945426 +20170101 test 576 9.418003682723356 +20170101 control 576 11.960444197564545 +20170101 test 577 10.485300249301323 +20170101 control 577 9.395434437231483 +20170101 test 578 10.672437041093639 +20170101 control 578 11.071908526793571 +20170101 test 579 9.617454281870407 +20170101 control 579 8.838640092908186 +20170101 test 580 9.765337659400613 +20170101 control 580 9.86058915082299 +20170101 test 581 9.373802045433777 +20170101 control 581 10.245226933847029 +20170101 test 582 9.844138023257292 +20170101 control 582 9.79022913038607 +20170101 test 583 9.675687315936184 +20170101 control 583 10.749528373305521 +20170101 test 584 9.807315756861831 +20170101 control 584 11.325364922717013 +20170101 test 585 10.657573318963767 +20170101 control 585 11.089830927067299 +20170101 test 586 10.329972511120745 +20170101 control 586 10.043201923846208 +20170101 test 587 10.61609685835999 +20170101 control 587 10.195772270418747 +20170101 test 588 11.62476197677297 +20170101 control 588 9.459551555890162 +20170101 test 589 9.953064923441978 +20170101 control 589 13.006797653120849 +20170101 test 590 11.433444330203816 +20170101 control 590 11.390994780810919 +20170101 test 591 9.021242187182457 +20170101 control 591 9.951649292073023 +20170101 test 592 9.167945113740648 +20170101 control 592 11.330909719466748 +20170101 test 593 9.906912039059032 +20170101 control 593 8.33267983485274 +20170101 test 594 11.075795045330597 +20170101 control 594 8.463355475975474 +20170101 test 595 10.426415452235307 +20170101 control 595 10.650288858393724 +20170101 test 596 9.87884691606594 +20170101 control 596 9.835208886084805 +20170101 test 597 9.53251893603889 +20170101 control 597 10.880099641540422 +20170101 test 598 9.872101233286225 +20170101 control 598 11.105715596125346 +20170101 test 599 10.956014282533843 +20170101 control 599 10.313836236753362 +20170101 test 600 12.139293361090543 +20170101 control 600 10.432184091751777 +20170101 test 601 11.364394465132058 +20170101 control 601 10.042792754145077 +20170101 test 602 12.095426931184505 +20170101 control 602 10.648090820094291 +20170101 test 603 10.83005159209161 +20170101 control 603 9.965450575406267 +20170101 test 604 8.691458468764292 +20170101 control 604 8.892150252053428 +20170101 test 605 12.291787928412955 +20170101 control 605 11.242235261577537 +20170101 test 606 10.61272707585233 +20170101 control 606 9.557427889408613 +20170101 test 607 10.747672642732827 +20170101 control 607 8.463510962078589 +20170101 test 608 11.300937875760328 +20170101 control 608 10.187509191868637 +20170101 test 609 10.408442274588605 +20170101 control 609 9.411888355329054 +20170101 test 610 11.027781348156253 +20170101 control 610 9.861313664774324 +20170101 test 611 9.215555738857375 +20170101 control 611 11.104940599039784 +20170101 test 612 8.355659352646622 +20170101 control 612 9.789517779341951 +20170101 test 613 9.18275094879598 +20170101 control 613 8.960081748269717 +20170101 test 614 9.116317495567959 +20170101 control 614 10.602275036299378 +20170101 test 615 8.949954621796346 +20170101 control 615 10.126652759783841 +20170101 test 616 10.026324990681854 +20170101 control 616 12.413149219721394 +20170101 test 617 10.2824579242825 +20170101 control 617 10.93624064382429 +20170101 test 618 10.259857413455775 +20170101 control 618 9.936888367747372 +20170101 test 619 9.914978490823922 +20170101 control 619 9.767093265265993 +20170101 test 620 9.377330678557701 +20170101 control 620 9.837821343354598 +20170101 test 621 10.040528184034784 +20170101 control 621 10.376984561358155 +20170101 test 622 9.501125370996803 +20170101 control 622 9.58931513300563 +20170101 test 623 9.719930216617012 +20170101 control 623 8.400074484597473 +20170101 test 624 9.342546805828654 +20170101 control 624 10.944403498995845 +20170101 test 625 10.568058981774385 +20170101 control 625 10.845510623412277 +20170101 test 626 10.210816932137098 +20170101 control 626 10.21780486393783 +20170101 test 627 10.144898633250486 +20170101 control 627 10.706482335069563 +20170101 test 628 10.06811963326046 +20170101 control 628 9.271692847154961 +20170101 test 629 9.168963715452573 +20170101 control 629 10.548109329647898 +20170101 test 630 10.227276403553375 +20170101 control 630 11.819439239989892 +20170101 test 631 10.67798693211855 +20170101 control 631 9.907739352740602 +20170101 test 632 10.431503170744431 +20170101 control 632 10.513458039792877 +20170101 test 633 9.487989709144788 +20170101 control 633 10.165203818261077 +20170101 test 634 9.988310076545895 +20170101 control 634 8.281874876155955 +20170101 test 635 8.937367890559536 +20170101 control 635 10.801972473634367 +20170101 test 636 10.347166711075085 +20170101 control 636 10.012706937914116 +20170101 test 637 9.99631399619703 +20170101 control 637 9.914650603307974 +20170101 test 638 7.887109194679841 +20170101 control 638 9.84527244415574 +20170101 test 639 10.784231229296962 +20170101 control 639 9.668097194880692 +20170101 test 640 11.294942512796691 +20170101 control 640 10.749325768071895 +20170101 test 641 9.385624302505734 +20170101 control 641 11.164649565077184 +20170101 test 642 9.95580737099436 +20170101 control 642 10.447701399874136 +20170101 test 643 8.798815335315858 +20170101 control 643 12.302890732546437 +20170101 test 644 10.9265636404504 +20170101 control 644 9.91831914394831 +20170101 test 645 11.432114262468787 +20170101 control 645 10.180623681831674 +20170101 test 646 8.867800560080713 +20170101 control 646 12.7750915674569 +20170101 test 647 9.220854096537073 +20170101 control 647 11.427728504710348 +20170101 test 648 9.122535786235998 +20170101 control 648 9.803581118168717 +20170101 test 649 11.24194283373842 +20170101 control 649 8.427903024401779 +20170101 test 650 8.973431937970528 +20170101 control 650 9.587193279521477 +20170101 test 651 11.375411520149987 +20170101 control 651 10.697048554577473 +20170101 test 652 9.132747510382469 +20170101 control 652 11.841629624953901 +20170101 test 653 10.961629945565408 +20170101 control 653 10.90080092146465 +20170101 test 654 11.330997277400034 +20170101 control 654 10.37015173785771 +20170101 test 655 11.543312050421838 +20170101 control 655 11.508053336481233 +20170101 test 656 11.625813831678329 +20170101 control 656 10.890998030063644 +20170101 test 657 8.479278048490826 +20170101 control 657 9.350378764376442 +20170101 test 658 10.872565799129463 +20170101 control 658 10.38505013495452 +20170101 test 659 11.722595277384906 +20170101 control 659 9.6415232760352 +20170101 test 660 10.385911267182955 +20170101 control 660 10.772933342897405 +20170101 test 661 8.723748436730812 +20170101 control 661 10.261198612615232 +20170101 test 662 8.94438751834747 +20170101 control 662 10.62892584172847 +20170101 test 663 9.590292759488843 +20170101 control 663 10.209742310380115 +20170101 test 664 9.996403759514548 +20170101 control 664 9.60055231172136 +20170101 test 665 9.946527231878749 +20170101 control 665 10.388044509436183 +20170101 test 666 10.099979649761867 +20170101 control 666 8.56792484244315 +20170101 test 667 9.710880334367257 +20170101 control 667 11.437134102742295 +20170101 test 668 8.567564629652276 +20170101 control 668 10.991505622220933 +20170101 test 669 9.410994474673025 +20170101 control 669 9.504818667882757 +20170101 test 670 9.989307325989406 +20170101 control 670 9.903773952390438 +20170101 test 671 9.988908812801323 +20170101 control 671 10.894626257235155 +20170101 test 672 11.123320090579472 +20170101 control 672 9.595459676721337 +20170101 test 673 10.939354402284703 +20170101 control 673 9.40081790782417 +20170101 test 674 8.377910539994678 +20170101 control 674 10.214203110230795 +20170101 test 675 10.355715014431505 +20170101 control 675 10.938524302260957 +20170101 test 676 10.939570229588995 +20170101 control 676 11.66108939405533 +20170101 test 677 13.049384764338026 +20170101 control 677 10.825124074860584 +20170101 test 678 10.288616667891258 +20170101 control 678 10.83004273611929 +20170101 test 679 10.728424093746373 +20170101 control 679 9.720794607752756 +20170101 test 680 10.42706884128724 +20170101 control 680 10.90388443948235 +20170101 test 681 9.749957424233644 +20170101 control 681 10.824799747932563 +20170101 test 682 9.729855002181562 +20170101 control 682 11.278280300561637 +20170101 test 683 9.14044277272956 +20170101 control 683 11.014662197819055 +20170101 test 684 9.636782022004711 +20170101 control 684 10.586012574216074 +20170101 test 685 10.26663261460577 +20170101 control 685 9.263717099823754 +20170101 test 686 9.664570405902241 +20170101 control 686 10.387654873098981 +20170101 test 687 8.247646213151828 +20170101 control 687 10.757619307672384 +20170101 test 688 9.375143438553328 +20170101 control 688 9.289343025271426 +20170101 test 689 9.19752427495193 +20170101 control 689 10.408461462267043 +20170101 test 690 11.142013567022858 +20170101 control 690 9.128820215373242 +20170101 test 691 9.584340095218353 +20170101 control 691 10.069193664927822 +20170101 test 692 9.855201399194979 +20170101 control 692 9.24633538485338 +20170101 test 693 10.77234200931022 +20170101 control 693 10.013458507683682 +20170101 test 694 9.092469331235854 +20170101 control 694 9.538024287499363 +20170101 test 695 9.003088408044064 +20170101 control 695 11.438481056185301 +20170101 test 696 11.00009420863133 +20170101 control 696 8.048588981600183 +20170101 test 697 10.30050169502788 +20170101 control 697 12.355436750191062 +20170101 test 698 10.179641726069496 +20170101 control 698 10.368283697478846 +20170101 test 699 8.086259357720035 +20170101 control 699 10.20709209526196 +20170101 test 700 12.00576925760996 +20170101 control 700 9.162728200410454 +20170101 test 701 10.844687392189138 +20170101 control 701 8.948349416806895 +20170101 test 702 10.481360018897515 +20170101 control 702 8.729788365227579 +20170101 test 703 10.275465202939742 +20170101 control 703 12.685951547832191 +20170101 test 704 11.865696009341148 +20170101 control 704 9.584160178546684 +20170101 test 705 10.32907101629829 +20170101 control 705 8.967890745820013 +20170101 test 706 11.309869906776143 +20170101 control 706 11.67282394745789 +20170101 test 707 10.30554820967705 +20170101 control 707 9.047014179897019 +20170101 test 708 9.469387648245965 +20170101 control 708 9.97826398155068 +20170101 test 709 8.98577627241709 +20170101 control 709 11.851722944865207 +20170101 test 710 10.045778121801098 +20170101 control 710 10.74695423115229 +20170101 test 711 9.98434819451685 +20170101 control 711 9.81378215381362 +20170101 test 712 10.394517468971475 +20170101 control 712 11.277483363537137 +20170101 test 713 9.005319744608222 +20170101 control 713 9.56794933668277 +20170101 test 714 11.098889116919567 +20170101 control 714 9.756937737597235 +20170101 test 715 11.825249249926115 +20170101 control 715 11.495089037222943 +20170101 test 716 9.08044993493522 +20170101 control 716 9.65188460235734 +20170101 test 717 11.40940343435911 +20170101 control 717 10.377929509496468 +20170101 test 718 8.39153268136669 +20170101 control 718 9.599458469735357 +20170101 test 719 10.379533867164321 +20170101 control 719 11.339364449410082 +20170101 test 720 10.75306055699213 +20170101 control 720 11.372989646571451 +20170101 test 721 10.353797362150258 +20170101 control 721 10.611539091753594 +20170101 test 722 9.807856455779428 +20170101 control 722 9.613278458781565 +20170101 test 723 10.497774181635513 +20170101 control 723 10.801916338785098 +20170101 test 724 10.074153284181515 +20170101 control 724 11.055887456480646 +20170101 test 725 10.506723152026527 +20170101 control 725 10.529725770364717 +20170101 test 726 11.689499650488509 +20170101 control 726 10.146857612424954 +20170101 test 727 10.206961432196 +20170101 control 727 11.859907462611368 +20170101 test 728 9.813277074509825 +20170101 control 728 9.872920008140813 +20170101 test 729 10.197966014016876 +20170101 control 729 10.399293791216207 +20170101 test 730 9.741816686458858 +20170101 control 730 10.438670130732339 +20170101 test 731 9.678148380422188 +20170101 control 731 9.33301015511783 +20170101 test 732 9.178218403090277 +20170101 control 732 8.254618785137081 +20170101 test 733 10.463055017135227 +20170101 control 733 7.9390715645595265 +20170101 test 734 9.160242847984676 +20170101 control 734 10.491418516225414 +20170101 test 735 10.521463145773431 +20170101 control 735 10.388051337691765 +20170101 test 736 9.378405886210468 +20170101 control 736 9.318196385463319 +20170101 test 737 9.677445149506397 +20170101 control 737 9.70711570635284 +20170101 test 738 9.89634409168175 +20170101 control 738 11.351760182908244 +20170101 test 739 8.903082075940702 +20170101 control 739 8.361512579393377 +20170101 test 740 10.162172231090985 +20170101 control 740 10.173360690472727 +20170101 test 741 10.124721164024345 +20170101 control 741 10.586767131928083 +20170101 test 742 9.169968898191101 +20170101 control 742 11.666068440066761 +20170101 test 743 9.667652614685819 +20170101 control 743 10.514145116652989 +20170101 test 744 9.532230811939003 +20170101 control 744 9.292629004156561 +20170101 test 745 11.992690784353474 +20170101 control 745 9.708041052790604 +20170101 test 746 11.483857641590191 +20170101 control 746 9.813633993245814 +20170101 test 747 9.462731508993901 +20170101 control 747 10.028615233894906 +20170101 test 748 9.274050664994661 +20170101 control 748 11.349252957116597 +20170101 test 749 11.609438422144098 +20170101 control 749 9.702997203438112 +20170101 test 750 10.490620620399586 +20170101 control 750 9.221680687357354 +20170101 test 751 10.646862025862196 +20170101 control 751 9.677072392164408 +20170101 test 752 12.509021714191082 +20170101 control 752 10.219178677306246 +20170101 test 753 10.041731080013735 +20170101 control 753 10.000029766834897 +20170101 test 754 10.300360301943897 +20170101 control 754 12.358779163599348 +20170101 test 755 9.857111913413483 +20170101 control 755 11.206179660741435 +20170101 test 756 9.377889794425654 +20170101 control 756 10.587816538381304 +20170101 test 757 8.533605249388762 +20170101 control 757 9.867694364639275 +20170101 test 758 10.50440397434886 +20170101 control 758 9.984899364828795 +20170101 test 759 11.046323821664496 +20170101 control 759 10.311689541969276 +20170101 test 760 9.14840012416305 +20170101 control 760 9.915073842975268 +20170101 test 761 9.051954363636124 +20170101 control 761 11.35181651171452 +20170101 test 762 10.534557160637283 +20170101 control 762 10.110527691913427 +20170101 test 763 9.984551629280435 +20170101 control 763 11.526295734762904 +20170101 test 764 9.422641210170227 +20170101 control 764 10.918197009453273 +20170101 test 765 8.667176258923892 +20170101 control 765 8.258895528646725 +20170101 test 766 11.069072322568589 +20170101 control 766 10.785735580284909 +20170101 test 767 11.396631958948653 +20170101 control 767 10.584317062518643 +20170101 test 768 9.275120102835743 +20170101 control 768 9.630246124495308 +20170101 test 769 11.360825162031514 +20170101 control 769 10.071678692884552 +20170101 test 770 9.41577127124556 +20170101 control 770 8.779150722154215 +20170101 test 771 11.497750700657518 +20170101 control 771 9.05626425636076 +20170101 test 772 12.17403829118958 +20170101 control 772 10.14388411049888 +20170101 test 773 10.109016639908027 +20170101 control 773 11.090125301267365 +20170101 test 774 9.058312109571748 +20170101 control 774 10.074817464171177 +20170101 test 775 10.919036209181504 +20170101 control 775 10.073482647804473 +20170101 test 776 9.706373860476212 +20170101 control 776 10.869684810085877 +20170101 test 777 10.891255966529485 +20170101 control 777 10.107932667423354 +20170101 test 778 9.067380155069314 +20170101 control 778 10.52761084445883 +20170101 test 779 9.808575351184825 +20170101 control 779 11.078543592228234 +20170101 test 780 9.591601359685352 +20170101 control 780 10.02331681680986 +20170101 test 781 8.584366717410548 +20170101 control 781 11.129530594134135 +20170101 test 782 9.75582654106256 +20170101 control 782 10.054914444937868 +20170101 test 783 9.56527503555549 +20170101 control 783 9.522098792273226 +20170101 test 784 11.197401651682117 +20170101 control 784 9.56558367613428 +20170101 test 785 10.39920819196053 +20170101 control 785 11.372423488445014 +20170101 test 786 9.053366813998542 +20170101 control 786 10.852870333157464 +20170101 test 787 9.528195858719497 +20170101 control 787 9.380645840894323 +20170101 test 788 10.432609877509416 +20170101 control 788 9.702460754423827 +20170101 test 789 9.873210360362684 +20170101 control 789 10.028498220935797 +20170101 test 790 10.67663227075578 +20170101 control 790 12.696876742051703 +20170101 test 791 9.9505242939861 +20170101 control 791 8.341089901383542 +20170101 test 792 10.826538360675514 +20170101 control 792 11.018189264859688 +20170101 test 793 10.386295887562516 +20170101 control 793 10.550288096459163 +20170101 test 794 9.68093100293791 +20170101 control 794 12.491420974727998 +20170101 test 795 10.59561144316713 +20170101 control 795 10.994766795773442 +20170101 test 796 11.993470640734317 +20170101 control 796 10.365524699389287 +20170101 test 797 11.16926069961927 +20170101 control 797 12.44602553409729 +20170101 test 798 10.60314738765374 +20170101 control 798 8.537153431914032 +20170101 test 799 11.116329805955441 +20170101 control 799 12.13608727321467 +20170101 test 800 8.195149201816085 +20170101 control 800 8.188159319021272 +20170101 test 801 10.537025704851153 +20170101 control 801 8.590357071502131 +20170101 test 802 10.729016383458921 +20170101 control 802 10.62066088586015 +20170101 test 803 9.183988481973987 +20170101 control 803 10.650821076969171 +20170101 test 804 8.718452085633606 +20170101 control 804 11.447912376184112 +20170101 test 805 9.66720161244072 +20170101 control 805 9.86653047232184 +20170101 test 806 10.143394813685736 +20170101 control 806 11.692679229004378 +20170101 test 807 11.191654841857337 +20170101 control 807 8.522855000512413 +20170101 test 808 10.637970979992959 +20170101 control 808 10.4580594549191 +20170101 test 809 11.189352999171222 +20170101 control 809 12.109077241682657 +20170101 test 810 10.130633446771942 +20170101 control 810 10.488770979648912 +20170101 test 811 9.625836975591762 +20170101 control 811 9.56804254671464 +20170101 test 812 9.263893954116858 +20170101 control 812 10.184120105754474 +20170101 test 813 11.303734454752167 +20170101 control 813 10.377123539640545 +20170101 test 814 9.5164703529476 +20170101 control 814 9.066946964976434 +20170101 test 815 9.565891648500426 +20170101 control 815 10.175127662317648 +20170101 test 816 8.243003931057148 +20170101 control 816 10.806506083471685 +20170101 test 817 10.154924230142388 +20170101 control 817 10.121779110798792 +20170101 test 818 10.37981766350427 +20170101 control 818 9.786855010768525 +20170101 test 819 11.453920994978828 +20170101 control 819 9.35428579308695 +20170101 test 820 9.956879161695955 +20170101 control 820 10.259581046235686 +20170101 test 821 11.531313961781887 +20170101 control 821 9.045608952196085 +20170101 test 822 10.94948928670795 +20170101 control 822 9.939161685948477 +20170101 test 823 9.787317513089635 +20170101 control 823 9.297507862165421 +20170101 test 824 8.514018968915822 +20170101 control 824 10.661603682697917 +20170101 test 825 10.622726778514787 +20170101 control 825 7.478197479587161 +20170101 test 826 9.531172792573397 +20170101 control 826 10.027615161859869 +20170101 test 827 11.620585995869616 +20170101 control 827 10.685154020089318 +20170101 test 828 9.470403669699584 +20170101 control 828 10.989755954366538 +20170101 test 829 10.01347462494978 +20170101 control 829 10.266694824188438 +20170101 test 830 10.419822725077804 +20170101 control 830 10.229579060536368 +20170101 test 831 9.150335160881156 +20170101 control 831 10.058673350239578 +20170101 test 832 9.074649597219103 +20170101 control 832 11.762420812469621 +20170101 test 833 9.425671204918089 +20170101 control 833 8.793663299560984 +20170101 test 834 11.288839021046407 +20170101 control 834 9.87220196023651 +20170101 test 835 9.33173691563305 +20170101 control 835 10.447679684555572 +20170101 test 836 10.930228417056115 +20170101 control 836 9.735112053796396 +20170101 test 837 10.935934487705447 +20170101 control 837 10.30317673386172 +20170101 test 838 8.51778241240433 +20170101 control 838 9.16097001504812 +20170101 test 839 9.937230600985925 +20170101 control 839 10.602981619823765 +20170101 test 840 10.440352925704396 +20170101 control 840 10.270951805304357 +20170101 test 841 10.926786668365041 +20170101 control 841 10.104216237808266 +20170101 test 842 10.557566531620536 +20170101 control 842 10.070066409758091 +20170101 test 843 8.798587936882445 +20170101 control 843 8.793683142064532 +20170101 test 844 8.385824719289701 +20170101 control 844 8.092384345580694 +20170101 test 845 9.218058378429841 +20170101 control 845 10.427580471021148 +20170101 test 846 10.347157507385347 +20170101 control 846 11.198994277003365 +20170101 test 847 10.516570278527784 +20170101 control 847 11.47478691685479 +20170101 test 848 10.882821069651479 +20170101 control 848 9.195719505358149 +20170101 test 849 9.532044842844043 +20170101 control 849 9.387988396775164 +20170101 test 850 8.866902398133831 +20170101 control 850 9.04151258877566 +20170101 test 851 11.292346458750034 +20170101 control 851 8.638988451319811 +20170101 test 852 10.469261415017767 +20170101 control 852 11.926386511857544 +20170101 test 853 12.583145656549842 +20170101 control 853 8.899844225936858 +20170101 test 854 11.392623395583737 +20170101 control 854 11.2150869207338 +20170101 test 855 9.1567568230267 +20170101 control 855 11.669742505128983 +20170101 test 856 9.354136187203336 +20170101 control 856 10.272848054210426 +20170101 test 857 10.945518303013046 +20170101 control 857 11.080276775919739 +20170101 test 858 10.46907926947646 +20170101 control 858 8.93010479414152 +20170101 test 859 8.277190635289596 +20170101 control 859 8.456412544356654 +20170101 test 860 9.621906925515965 +20170101 control 860 11.951404294074582 +20170101 test 861 10.462870716315145 +20170101 control 861 10.477078042948088 +20170101 test 862 10.172599103029475 +20170101 control 862 9.52140882992848 +20170101 test 863 10.110620220493901 +20170101 control 863 10.573251340436865 +20170101 test 864 10.553899722421619 +20170101 control 864 10.316875021739103 +20170101 test 865 9.993579411770142 +20170101 control 865 8.28932057121516 +20170101 test 866 10.134803205788282 +20170101 control 866 8.10746714103194 +20170101 test 867 9.741156880869699 +20170101 control 867 10.094586898876441 +20170101 test 868 9.652050998656922 +20170101 control 868 8.835481348432491 +20170101 test 869 11.263258621708536 +20170101 control 869 9.324216488080351 +20170101 test 870 9.932208515934393 +20170101 control 870 11.179290650903262 +20170101 test 871 9.768230355744137 +20170101 control 871 10.505700357480034 +20170101 test 872 10.442081582041485 +20170101 control 872 11.67355573662886 +20170101 test 873 10.425629873133992 +20170101 control 873 9.562865818398674 +20170101 test 874 11.178881365254592 +20170101 control 874 11.053589013218229 +20170101 test 875 10.104742457111136 +20170101 control 875 10.584394777364347 +20170101 test 876 10.311042613399005 +20170101 control 876 10.419886644432149 +20170101 test 877 9.651257227321052 +20170101 control 877 8.879440270268422 +20170101 test 878 10.683964755573156 +20170101 control 878 12.096482875261552 +20170101 test 879 8.734270908552299 +20170101 control 879 12.443624857963153 +20170101 test 880 9.896644851397818 +20170101 control 880 10.92335324133239 +20170101 test 881 12.34094459581031 +20170101 control 881 10.542480061256175 +20170101 test 882 10.9201273108684 +20170101 control 882 10.274424578236136 +20170101 test 883 10.229092058059882 +20170101 control 883 11.779375417286264 +20170101 test 884 10.137869501154054 +20170101 control 884 11.339462620916168 +20170101 test 885 11.99845698122754 +20170101 control 885 11.226785369399684 +20170101 test 886 9.435645535952276 +20170101 control 886 8.946085958968242 +20170101 test 887 9.462797479142747 +20170101 control 887 8.612671937011147 +20170101 test 888 9.970252010395695 +20170101 control 888 10.044361750136634 +20170101 test 889 10.136129143887361 +20170101 control 889 10.306754815110894 +20170101 test 890 9.412487588245943 +20170101 control 890 10.030599372615677 +20170101 test 891 12.520069589972774 +20170101 control 891 10.201727117983753 +20170101 test 892 10.785786272281927 +20170101 control 892 10.147307110857488 +20170101 test 893 10.418557859187144 +20170101 control 893 9.601615698811827 +20170101 test 894 10.485078001179556 +20170101 control 894 12.150710616022673 +20170101 test 895 9.581611959833634 +20170101 control 895 9.62118754013293 +20170101 test 896 11.058666787974731 +20170101 control 896 10.494262795169636 +20170101 test 897 9.293073661602106 +20170101 control 897 9.168059893282125 +20170101 test 898 11.289136190782616 +20170101 control 898 9.819192465796705 +20170101 test 899 11.848605022641474 +20170101 control 899 9.460124079882418 +20170101 test 900 9.67046984413071 +20170101 control 900 10.41443965815764 +20170101 test 901 11.239075247769003 +20170101 control 901 10.288569244960028 +20170101 test 902 9.126527151556763 +20170101 control 902 11.489068515539255 +20170101 test 903 8.130140038060967 +20170101 control 903 9.648606314531442 +20170101 test 904 10.0199752027546 +20170101 control 904 10.174965836130534 +20170101 test 905 9.87628547460536 +20170101 control 905 11.098805043511842 +20170101 test 906 9.109430262145075 +20170101 control 906 10.195738087195684 +20170101 test 907 9.410961804973455 +20170101 control 907 10.76539769730324 +20170101 test 908 9.701643390232322 +20170101 control 908 10.276068978963117 +20170101 test 909 9.753671079716757 +20170101 control 909 11.661544406899441 +20170101 test 910 10.65363745250219 +20170101 control 910 12.383885627942764 +20170101 test 911 10.022861064172776 +20170101 control 911 10.132466542960051 +20170101 test 912 8.946117811888364 +20170101 control 912 9.54670624140057 +20170101 test 913 10.229344087056655 +20170101 control 913 11.887883542515533 +20170101 test 914 9.869090232197134 +20170101 control 914 10.387388717822938 +20170101 test 915 9.848002425865694 +20170101 control 915 10.016717762315105 +20170101 test 916 10.397487239323024 +20170101 control 916 9.699040523705126 +20170101 test 917 9.560595459211324 +20170101 control 917 8.581796958344507 +20170101 test 918 7.989590869474101 +20170101 control 918 10.695392274146876 +20170101 test 919 10.248109305157136 +20170101 control 919 9.915048193198674 +20170101 test 920 9.29537964528936 +20170101 control 920 8.9170233536614 +20170101 test 921 11.27006666254245 +20170101 control 921 10.812044386533934 +20170101 test 922 9.315260152432668 +20170101 control 922 9.624796864826063 +20170101 test 923 8.752089593149625 +20170101 control 923 8.862731012146767 +20170101 test 924 9.53450047914973 +20170101 control 924 12.085144147920854 +20170101 test 925 9.82471546730566 +20170101 control 925 9.471878558784956 +20170101 test 926 10.718203011847027 +20170101 control 926 10.068950761694774 +20170101 test 927 9.801687313459572 +20170101 control 927 10.942976907778215 +20170101 test 928 9.629916866276647 +20170101 control 928 10.220638453403293 +20170101 test 929 9.55938240693021 +20170101 control 929 11.41688527672832 +20170101 test 930 8.341917978183357 +20170101 control 930 11.63227470383992 +20170101 test 931 9.06177208215208 +20170101 control 931 10.451665220674304 +20170101 test 932 9.619162093673644 +20170101 control 932 11.130548346865709 +20170101 test 933 10.626004289149213 +20170101 control 933 8.615989962036707 +20170101 test 934 10.289251732295615 +20170101 control 934 10.115458492898732 +20170101 test 935 11.499506891805337 +20170101 control 935 9.317754433301534 +20170101 test 936 9.541081914389988 +20170101 control 936 12.019114020152259 +20170101 test 937 9.272365388255977 +20170101 control 937 10.242166576702205 +20170101 test 938 10.77545845202802 +20170101 control 938 10.786000984129661 +20170101 test 939 10.596791411551518 +20170101 control 939 11.264931088868188 +20170101 test 940 9.494957610949958 +20170101 control 940 9.505783392038442 +20170101 test 941 11.51081416749609 +20170101 control 941 10.031876447157195 +20170101 test 942 9.92456801698456 +20170101 control 942 11.258311742107 +20170101 test 943 9.182271363196204 +20170101 control 943 10.634340440289618 +20170101 test 944 10.105763208237162 +20170101 control 944 9.559066707765345 +20170101 test 945 12.009660146619307 +20170101 control 945 10.70207048423312 +20170101 test 946 9.779737389748213 +20170101 control 946 9.505379277389954 +20170101 test 947 9.080985644418448 +20170101 control 947 10.528142843881325 +20170101 test 948 10.43856362104169 +20170101 control 948 9.26111083018678 +20170101 test 949 8.309786063322239 +20170101 control 949 9.747372559011705 +20170101 test 950 12.455397995168742 +20170101 control 950 9.844994089023189 +20170101 test 951 8.961443952220112 +20170101 control 951 10.033956736826811 +20170101 test 952 8.75886724279289 +20170101 control 952 7.092268670211469 +20170101 test 953 9.86688866459217 +20170101 control 953 10.150023121199075 +20170101 test 954 10.154280828091974 +20170101 control 954 8.826513595725238 +20170101 test 955 9.095043085387438 +20170101 control 955 7.222653987383785 +20170101 test 956 11.114260828672299 +20170101 control 956 9.241635627956834 +20170101 test 957 9.018608961470878 +20170101 control 957 10.440848531529683 +20170101 test 958 10.089776291825165 +20170101 control 958 9.966386095205236 +20170101 test 959 8.159497417997382 +20170101 control 959 9.019227086698494 +20170101 test 960 11.793539607924576 +20170101 control 960 10.501787336376392 +20170101 test 961 8.831676649502176 +20170101 control 961 11.735833938715993 +20170101 test 962 10.943051620351614 +20170101 control 962 8.780874896659233 +20170101 test 963 9.536747600899925 +20170101 control 963 10.335408711741794 +20170101 test 964 6.461385969974148 +20170101 control 964 10.641032333915721 +20170101 test 965 10.711294142441833 +20170101 control 965 8.948236546799372 +20170101 test 966 10.19498259530505 +20170101 control 966 8.579187708241824 +20170101 test 967 10.06592406276892 +20170101 control 967 8.89909091358044 +20170101 test 968 9.767581811175177 +20170101 control 968 10.709050114240096 +20170101 test 969 11.583088233229766 +20170101 control 969 10.256435115201901 +20170101 test 970 8.4270512965748 +20170101 control 970 10.274704388266152 +20170101 test 971 9.749319852952672 +20170101 control 971 10.041543078130067 +20170101 test 972 9.724666532605008 +20170101 control 972 9.260608556688773 +20170101 test 973 8.81961269204787 +20170101 control 973 11.39668663688623 +20170101 test 974 9.00429767845452 +20170101 control 974 10.816194361766389 +20170101 test 975 11.124177297386353 +20170101 control 975 9.391613862098572 +20170101 test 976 7.721726392201921 +20170101 control 976 10.207014468072266 +20170101 test 977 7.783387632161535 +20170101 control 977 10.193060260262108 +20170101 test 978 9.463027193535979 +20170101 control 978 8.513520079233654 +20170101 test 979 9.942031069711891 +20170101 control 979 8.709934935831027 +20170101 test 980 12.057565822078924 +20170101 control 980 10.429232333509068 +20170101 test 981 7.21837016950775 +20170101 control 981 10.759058658855269 +20170101 test 982 10.275317564993339 +20170101 control 982 9.095943628478569 +20170101 test 983 9.87001383992045 +20170101 control 983 9.278343292562603 +20170101 test 984 10.253655087635526 +20170101 control 984 10.164326422289836 +20170101 test 985 9.443649819816663 +20170101 control 985 11.109928217974016 +20170101 test 986 11.075627977771136 +20170101 control 986 10.364790996792104 +20170101 test 987 9.425165193368619 +20170101 control 987 8.808344724646322 +20170101 test 988 9.397012168891028 +20170101 control 988 8.37848605426422 +20170101 test 989 9.263408944095008 +20170101 control 989 10.88273680569358 +20170101 test 990 9.677278275879083 +20170101 control 990 8.846697131867758 +20170101 test 991 8.543260517439974 +20170101 control 991 9.051188362447586 +20170101 test 992 10.878763910754323 +20170101 control 992 9.837594065903115 +20170101 test 993 11.567161653670343 +20170101 control 993 10.549688944147714 +20170101 test 994 8.915612543694827 +20170101 control 994 10.185772128768106 +20170101 test 995 10.174305254892651 +20170101 control 995 9.31621352169924 +20170101 test 996 10.436479468345762 +20170101 control 996 12.224008314869552 +20170101 test 997 8.8682710237251 +20170101 control 997 10.36551308841424 +20170101 test 998 10.104376843422555 +20170101 control 998 9.653699625654761 +20170101 test 999 9.531134781681008 +20170101 control 999 11.973465361688838 diff --git a/processing/src/main/java/io/druid/query/aggregation/AggregatorUtil.java b/processing/src/main/java/io/druid/query/aggregation/AggregatorUtil.java index 8a7477a7358..833eed38560 100644 --- a/processing/src/main/java/io/druid/query/aggregation/AggregatorUtil.java +++ b/processing/src/main/java/io/druid/query/aggregation/AggregatorUtil.java @@ -82,6 +82,18 @@ public class AggregatorUtil public static final byte QUANTILES_DOUBLES_SKETCH_TO_QUANTILES_CACHE_TYPE_ID = 0x1F; public static final byte QUANTILES_DOUBLES_SKETCH_TO_STRING_CACHE_TYPE_ID = 0x20; + // ArrayOfDoublesSketch aggregator + public static final byte ARRAY_OF_DOUBLES_SKETCH_CACHE_TYPE_ID = 0x21; + public static final byte ARRAY_OF_DOUBLES_SKETCH_SET_OP_CACHE_TYPE_ID = 0x22; + public static final byte ARRAY_OF_DOUBLES_SKETCH_TO_ESTIMATE_CACHE_TYPE_ID = 0x23; + public static final byte ARRAY_OF_DOUBLES_SKETCH_TO_ESTIMATE_AND_BOUNDS_CACHE_TYPE_ID = 0x24; + public static final byte ARRAY_OF_DOUBLES_SKETCH_TO_MEANS_CACHE_TYPE_ID = 0x25; + public static final byte ARRAY_OF_DOUBLES_SKETCH_TO_VARIANCES_CACHE_TYPE_ID = 0x26; + public static final byte ARRAY_OF_DOUBLES_SKETCH_TO_NUM_ENTRIES_CACHE_TYPE_ID = 0x27; + public static final byte ARRAY_OF_DOUBLES_SKETCH_TO_QUANTILES_SKETCH_CACHE_TYPE_ID = 0x28; + public static final byte ARRAY_OF_DOUBLES_SKETCH_T_TEST_CACHE_TYPE_ID = 0x29; + public static final byte ARRAY_OF_DOUBLES_SKETCH_TO_STRING_CACHE_TYPE_ID = 0x2A; + /** * returns the list of dependent postAggregators that should be calculated in order to calculate given postAgg * diff --git a/processing/src/main/java/io/druid/query/search/SearchQueryRunner.java b/processing/src/main/java/io/druid/query/search/SearchQueryRunner.java index 81d4cb1ff3e..27a838f0990 100644 --- a/processing/src/main/java/io/druid/query/search/SearchQueryRunner.java +++ b/processing/src/main/java/io/druid/query/search/SearchQueryRunner.java @@ -40,6 +40,7 @@ import io.druid.segment.BaseFloatColumnValueSelector; import io.druid.segment.BaseLongColumnValueSelector; import io.druid.segment.ColumnValueSelector; import io.druid.segment.DimensionSelector; +import io.druid.segment.DimensionSelectorUtils; import io.druid.segment.Segment; import io.druid.segment.column.ColumnCapabilities; import io.druid.segment.column.ValueType; @@ -126,7 +127,7 @@ public class SearchQueryRunner implements QueryRunner> final Object2IntRBTreeMap set ) { - if (selector != null && !isNilSelector(selector)) { + if (selector != null && !DimensionSelectorUtils.isNilSelector(selector)) { final IndexedInts row = selector.getRow(); for (int i = 0, rowSize = row.size(); i < rowSize; ++i) { final String dimVal = selector.lookupName(row.get(i)); @@ -141,13 +142,6 @@ public class SearchQueryRunner implements QueryRunner> } } - private static boolean isNilSelector(final DimensionSelector selector) - { - return selector.nameLookupPossibleInAdvance() - && selector.getValueCardinality() == 1 - && selector.lookupName(0) == null; - } - public static class LongSearchColumnSelectorStrategy implements SearchColumnSelectorStrategy { diff --git a/processing/src/main/java/io/druid/segment/DimensionSelectorUtils.java b/processing/src/main/java/io/druid/segment/DimensionSelectorUtils.java index 1fbaf72c76a..6901b477424 100644 --- a/processing/src/main/java/io/druid/segment/DimensionSelectorUtils.java +++ b/processing/src/main/java/io/druid/segment/DimensionSelectorUtils.java @@ -270,4 +270,12 @@ public final class DimensionSelectorUtils return constantSelector(extractionFn.apply(value)); } } + + public static boolean isNilSelector(final DimensionSelector selector) + { + return selector.nameLookupPossibleInAdvance() + && selector.getValueCardinality() == 1 + && selector.lookupName(0) == null; + } + }