[ASTERIXDB-3134][STO][COMP] Enable Columnar Filter - Part 1
- user mode changes: no
- storage format changes: no
- interface changes: yes
Details:
This is the first part to enable columnar filters.
In this patch, we extended FunctionCallInformation
to throw different types of warnings.
Change-Id: I8d8e468709956f9a63e5a46569830377282b0905
Reviewed-on: https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/17418
Integration-Tests: Jenkins <jenkins@fulliautomatix.ics.uci.edu>
Tested-by: Jenkins <jenkins@fulliautomatix.ics.uci.edu>
Reviewed-by: Wail Alkowaileet <wael.y.k@gmail.com>
Reviewed-by: Murtadha Hubail <mhubail@apache.org>
diff --git a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/pushdown/ExpectedSchemaNodeToIATypeTranslatorVisitor.java b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/pushdown/ExpectedSchemaNodeToIATypeTranslatorVisitor.java
index c746994..0da2c05 100644
--- a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/pushdown/ExpectedSchemaNodeToIATypeTranslatorVisitor.java
+++ b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/optimizer/rules/pushdown/ExpectedSchemaNodeToIATypeTranslatorVisitor.java
@@ -39,6 +39,7 @@
import org.apache.asterix.optimizer.rules.pushdown.schema.UnionExpectedSchemaNode;
import org.apache.asterix.runtime.projection.DataProjectionInfo;
import org.apache.asterix.runtime.projection.FunctionCallInformation;
+import org.apache.asterix.runtime.projection.ProjectionFiltrationWarningFactoryProvider;
/**
* This visitor translates the {@link IExpectedSchemaNode} to {@link IAType} record.
@@ -109,6 +110,7 @@
}
private FunctionCallInformation createFunctionCallInformation(IExpectedSchemaNode node) {
- return new FunctionCallInformation(node.getFunctionName(), node.getSourceLocation());
+ return new FunctionCallInformation(node.getFunctionName(), node.getSourceLocation(),
+ ProjectionFiltrationWarningFactoryProvider.TYPE_MISMATCH_FACTORY);
}
}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/metadata/schema/UnionSchemaNode.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/metadata/schema/UnionSchemaNode.java
index 4c067bd..eba5ac0 100644
--- a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/metadata/schema/UnionSchemaNode.java
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/metadata/schema/UnionSchemaNode.java
@@ -26,6 +26,7 @@
import org.apache.asterix.column.metadata.PathInfoSerializer;
import org.apache.asterix.column.metadata.schema.primitive.MissingFieldSchemaNode;
+import org.apache.asterix.column.metadata.schema.visitor.SchemaClipperVisitor;
import org.apache.asterix.column.operation.lsm.flush.FlushColumnMetadata;
import org.apache.asterix.column.util.RunLengthIntArray;
import org.apache.asterix.om.types.ATypeTag;
@@ -122,7 +123,7 @@
* This would return any numeric node
*
* @return first numeric node or missing node
- * @see org.apache.asterix.column.operation.query.SchemaClipperVisitor
+ * @see SchemaClipperVisitor
*/
public AbstractSchemaNode getNumericChildOrMissing() {
for (AbstractSchemaNode node : children.values()) {
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/metadata/schema/visitor/SchemaClipperVisitor.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/metadata/schema/visitor/SchemaClipperVisitor.java
new file mode 100644
index 0000000..0771ccb
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/metadata/schema/visitor/SchemaClipperVisitor.java
@@ -0,0 +1,171 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.metadata.schema.visitor;
+
+import java.io.IOException;
+import java.util.Map;
+
+import org.apache.asterix.column.metadata.FieldNamesDictionary;
+import org.apache.asterix.column.metadata.schema.AbstractSchemaNode;
+import org.apache.asterix.column.metadata.schema.ObjectSchemaNode;
+import org.apache.asterix.column.metadata.schema.UnionSchemaNode;
+import org.apache.asterix.column.metadata.schema.collection.AbstractCollectionSchemaNode;
+import org.apache.asterix.column.metadata.schema.primitive.MissingFieldSchemaNode;
+import org.apache.asterix.column.metadata.schema.primitive.PrimitiveSchemaNode;
+import org.apache.asterix.om.types.ARecordType;
+import org.apache.asterix.om.types.ATypeTag;
+import org.apache.asterix.om.types.AUnionType;
+import org.apache.asterix.om.types.AbstractCollectionType;
+import org.apache.asterix.om.types.IAType;
+import org.apache.asterix.om.types.IATypeVisitor;
+import org.apache.asterix.om.types.hierachy.ATypeHierarchy;
+import org.apache.asterix.runtime.projection.FunctionCallInformation;
+import org.apache.hyracks.api.exceptions.IWarningCollector;
+import org.apache.hyracks.api.exceptions.Warning;
+
+public class SchemaClipperVisitor implements IATypeVisitor<AbstractSchemaNode, AbstractSchemaNode> {
+ private final FieldNamesDictionary fieldNamesDictionary;
+ private final IWarningCollector warningCollector;
+ private final Map<String, FunctionCallInformation> functionCallInfoMap;
+
+ public SchemaClipperVisitor(FieldNamesDictionary fieldNamesDictionary,
+ Map<String, FunctionCallInformation> functionCallInfoMap, IWarningCollector warningCollector) {
+ this.fieldNamesDictionary = fieldNamesDictionary;
+ this.functionCallInfoMap = functionCallInfoMap;
+ this.warningCollector = warningCollector;
+ }
+
+ @Override
+ public AbstractSchemaNode visit(ARecordType recordType, AbstractSchemaNode arg) {
+ if (isNotCompatible(recordType, arg)) {
+ return MissingFieldSchemaNode.INSTANCE;
+ }
+
+ String[] fieldNames = recordType.getFieldNames();
+ IAType[] fieldTypes = recordType.getFieldTypes();
+ ObjectSchemaNode objectNode = getActualNode(arg, ATypeTag.OBJECT, ObjectSchemaNode.class);
+
+ ObjectSchemaNode clippedObjectNode = new ObjectSchemaNode();
+ try {
+ for (int i = 0; i < fieldNames.length; i++) {
+ int fieldNameIndex = fieldNamesDictionary.getFieldNameIndex(fieldNames[i]);
+ AbstractSchemaNode child = objectNode.getChild(fieldNameIndex);
+ clippedObjectNode.addChild(fieldNameIndex, fieldTypes[i].accept(this, child));
+ }
+ } catch (IOException e) {
+ throw new IllegalStateException(e);
+ }
+
+ return clippedObjectNode;
+ }
+
+ @Override
+ public AbstractSchemaNode visit(AbstractCollectionType collectionType, AbstractSchemaNode arg) {
+ if (isNotCompatible(collectionType, arg)) {
+ return MissingFieldSchemaNode.INSTANCE;
+ }
+ AbstractCollectionSchemaNode collectionNode =
+ getActualNode(arg, collectionType.getTypeTag(), AbstractCollectionSchemaNode.class);
+ AbstractSchemaNode newItemNode = collectionType.getItemType().accept(this, collectionNode.getItemNode());
+ AbstractCollectionSchemaNode clippedCollectionNode =
+ AbstractCollectionSchemaNode.create(collectionType.getTypeTag());
+ clippedCollectionNode.setItemNode(newItemNode);
+ return clippedCollectionNode;
+ }
+
+ @Override
+ public AbstractSchemaNode visit(AUnionType unionType, AbstractSchemaNode arg) {
+ return arg;
+ }
+
+ @Override
+ public AbstractSchemaNode visitFlat(IAType flatType, AbstractSchemaNode arg) {
+ if (flatType.getTypeTag() == ATypeTag.ANY) {
+ return arg;
+ } else if (isNotCompatible(flatType, arg)) {
+ return getNonCompatibleNumericNodeIfAny(flatType, arg);
+ }
+ return getActualNode(arg, flatType.getTypeTag(), PrimitiveSchemaNode.class);
+ }
+
+ private AbstractSchemaNode getNonCompatibleNumericNodeIfAny(IAType flatType, AbstractSchemaNode arg) {
+ ATypeHierarchy.Domain requestedDomain = ATypeHierarchy.getTypeDomain(flatType.getTypeTag());
+ ATypeHierarchy.Domain nodeDomain = ATypeHierarchy.getTypeDomain(arg.getTypeTag());
+ if (nodeDomain == requestedDomain && nodeDomain == ATypeHierarchy.Domain.NUMERIC) {
+ // This will be reconciled by the filter accessor
+ return arg;
+ } else if (arg.getTypeTag() == ATypeTag.UNION) {
+ UnionSchemaNode unionNode = (UnionSchemaNode) arg;
+ return unionNode.getNumericChildOrMissing();
+ }
+
+ return MissingFieldSchemaNode.INSTANCE;
+ }
+
+ private <T extends AbstractSchemaNode> T getActualNode(AbstractSchemaNode node, ATypeTag typeTag, Class<T> clazz) {
+ if (node.getTypeTag() == typeTag) {
+ return clazz.cast(node);
+ } else {
+ //Then it is a union (as we check for incompatibility before we call this method)
+ UnionSchemaNode unionNode = (UnionSchemaNode) node;
+ return clazz.cast(unionNode.getChild(typeTag));
+ }
+ }
+
+ private boolean isNotCompatible(IAType requestedType, AbstractSchemaNode schemaNode) {
+ if (requestedType.getTypeTag() != schemaNode.getTypeTag()) {
+ if (schemaNode.getTypeTag() != ATypeTag.UNION) {
+ warn(requestedType, schemaNode);
+ return true;
+ }
+ // Handle union
+ UnionSchemaNode unionNode = (UnionSchemaNode) schemaNode;
+ return notInUnion(requestedType, unionNode);
+ }
+ return unionContainsMultipleNumeric(schemaNode);
+ }
+
+ private boolean notInUnion(IAType requestedType, UnionSchemaNode unionNode) {
+ for (AbstractSchemaNode unionChildNode : unionNode.getChildren().values()) {
+ warn(requestedType, unionChildNode);
+ }
+ return !unionNode.getChildren().containsKey(requestedType.getTypeTag());
+ }
+
+ private void warn(IAType requestedType, AbstractSchemaNode schemaNode) {
+ if (ATypeHierarchy.isCompatible(requestedType.getTypeTag(), schemaNode.getTypeTag())) {
+ return;
+ }
+ if (warningCollector.shouldWarn()) {
+ Warning warning = functionCallInfoMap.get(requestedType.getTypeName())
+ .createWarning(requestedType.getTypeTag(), schemaNode.getTypeTag());
+ if (warning != null) {
+ warningCollector.warn(warning);
+ }
+ }
+ }
+
+ private boolean unionContainsMultipleNumeric(AbstractSchemaNode schemaNode) {
+ if (schemaNode.getTypeTag() == ATypeTag.UNION) {
+ UnionSchemaNode unionNode = (UnionSchemaNode) schemaNode;
+ return unionNode.getNumberOfNumericChildren() > 1;
+ }
+ return false;
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/IColumnValuesReaderFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/IColumnValuesReaderFactory.java
index 98837ac..7c41512 100644
--- a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/IColumnValuesReaderFactory.java
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/IColumnValuesReaderFactory.java
@@ -21,6 +21,7 @@
import java.io.DataInput;
import java.io.IOException;
+import org.apache.asterix.column.metadata.PathInfoSerializer;
import org.apache.asterix.om.types.ATypeTag;
public interface IColumnValuesReaderFactory {
@@ -51,6 +52,7 @@
*
* @param input column metadata info
* @return columnar reader
+ * @see PathInfoSerializer#writePathInfo(ATypeTag, int, boolean) for more information on how the path info is serialized
*/
IColumnValuesReader createValueReader(DataInput input) throws IOException;
}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/FilterAccessorProvider.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/FilterAccessorProvider.java
new file mode 100644
index 0000000..2c8cf19
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/FilterAccessorProvider.java
@@ -0,0 +1,108 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter;
+
+import java.nio.ByteBuffer;
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.asterix.column.metadata.schema.ObjectSchemaNode;
+import org.apache.asterix.column.metadata.schema.primitive.PrimitiveSchemaNode;
+import org.apache.asterix.column.metadata.schema.visitor.PathExtractorVisitor;
+import org.apache.asterix.column.metadata.schema.visitor.SchemaClipperVisitor;
+import org.apache.asterix.column.values.reader.filter.value.ColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.value.NoOpColumnFilterValueAccessor;
+import org.apache.asterix.column.values.writer.filters.AbstractColumnFilterWriter;
+import org.apache.asterix.om.types.ARecordType;
+import org.apache.asterix.om.types.ATypeTag;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+public class FilterAccessorProvider {
+ private final ObjectSchemaNode root;
+ private final ObjectSchemaNode metaRoot;
+ private final SchemaClipperVisitor clipperVisitor;
+ private final PathExtractorVisitor pathExtractorVisitor;
+ private final Map<ARecordType, PrimitiveSchemaNode> cachedNodes;
+ private final List<IColumnFilterValueAccessor> filterAccessors;
+
+ public FilterAccessorProvider(ObjectSchemaNode root, SchemaClipperVisitor clipperVisitor) {
+ this(root, null, clipperVisitor);
+ }
+
+ public FilterAccessorProvider(ObjectSchemaNode root, ObjectSchemaNode metaRoot,
+ SchemaClipperVisitor clipperVisitor) {
+ this.root = root;
+ this.metaRoot = metaRoot;
+ this.clipperVisitor = clipperVisitor;
+ this.pathExtractorVisitor = new PathExtractorVisitor();
+ cachedNodes = new HashMap<>();
+ filterAccessors = new ArrayList<>();
+ }
+
+ public IColumnFilterValueAccessor createColumnFilterValueAccessor(ARecordType path, boolean min)
+ throws HyracksDataException {
+ PrimitiveSchemaNode node = getNode(path);
+ ATypeTag typeTag = node.getTypeTag();
+ if (typeTag == ATypeTag.MISSING) {
+ return NoOpColumnFilterValueAccessor.INSTANCE;
+ }
+ IColumnFilterValueAccessor accessor = new ColumnFilterValueAccessor(node.getColumnIndex(), typeTag, min);
+ filterAccessors.add(accessor);
+ return accessor;
+ }
+
+ public List<IColumnFilterValueAccessor> getFilterAccessors() {
+ return filterAccessors;
+ }
+
+ public static void setFilterValues(List<IColumnFilterValueAccessor> filterValueAccessors, ByteBuffer pageZero,
+ int numberOfColumns) {
+ for (int i = 0; i < filterValueAccessors.size(); i++) {
+ ColumnFilterValueAccessor accessor = (ColumnFilterValueAccessor) filterValueAccessors.get(i);
+ int columnIndex = accessor.getColumnIndex();
+ long normalizedValue;
+ if (columnIndex < numberOfColumns) {
+ int filterOffset = pageZero.position() + columnIndex * AbstractColumnFilterWriter.FILTER_SIZE;
+ normalizedValue =
+ accessor.isMin() ? pageZero.getLong(filterOffset) : pageZero.getLong(filterOffset + Long.BYTES);
+ } else {
+ // Column is missing
+ normalizedValue = accessor.isMin() ? Long.MAX_VALUE : Long.MIN_VALUE;
+ }
+ accessor.setNormalizedValue(normalizedValue);
+ }
+ }
+
+ private PrimitiveSchemaNode getNode(ARecordType path) throws HyracksDataException {
+ PrimitiveSchemaNode node = cachedNodes.get(path);
+ if (node == null) {
+ ObjectSchemaNode dataPath = (ObjectSchemaNode) path.accept(clipperVisitor, root);
+ node = (PrimitiveSchemaNode) dataPath.accept(pathExtractorVisitor, null);
+ if (node.getTypeTag() == ATypeTag.MISSING && metaRoot != null) {
+ //Try meta
+ ObjectSchemaNode metaPath = (ObjectSchemaNode) path.accept(clipperVisitor, metaRoot);
+ node = (PrimitiveSchemaNode) metaPath.accept(pathExtractorVisitor, null);
+ }
+ cachedNodes.put(path, node);
+ }
+ return node;
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterEvaluator.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterEvaluator.java
new file mode 100644
index 0000000..bc1f51c
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterEvaluator.java
@@ -0,0 +1,23 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter;
+
+public interface IColumnFilterEvaluator {
+ boolean evaluate();
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..62abb92
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterEvaluatorFactory.java
@@ -0,0 +1,28 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter;
+
+import java.io.Serializable;
+
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+@FunctionalInterface
+public interface IColumnFilterEvaluatorFactory extends Serializable {
+ IColumnFilterEvaluator create(FilterAccessorProvider filterAccessorProvider) throws HyracksDataException;
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterValueAccessor.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterValueAccessor.java
new file mode 100644
index 0000000..69abb37
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterValueAccessor.java
@@ -0,0 +1,27 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter;
+
+import org.apache.asterix.om.types.ATypeTag;
+
+public interface IColumnFilterValueAccessor {
+ long getNormalizedValue();
+
+ ATypeTag getTypeTag();
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterValueAccessorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterValueAccessorFactory.java
new file mode 100644
index 0000000..9621c02
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/IColumnFilterValueAccessorFactory.java
@@ -0,0 +1,28 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter;
+
+import java.io.Serializable;
+
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+@FunctionalInterface
+public interface IColumnFilterValueAccessorFactory extends Serializable {
+ IColumnFilterValueAccessor create(FilterAccessorProvider filterAccessorProvider) throws HyracksDataException;
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/AbstractColumnFilterComparatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/AbstractColumnFilterComparatorFactory.java
new file mode 100644
index 0000000..8244876
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/AbstractColumnFilterComparatorFactory.java
@@ -0,0 +1,83 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.compartor;
+
+import org.apache.asterix.column.values.reader.filter.FilterAccessorProvider;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluatorFactory;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessorFactory;
+import org.apache.asterix.column.values.reader.filter.evaluator.FalseColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.evaluator.TrueColumnFilterEvaluator;
+import org.apache.asterix.om.types.ATypeTag;
+import org.apache.asterix.om.types.hierachy.ATypeHierarchy;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+abstract class AbstractColumnFilterComparatorFactory implements IColumnFilterEvaluatorFactory {
+ private static final long serialVersionUID = 4229059703449173694L;
+ private final IColumnFilterValueAccessorFactory left;
+ private final IColumnFilterValueAccessorFactory right;
+
+ AbstractColumnFilterComparatorFactory(IColumnFilterValueAccessorFactory left,
+ IColumnFilterValueAccessorFactory right) {
+ this.left = left;
+ this.right = right;
+ }
+
+ @Override
+ public final IColumnFilterEvaluator create(FilterAccessorProvider filterAccessorProvider)
+ throws HyracksDataException {
+ IColumnFilterValueAccessor leftAccessor = left.create(filterAccessorProvider);
+ IColumnFilterValueAccessor rightAccessor = right.create(filterAccessorProvider);
+
+ ATypeTag leftTypeTag = leftAccessor.getTypeTag();
+ ATypeTag rightTypeTag = rightAccessor.getTypeTag();
+ if (leftTypeTag != rightTypeTag && ATypeHierarchy.isCompatible(leftTypeTag, rightTypeTag)) {
+ // Cannot compare comparable values with different types. Bail out.
+ return TrueColumnFilterEvaluator.INSTANCE;
+ } else if (cannotCompare(leftTypeTag, rightTypeTag)) {
+ return FalseColumnFilterEvaluator.INSTANCE;
+ }
+ return createComparator(leftAccessor, rightAccessor);
+ }
+
+ private boolean cannotCompare(ATypeTag leftTypeTag, ATypeTag rightTypeTag) {
+ return rightTypeTag == ATypeTag.MISSING || leftTypeTag != rightTypeTag;
+ }
+
+ protected abstract IColumnFilterEvaluator createComparator(IColumnFilterValueAccessor left,
+ IColumnFilterValueAccessor right);
+
+ protected abstract String getOpt();
+
+ @Override
+ public String toString() {
+ return left.toString() + " " + getOpt() + " " + right.toString();
+ }
+
+ static abstract class AbstractComparator implements IColumnFilterEvaluator {
+ protected final IColumnFilterValueAccessor left;
+ protected final IColumnFilterValueAccessor right;
+
+ AbstractComparator(IColumnFilterValueAccessor left, IColumnFilterValueAccessor right) {
+ this.left = left;
+ this.right = right;
+ }
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/GEColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/GEColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..81d58db
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/GEColumnFilterEvaluatorFactory.java
@@ -0,0 +1,48 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.compartor;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessorFactory;
+
+public class GEColumnFilterEvaluatorFactory extends AbstractColumnFilterComparatorFactory {
+ private static final long serialVersionUID = 6879193736347174789L;
+
+ public GEColumnFilterEvaluatorFactory(IColumnFilterValueAccessorFactory left,
+ IColumnFilterValueAccessorFactory right) {
+ super(left, right);
+ }
+
+ @Override
+ protected IColumnFilterEvaluator createComparator(IColumnFilterValueAccessor left,
+ IColumnFilterValueAccessor right) {
+ return new AbstractComparator(left, right) {
+ @Override
+ public boolean evaluate() {
+ return left.getNormalizedValue() >= right.getNormalizedValue();
+ }
+ };
+ }
+
+ @Override
+ protected String getOpt() {
+ return ">=";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/GTColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/GTColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..b24e18bf
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/GTColumnFilterEvaluatorFactory.java
@@ -0,0 +1,48 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.compartor;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessorFactory;
+
+public class GTColumnFilterEvaluatorFactory extends AbstractColumnFilterComparatorFactory {
+ private static final long serialVersionUID = -3104103170926445020L;
+
+ public GTColumnFilterEvaluatorFactory(IColumnFilterValueAccessorFactory left,
+ IColumnFilterValueAccessorFactory right) {
+ super(left, right);
+ }
+
+ @Override
+ protected IColumnFilterEvaluator createComparator(IColumnFilterValueAccessor left,
+ IColumnFilterValueAccessor right) {
+ return new AbstractComparator(left, right) {
+ @Override
+ public boolean evaluate() {
+ return left.getNormalizedValue() > right.getNormalizedValue();
+ }
+ };
+ }
+
+ @Override
+ protected String getOpt() {
+ return ">";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/LEColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/LEColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..f195d03
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/LEColumnFilterEvaluatorFactory.java
@@ -0,0 +1,48 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.compartor;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessorFactory;
+
+public class LEColumnFilterEvaluatorFactory extends AbstractColumnFilterComparatorFactory {
+ private static final long serialVersionUID = 1068661809768620550L;
+
+ public LEColumnFilterEvaluatorFactory(IColumnFilterValueAccessorFactory left,
+ IColumnFilterValueAccessorFactory right) {
+ super(left, right);
+ }
+
+ @Override
+ protected IColumnFilterEvaluator createComparator(IColumnFilterValueAccessor left,
+ IColumnFilterValueAccessor right) {
+ return new AbstractComparator(left, right) {
+ @Override
+ public boolean evaluate() {
+ return left.getNormalizedValue() <= right.getNormalizedValue();
+ }
+ };
+ }
+
+ @Override
+ protected String getOpt() {
+ return "<=";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/LTColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/LTColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..41c8018
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/compartor/LTColumnFilterEvaluatorFactory.java
@@ -0,0 +1,48 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.compartor;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessorFactory;
+
+public class LTColumnFilterEvaluatorFactory extends AbstractColumnFilterComparatorFactory {
+ private static final long serialVersionUID = -4066709771630858677L;
+
+ public LTColumnFilterEvaluatorFactory(IColumnFilterValueAccessorFactory left,
+ IColumnFilterValueAccessorFactory right) {
+ super(left, right);
+ }
+
+ @Override
+ protected IColumnFilterEvaluator createComparator(IColumnFilterValueAccessor left,
+ IColumnFilterValueAccessor right) {
+ return new AbstractComparator(left, right) {
+ @Override
+ public boolean evaluate() {
+ return left.getNormalizedValue() < right.getNormalizedValue();
+ }
+ };
+ }
+
+ @Override
+ protected String getOpt() {
+ return "<";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/ANDColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/ANDColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..03ae64e
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/ANDColumnFilterEvaluatorFactory.java
@@ -0,0 +1,68 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.evaluator;
+
+import org.apache.asterix.column.values.reader.filter.FilterAccessorProvider;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluatorFactory;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+public class ANDColumnFilterEvaluatorFactory extends AbstractColumnFilterEvaluatorFactory {
+ private static final long serialVersionUID = -7902069740719309586L;
+
+ public ANDColumnFilterEvaluatorFactory(IColumnFilterEvaluatorFactory left, IColumnFilterEvaluatorFactory right) {
+ super(left, right);
+ }
+
+ @Override
+ public IColumnFilterEvaluator create(FilterAccessorProvider filterAccessorProvider) throws HyracksDataException {
+ IColumnFilterEvaluator leftEval = left.create(filterAccessorProvider);
+ IColumnFilterEvaluator rightEval = right.create(filterAccessorProvider);
+ if (leftEval == FalseColumnFilterEvaluator.INSTANCE || rightEval == FalseColumnFilterEvaluator.INSTANCE) {
+ // Either is false, then return false
+ return FalseColumnFilterEvaluator.INSTANCE;
+ } else if (leftEval == TrueColumnFilterEvaluator.INSTANCE && rightEval == TrueColumnFilterEvaluator.INSTANCE) {
+ //Skip both operands and return TrueColumnFilterEvaluator
+ return TrueColumnFilterEvaluator.INSTANCE;
+ } else if (leftEval == TrueColumnFilterEvaluator.INSTANCE) {
+ //Left is true, return the right evaluator
+ return rightEval;
+ } else if (rightEval == TrueColumnFilterEvaluator.INSTANCE) {
+ //Same as above but the right is true
+ return leftEval;
+ } else {
+ //Both are actual filters
+ return create(leftEval, rightEval);
+ }
+ }
+
+ private IColumnFilterEvaluator create(IColumnFilterEvaluator left, IColumnFilterEvaluator right) {
+ return new AbstractEvaluator(left, right) {
+ @Override
+ public boolean evaluate() {
+ return left.evaluate() && right.evaluate();
+ }
+ };
+ }
+
+ @Override
+ protected String getOp() {
+ return "&&";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/AbstractColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/AbstractColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..d1a53d1
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/AbstractColumnFilterEvaluatorFactory.java
@@ -0,0 +1,52 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.evaluator;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluatorFactory;
+
+public abstract class AbstractColumnFilterEvaluatorFactory implements IColumnFilterEvaluatorFactory {
+ private static final long serialVersionUID = 1436531448052787426L;
+
+ protected final IColumnFilterEvaluatorFactory left;
+ protected final IColumnFilterEvaluatorFactory right;
+
+ public AbstractColumnFilterEvaluatorFactory(IColumnFilterEvaluatorFactory left,
+ IColumnFilterEvaluatorFactory right) {
+ this.left = left;
+ this.right = right;
+ }
+
+ protected abstract String getOp();
+
+ @Override
+ public String toString() {
+ return left.toString() + " " + getOp() + " " + right.toString();
+ }
+
+ static abstract class AbstractEvaluator implements IColumnFilterEvaluator {
+ protected final IColumnFilterEvaluator left;
+ protected final IColumnFilterEvaluator right;
+
+ AbstractEvaluator(IColumnFilterEvaluator left, IColumnFilterEvaluator right) {
+ this.left = left;
+ this.right = right;
+ }
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/FalseColumnFilterEvaluator.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/FalseColumnFilterEvaluator.java
new file mode 100644
index 0000000..fccd015
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/FalseColumnFilterEvaluator.java
@@ -0,0 +1,38 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.evaluator;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+
+public class FalseColumnFilterEvaluator implements IColumnFilterEvaluator {
+ public static final IColumnFilterEvaluator INSTANCE = new FalseColumnFilterEvaluator();
+
+ private FalseColumnFilterEvaluator() {
+ }
+
+ @Override
+ public boolean evaluate() {
+ return false;
+ }
+
+ @Override
+ public String toString() {
+ return "FALSE";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/NoOpColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/NoOpColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..765f3c6
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/NoOpColumnFilterEvaluatorFactory.java
@@ -0,0 +1,38 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.evaluator;
+
+import org.apache.asterix.column.values.reader.filter.FilterAccessorProvider;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluatorFactory;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+public class NoOpColumnFilterEvaluatorFactory implements IColumnFilterEvaluatorFactory {
+ private static final long serialVersionUID = -7122361396576592000L;
+ public static final IColumnFilterEvaluatorFactory INSTANCE = new NoOpColumnFilterEvaluatorFactory();
+
+ private NoOpColumnFilterEvaluatorFactory() {
+ }
+
+ @Override
+ public IColumnFilterEvaluator create(FilterAccessorProvider filterAccessorProvider) throws HyracksDataException {
+ // True is also NoOp
+ return TrueColumnFilterEvaluator.INSTANCE;
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/ORColumnFilterEvaluatorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/ORColumnFilterEvaluatorFactory.java
new file mode 100644
index 0000000..aedc9f4
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/ORColumnFilterEvaluatorFactory.java
@@ -0,0 +1,69 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.evaluator;
+
+import org.apache.asterix.column.values.reader.filter.FilterAccessorProvider;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluatorFactory;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+public class ORColumnFilterEvaluatorFactory extends AbstractColumnFilterEvaluatorFactory {
+ private static final long serialVersionUID = 9029706131191375500L;
+
+ public ORColumnFilterEvaluatorFactory(IColumnFilterEvaluatorFactory left, IColumnFilterEvaluatorFactory right) {
+ super(left, right);
+ }
+
+ @Override
+ public IColumnFilterEvaluator create(FilterAccessorProvider filterAccessorProvider) throws HyracksDataException {
+ IColumnFilterEvaluator leftEval = left.create(filterAccessorProvider);
+ IColumnFilterEvaluator rightEval = right.create(filterAccessorProvider);
+ if (leftEval == TrueColumnFilterEvaluator.INSTANCE || rightEval == TrueColumnFilterEvaluator.INSTANCE) {
+ // Either is true, then return true
+ return TrueColumnFilterEvaluator.INSTANCE;
+ } else if (leftEval == FalseColumnFilterEvaluator.INSTANCE
+ && rightEval == FalseColumnFilterEvaluator.INSTANCE) {
+ // Both are false, then return false
+ return FalseColumnFilterEvaluator.INSTANCE;
+ } else if (leftEval == FalseColumnFilterEvaluator.INSTANCE) {
+ //Left is NoOp, which evaluates to true, return the right evaluator
+ return rightEval;
+ } else if (rightEval == FalseColumnFilterEvaluator.INSTANCE) {
+ //Same as above but the right is NoOp
+ return leftEval;
+ } else {
+ //Both are actual evaluators
+ return create(leftEval, rightEval);
+ }
+ }
+
+ private IColumnFilterEvaluator create(IColumnFilterEvaluator left, IColumnFilterEvaluator right) {
+ return new AbstractEvaluator(left, right) {
+ @Override
+ public boolean evaluate() {
+ return left.evaluate() || right.evaluate();
+ }
+ };
+ }
+
+ @Override
+ protected String getOp() {
+ return "||";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/TrueColumnFilterEvaluator.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/TrueColumnFilterEvaluator.java
new file mode 100644
index 0000000..37f1c5c
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/evaluator/TrueColumnFilterEvaluator.java
@@ -0,0 +1,41 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.evaluator;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterEvaluator;
+
+/**
+ * This evaluator is also used to indicate a NoOp filter
+ */
+public class TrueColumnFilterEvaluator implements IColumnFilterEvaluator {
+ public static final IColumnFilterEvaluator INSTANCE = new TrueColumnFilterEvaluator();
+
+ private TrueColumnFilterEvaluator() {
+ }
+
+ @Override
+ public boolean evaluate() {
+ return true;
+ }
+
+ @Override
+ public String toString() {
+ return "TRUE";
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ColumnFilterValueAccessor.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ColumnFilterValueAccessor.java
new file mode 100644
index 0000000..0aa2b31
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ColumnFilterValueAccessor.java
@@ -0,0 +1,57 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.value;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.om.types.ATypeTag;
+
+public class ColumnFilterValueAccessor implements IColumnFilterValueAccessor {
+ private final int columnIndex;
+ private final ATypeTag typeTag;
+ private final boolean min;
+ private long normalizedValue;
+
+ public ColumnFilterValueAccessor(int columnIndex, ATypeTag typeTag, boolean min) {
+ this.columnIndex = columnIndex;
+ this.typeTag = typeTag;
+ this.min = min;
+ }
+
+ public int getColumnIndex() {
+ return columnIndex;
+ }
+
+ public boolean isMin() {
+ return min;
+ }
+
+ public void setNormalizedValue(long normalizedValue) {
+ this.normalizedValue = normalizedValue;
+ }
+
+ @Override
+ public long getNormalizedValue() {
+ return normalizedValue;
+ }
+
+ @Override
+ public ATypeTag getTypeTag() {
+ return typeTag;
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ColumnFilterValueAccessorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ColumnFilterValueAccessorFactory.java
new file mode 100644
index 0000000..fb02321
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ColumnFilterValueAccessorFactory.java
@@ -0,0 +1,57 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.value;
+
+import org.apache.asterix.column.values.reader.filter.FilterAccessorProvider;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessorFactory;
+import org.apache.asterix.om.types.ARecordType;
+import org.apache.asterix.om.types.visitor.PathStringBuilderForIATypeVisitor;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+public class ColumnFilterValueAccessorFactory implements IColumnFilterValueAccessorFactory {
+ private static final long serialVersionUID = -6341611172763952841L;
+ private final ARecordType path;
+ private final boolean min;
+
+ public ColumnFilterValueAccessorFactory(ARecordType path, boolean min) {
+ this.path = path;
+ this.min = min;
+ }
+
+ @Override
+ public IColumnFilterValueAccessor create(FilterAccessorProvider filterAccessorProvider)
+ throws HyracksDataException {
+ return filterAccessorProvider.createColumnFilterValueAccessor(path, min);
+ }
+
+ @Override
+ public String toString() {
+ PathStringBuilderForIATypeVisitor pathBuilder = new PathStringBuilderForIATypeVisitor();
+ StringBuilder stringBuilder = new StringBuilder();
+
+ stringBuilder.append(min ? "min" : "max");
+ stringBuilder.append('(');
+ path.accept(pathBuilder, stringBuilder);
+ stringBuilder.append(')');
+
+ return stringBuilder.toString();
+ }
+
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ConstantColumnFilterValueAccessor.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ConstantColumnFilterValueAccessor.java
new file mode 100644
index 0000000..db9ef9c
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ConstantColumnFilterValueAccessor.java
@@ -0,0 +1,49 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.value;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.om.types.ATypeTag;
+
+public final class ConstantColumnFilterValueAccessor implements IColumnFilterValueAccessor {
+ private final long normalizedValue;
+ private final ATypeTag typeTag;
+
+ //TODO add UUID
+
+ public ConstantColumnFilterValueAccessor(long normalizedValue, ATypeTag typeTag) {
+ this.normalizedValue = normalizedValue;
+ this.typeTag = typeTag;
+ }
+
+ @Override
+ public long getNormalizedValue() {
+ return normalizedValue;
+ }
+
+ @Override
+ public ATypeTag getTypeTag() {
+ return typeTag;
+ }
+
+ @Override
+ public String toString() {
+ return Long.toString(normalizedValue);
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ConstantColumnFilterValueAccessorFactory.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ConstantColumnFilterValueAccessorFactory.java
new file mode 100644
index 0000000..eeb903d
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/ConstantColumnFilterValueAccessorFactory.java
@@ -0,0 +1,94 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.value;
+
+import org.apache.asterix.column.values.reader.filter.FilterAccessorProvider;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessorFactory;
+import org.apache.asterix.om.base.ADouble;
+import org.apache.asterix.om.base.AInt64;
+import org.apache.asterix.om.base.AString;
+import org.apache.asterix.om.base.IAObject;
+import org.apache.asterix.om.types.ATypeTag;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+public class ConstantColumnFilterValueAccessorFactory implements IColumnFilterValueAccessorFactory {
+ private static final long serialVersionUID = -4835407779342615453L;
+ private final long normalizedValue;
+ private final ATypeTag typeTag;
+ private final String stringValue;
+
+ private ConstantColumnFilterValueAccessorFactory(String stringValue, long normalizedValue, ATypeTag typeTag) {
+ this.stringValue = stringValue;
+ this.normalizedValue = normalizedValue;
+ this.typeTag = typeTag;
+ }
+
+ public static ConstantColumnFilterValueAccessorFactory createFactory(IAObject value) {
+ String stringValue;
+ long normalizedValue;
+ ATypeTag typeTag = value.getType().getTypeTag();
+ switch (typeTag) {
+ case BIGINT:
+ long longVal = ((AInt64) value).getLongValue();
+ stringValue = Long.toString(longVal);
+ normalizedValue = longVal;
+ break;
+ case DOUBLE:
+ double doubleVal = ((ADouble) value).getDoubleValue();
+ stringValue = Double.toString(doubleVal);
+ normalizedValue = Double.doubleToLongBits(doubleVal);
+ break;
+ case STRING:
+ stringValue = ((AString) value).getStringValue();
+ normalizedValue = normalize(stringValue);
+ break;
+ default:
+ return null;
+ }
+
+ return new ConstantColumnFilterValueAccessorFactory(stringValue, normalizedValue, typeTag);
+ }
+
+ @Override
+ public IColumnFilterValueAccessor create(FilterAccessorProvider filterAccessorProvider)
+ throws HyracksDataException {
+ return new ConstantColumnFilterValueAccessor(normalizedValue, typeTag);
+ }
+
+ @Override
+ public String toString() {
+ if (typeTag == ATypeTag.STRING) {
+ return "\"" + stringValue + "\"";
+ }
+ return stringValue;
+ }
+
+ private static long normalize(String value) {
+ long nk = 0;
+ for (int i = 0; i < 4; ++i) {
+ nk <<= 16;
+ if (i < value.length()) {
+ nk += value.charAt(i) & 0xffff;
+ }
+ }
+ //Make it always positive
+ return nk >>> 1;
+ }
+}
diff --git a/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/NoOpColumnFilterValueAccessor.java b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/NoOpColumnFilterValueAccessor.java
new file mode 100644
index 0000000..c52eeda
--- /dev/null
+++ b/asterixdb/asterix-column/src/main/java/org/apache/asterix/column/values/reader/filter/value/NoOpColumnFilterValueAccessor.java
@@ -0,0 +1,39 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.column.values.reader.filter.value;
+
+import org.apache.asterix.column.values.reader.filter.IColumnFilterValueAccessor;
+import org.apache.asterix.om.types.ATypeTag;
+
+public class NoOpColumnFilterValueAccessor implements IColumnFilterValueAccessor {
+ public static final IColumnFilterValueAccessor INSTANCE = new NoOpColumnFilterValueAccessor();
+
+ private NoOpColumnFilterValueAccessor() {
+ }
+
+ @Override
+ public long getNormalizedValue() {
+ throw new IllegalStateException("should not be invoked");
+ }
+
+ @Override
+ public ATypeTag getTypeTag() {
+ return ATypeTag.MISSING;
+ }
+}
diff --git a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/input/record/reader/hdfs/parquet/AsterixTypeToParquetTypeVisitor.java b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/input/record/reader/hdfs/parquet/AsterixTypeToParquetTypeVisitor.java
index 7258359..68d5b86 100644
--- a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/input/record/reader/hdfs/parquet/AsterixTypeToParquetTypeVisitor.java
+++ b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/input/record/reader/hdfs/parquet/AsterixTypeToParquetTypeVisitor.java
@@ -174,7 +174,7 @@
//If no warning is created, then it means it has been reported
Warning warning = null;
if (actualType != ATypeTag.SYSTEM_NULL) {
- warning = info.createTypeMismatchWarning(expectedType, actualType);
+ warning = info.createWarning(expectedType, actualType);
}
if (warning != null) {
//New warning that we saw for the first time. We should report it.
diff --git a/asterixdb/asterix-metadata/pom.xml b/asterixdb/asterix-metadata/pom.xml
index 61507d1..fd97062 100644
--- a/asterixdb/asterix-metadata/pom.xml
+++ b/asterixdb/asterix-metadata/pom.xml
@@ -180,5 +180,10 @@
<groupId>org.apache.hyracks</groupId>
<artifactId>hyracks-ipc</artifactId>
</dependency>
+ <dependency>
+ <groupId>org.apache.asterix</groupId>
+ <artifactId>asterix-column</artifactId>
+ <version>${project.version}</version>
+ </dependency>
</dependencies>
</project>
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/visitor/PathStringBuilderForIATypeVisitor.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/visitor/PathStringBuilderForIATypeVisitor.java
new file mode 100644
index 0000000..ddc2437
--- /dev/null
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/visitor/PathStringBuilderForIATypeVisitor.java
@@ -0,0 +1,63 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.om.types.visitor;
+
+import org.apache.asterix.om.types.ARecordType;
+import org.apache.asterix.om.types.AUnionType;
+import org.apache.asterix.om.types.AbstractCollectionType;
+import org.apache.asterix.om.types.IAType;
+import org.apache.asterix.om.types.IATypeVisitor;
+import org.apache.hyracks.algebricks.common.exceptions.NotImplementedException;
+
+/**
+ * Produces a path from a type
+ */
+public class PathStringBuilderForIATypeVisitor implements IATypeVisitor<Void, StringBuilder> {
+ private boolean root = true;
+
+ @Override
+ public Void visit(ARecordType recordType, StringBuilder arg) {
+ // here we assume the record type has only one child
+ if (root) {
+ arg.append("$$root");
+ root = false;
+ }
+ arg.append('.');
+ arg.append(recordType.getFieldNames()[0]);
+ recordType.getFieldTypes()[0].accept(this, arg);
+ return null;
+ }
+
+ @Override
+ public Void visit(AbstractCollectionType collectionType, StringBuilder arg) {
+ arg.append("[*]");
+ collectionType.getItemType().accept(this, arg);
+ return null;
+ }
+
+ @Override
+ public Void visit(AUnionType unionType, StringBuilder arg) {
+ throw new NotImplementedException("Check how to represent this");
+ }
+
+ @Override
+ public Void visitFlat(IAType flatType, StringBuilder arg) {
+ return null;
+ }
+}
diff --git a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/FunctionCallInformation.java b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/FunctionCallInformation.java
index 5cb26fd..be96358 100644
--- a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/FunctionCallInformation.java
+++ b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/FunctionCallInformation.java
@@ -27,7 +27,6 @@
import java.util.Objects;
import java.util.Set;
-import org.apache.asterix.common.exceptions.ErrorCode;
import org.apache.asterix.om.exceptions.ExceptionUtil;
import org.apache.asterix.om.types.ATypeTag;
import org.apache.hyracks.algebricks.core.algebra.functions.FunctionIdentifier;
@@ -41,16 +40,20 @@
private static final long serialVersionUID = -7884346933746232736L;
private final String functionName;
private final SourceLocation sourceLocation;
+ private final IProjectionFiltrationWarningFactory warningFactory;
private Set<ATypeTag> typeMismatches;
- public FunctionCallInformation(String functionName, SourceLocation sourceLocation) {
- this(functionName, sourceLocation, Collections.emptySet());
+ public FunctionCallInformation(String functionName, SourceLocation sourceLocation,
+ IProjectionFiltrationWarningFactory warningFactory) {
+ this(functionName, sourceLocation, Collections.emptySet(), warningFactory);
}
- private FunctionCallInformation(String functionName, SourceLocation sourceLocation, Set<ATypeTag> typeMismatches) {
+ private FunctionCallInformation(String functionName, SourceLocation sourceLocation, Set<ATypeTag> typeMismatches,
+ IProjectionFiltrationWarningFactory warningFactory) {
this.functionName = functionName;
this.sourceLocation = sourceLocation;
this.typeMismatches = typeMismatches;
+ this.warningFactory = warningFactory;
}
public String getFunctionName() {
@@ -61,22 +64,23 @@
return sourceLocation;
}
- public Warning createTypeMismatchWarning(ATypeTag expectedType, ATypeTag actualType) {
- if (typeMismatches == null) {
+ public Warning createWarning(ATypeTag expectedType, ATypeTag actualType) {
+ if (typeMismatches.isEmpty()) {
typeMismatches = EnumSet.noneOf(ATypeTag.class);
} else if (typeMismatches.contains(actualType)) {
//We already issued a warning containing the same actual type. So, we ignore it
return null;
}
typeMismatches.add(actualType);
- return Warning.of(getSourceLocation(), ErrorCode.TYPE_MISMATCH_FUNCTION, getFunctionName(),
- ExceptionUtil.indexToPosition(0), expectedType, actualType);
+ return warningFactory.createWarning(getSourceLocation(), getFunctionName(), ExceptionUtil.indexToPosition(0),
+ expectedType, actualType);
}
public void writeFields(DataOutput output) throws IOException {
output.writeUTF(functionName);
SourceLocation.writeFields(sourceLocation, output);
output.writeInt(typeMismatches.size());
+ output.writeInt(warningFactory.getErrorCode().intValue());
for (ATypeTag typeTag : typeMismatches) {
output.write(typeTag.serialize());
}
@@ -87,10 +91,12 @@
SourceLocation sourceLocation = SourceLocation.create(in);
int typeMismatchesLength = in.readInt();
Set<ATypeTag> typeMismatches = EnumSet.noneOf(ATypeTag.class);
+ IProjectionFiltrationWarningFactory warningFactory =
+ ProjectionFiltrationWarningFactoryProvider.getWarningFactory(in.readInt());
for (int i = 0; i < typeMismatchesLength; i++) {
typeMismatches.add(ATypeTag.VALUE_TYPE_MAPPING[in.readByte()]);
}
- return new FunctionCallInformation(functionName, sourceLocation, typeMismatches);
+ return new FunctionCallInformation(functionName, sourceLocation, typeMismatches, warningFactory);
}
@Override
diff --git a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/IProjectionFiltrationWarningFactory.java b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/IProjectionFiltrationWarningFactory.java
new file mode 100644
index 0000000..4612f05
--- /dev/null
+++ b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/IProjectionFiltrationWarningFactory.java
@@ -0,0 +1,33 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.runtime.projection;
+
+import java.io.Serializable;
+
+import org.apache.asterix.common.exceptions.ErrorCode;
+import org.apache.asterix.om.types.ATypeTag;
+import org.apache.hyracks.api.exceptions.SourceLocation;
+import org.apache.hyracks.api.exceptions.Warning;
+
+public interface IProjectionFiltrationWarningFactory extends Serializable {
+ Warning createWarning(SourceLocation sourceLocation, String functionName, String position, ATypeTag expectedType,
+ ATypeTag actualType);
+
+ ErrorCode getErrorCode();
+}
diff --git a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/ProjectionFiltrationWarningFactoryProvider.java b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/ProjectionFiltrationWarningFactoryProvider.java
new file mode 100644
index 0000000..7d548ce
--- /dev/null
+++ b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/projection/ProjectionFiltrationWarningFactoryProvider.java
@@ -0,0 +1,72 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.asterix.runtime.projection;
+
+import org.apache.asterix.common.exceptions.ErrorCode;
+import org.apache.asterix.om.exceptions.ExceptionUtil;
+import org.apache.asterix.om.types.ATypeTag;
+import org.apache.hyracks.api.exceptions.SourceLocation;
+import org.apache.hyracks.api.exceptions.Warning;
+
+public class ProjectionFiltrationWarningFactoryProvider {
+ private ProjectionFiltrationWarningFactoryProvider() {
+ }
+
+ public static IProjectionFiltrationWarningFactory getWarningFactory(int errorCode) {
+ if (errorCode == ErrorCode.TYPE_MISMATCH_FUNCTION.intValue()) {
+ return TYPE_MISMATCH_FACTORY;
+ } else if (errorCode == ErrorCode.INCOMPARABLE_TYPES.intValue()) {
+ return INCOMPARABLE_TYPES_FACTORY;
+ }
+ throw new UnsupportedOperationException("Unsupported error code " + errorCode);
+ }
+
+ public static final IProjectionFiltrationWarningFactory TYPE_MISMATCH_FACTORY =
+ new IProjectionFiltrationWarningFactory() {
+ private static final long serialVersionUID = 4263556611813387010L;
+
+ @Override
+ public Warning createWarning(SourceLocation sourceLocation, String functionName, String position,
+ ATypeTag expectedType, ATypeTag actualType) {
+ return Warning.of(sourceLocation, ErrorCode.TYPE_MISMATCH_FUNCTION, functionName,
+ ExceptionUtil.indexToPosition(0), expectedType, actualType);
+ }
+
+ @Override
+ public ErrorCode getErrorCode() {
+ return ErrorCode.TYPE_MISMATCH_FUNCTION;
+ }
+ };
+
+ public static final IProjectionFiltrationWarningFactory INCOMPARABLE_TYPES_FACTORY =
+ new IProjectionFiltrationWarningFactory() {
+ private static final long serialVersionUID = -7447187099851545763L;
+
+ @Override
+ public Warning createWarning(SourceLocation sourceLocation, String functionName, String position,
+ ATypeTag expectedType, ATypeTag actualType) {
+ return Warning.of(sourceLocation, ErrorCode.INCOMPARABLE_TYPES, expectedType, actualType);
+ }
+
+ @Override
+ public ErrorCode getErrorCode() {
+ return ErrorCode.INCOMPARABLE_TYPES;
+ }
+ };
+}