You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@asterixdb.apache.org by AsterixDB Code Review <do...@asterix-gerrit.ics.uci.edu> on 2021/10/06 20:10:30 UTC

Change in asterixdb[master]: [NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types

From Dmitry Lychagin <dm...@couchbase.com>:

Dmitry Lychagin has uploaded this change for review. ( https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/13586 )


Change subject: [NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types
......................................................................

[NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types

- user model changes: no
- storage format changes: no
- interface changes: no

Details:
- Introduce RecordFieldOrderAnnotation for record types
  which acts as a hint specifying field order
- Generate these annotations for types created by
  OpenRecordConstructorResultType type computer
- Modify TypeResolverUtil to propagate these annotations
  if they're available
- Add testcase

Change-Id: I53c8b15f2d2299e7df670bcd2ed7880e3bcca344
---
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
M asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
6 files changed, 173 insertions(+), 12 deletions(-)



  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/86/13586/1

diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
index 5edef2f..ec0917e 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
@@ -19,9 +19,10 @@
 package org.apache.asterix.common.annotations;
 
 public interface IRecordTypeAnnotation {
-    public enum Kind {
-        RECORD_DATA_GEN
+    enum Kind {
+        RECORD_DATA_GEN,
+        RECORD_FIELD_ORDER
     }
 
-    public Kind getKind();
+    Kind getKind();
 }
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java
new file mode 100644
index 0000000..04cf589
--- /dev/null
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java
@@ -0,0 +1,62 @@
+/*
+ * 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.common.annotations;
+
+import java.io.Serializable;
+import java.util.LinkedHashSet;
+import java.util.Objects;
+
+/**
+ * Contains an ordered set of fields of a record
+ */
+public final class RecordFieldOrderAnnotation implements IRecordTypeAnnotation, Serializable {
+
+    private static final long serialVersionUID = 1L;
+
+    private final LinkedHashSet<String> fieldNames;
+
+    public RecordFieldOrderAnnotation(LinkedHashSet<String> fieldNames) {
+        this.fieldNames = Objects.requireNonNull(fieldNames);
+    }
+
+    public LinkedHashSet<String> getFieldNames() {
+        return fieldNames;
+    }
+
+    @Override
+    public Kind getKind() {
+        return Kind.RECORD_FIELD_ORDER;
+    }
+
+    @Override
+    public boolean equals(Object o) {
+        if (this == o)
+            return true;
+        if (o == null || getClass() != o.getClass())
+            return false;
+        RecordFieldOrderAnnotation that = (RecordFieldOrderAnnotation) o;
+        return fieldNames.equals(that.fieldNames);
+    }
+
+    @Override
+    public int hashCode() {
+        return Objects.hash(fieldNames);
+    }
+}
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
index 3cc9e3d..9d52a85 100644
--- a/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
@@ -21,9 +21,12 @@
 
 import java.util.ArrayList;
 import java.util.HashSet;
+import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Set;
 
+import org.apache.asterix.common.annotations.IRecordTypeAnnotation;
+import org.apache.asterix.common.annotations.RecordFieldOrderAnnotation;
 import org.apache.asterix.om.typecomputer.impl.TypeComputeUtils;
 import org.apache.asterix.om.types.AOrderedListType;
 import org.apache.asterix.om.types.ARecordType;
@@ -168,9 +171,38 @@
                 generalizeRecordFields(leftType, rightType, allPossibleAdditionalFieldNames, fieldNames, fieldTypes);
         boolean rightAllMatched =
                 generalizeRecordFields(rightType, leftType, allPossibleAdditionalFieldNames, fieldNames, fieldTypes);
-        return new ARecordType("generalized-record-type", fieldNames.toArray(new String[fieldNames.size()]),
-                fieldTypes.toArray(new IAType[fieldTypes.size()]), !(canBeClosed && leftAllMatched && rightAllMatched),
-                knowsAdditonalFieldNames ? allPossibleAdditionalFieldNames : null);
+        boolean resultTypeIsOpen = !(canBeClosed && leftAllMatched && rightAllMatched);
+        String[] fieldNamesArray = fieldNames.toArray(new String[0]);
+        IAType[] fieldTypesArray = fieldTypes.toArray(new IAType[0]);
+        ARecordType resultType;
+        if (resultTypeIsOpen && knowsAdditonalFieldNames) {
+            resultType = new ARecordType("generalized-record-type", fieldNamesArray, fieldTypesArray, resultTypeIsOpen,
+                    allPossibleAdditionalFieldNames);
+            LinkedHashSet<String> resultFieldOrder = generalizeRecordFieldOrderHint(leftType, rightType);
+            if (resultFieldOrder != null) {
+                resultType.getAnnotations().add(new RecordFieldOrderAnnotation(resultFieldOrder));
+            }
+        } else {
+            resultType = new ARecordType("generalized-record-type", fieldNamesArray, fieldTypesArray, resultTypeIsOpen);
+        }
+        return resultType;
+    }
+
+    private static LinkedHashSet<String> generalizeRecordFieldOrderHint(ARecordType leftType, ARecordType rightType) {
+        IRecordTypeAnnotation leftFieldOrderHint =
+                leftType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        if (leftFieldOrderHint == null) {
+            return null;
+        }
+        IRecordTypeAnnotation rightFieldOrderHint =
+                rightType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        if (rightFieldOrderHint == null) {
+            return null;
+        }
+        LinkedHashSet<String> resultFieldOrder = new LinkedHashSet<>();
+        resultFieldOrder.addAll(((RecordFieldOrderAnnotation) leftFieldOrderHint).getFieldNames());
+        resultFieldOrder.addAll(((RecordFieldOrderAnnotation) rightFieldOrderHint).getFieldNames());
+        return resultFieldOrder;
     }
 
     // Generates closed fields and possible additional fields of a generalized type of two record types.
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
index 6deb17c..6b08230 100644
--- a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
@@ -22,9 +22,11 @@
 import java.util.ArrayList;
 import java.util.HashSet;
 import java.util.Iterator;
+import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Set;
 
+import org.apache.asterix.common.annotations.RecordFieldOrderAnnotation;
 import org.apache.asterix.common.exceptions.CompilationException;
 import org.apache.asterix.common.exceptions.ErrorCode;
 import org.apache.asterix.om.typecomputer.base.IResultTypeComputer;
@@ -51,7 +53,7 @@
             IMetadataProvider<?, ?> metadataProvider) throws AlgebricksException {
         AbstractFunctionCallExpression f = (AbstractFunctionCallExpression) expression;
 
-        /**
+        /*
          * if type has been top-down propagated, use the enforced type
          */
         ARecordType type = (ARecordType) TypeCastUtils.getRequiredType(f);
@@ -66,6 +68,7 @@
         // but are additional possible field names. For example, a field "foo" with type
         // ANY cannot be in the closed part, but "foo" is a possible field name.
         Set<String> allPossibleAdditionalFieldNames = new HashSet<>();
+        LinkedHashSet<String> allPossibleFieldNamesOrdered = new LinkedHashSet<>();
         boolean canProvideAdditionFieldInfo = true;
         boolean isOpen = false;
         while (argIter.hasNext()) {
@@ -91,11 +94,17 @@
                 }
                 isOpen = true;
             }
+            allPossibleFieldNamesOrdered.add(fieldName);
         }
         String[] fieldNames = namesList.toArray(new String[0]);
         IAType[] fieldTypes = typesList.toArray(new IAType[0]);
-        return canProvideAdditionFieldInfo
-                ? new ARecordType(null, fieldNames, fieldTypes, isOpen, allPossibleAdditionalFieldNames)
-                : new ARecordType(null, fieldNames, fieldTypes, isOpen);
+        ARecordType resultType;
+        if (isOpen && canProvideAdditionFieldInfo) {
+            resultType = new ARecordType(null, fieldNames, fieldTypes, isOpen, allPossibleAdditionalFieldNames);
+            resultType.getAnnotations().add(new RecordFieldOrderAnnotation(allPossibleFieldNamesOrdered));
+        } else {
+            resultType = new ARecordType(null, fieldNames, fieldTypes, isOpen);
+        }
+        return resultType;
     }
 }
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
index a79b8de..e256e1b 100644
--- a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
@@ -22,8 +22,10 @@
 import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.HashMap;
+import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
+import java.util.Objects;
 import java.util.Set;
 
 import org.apache.asterix.common.annotations.IRecordTypeAnnotation;
@@ -145,6 +147,17 @@
         return annotations;
     }
 
+    public IRecordTypeAnnotation findAnnotation(IRecordTypeAnnotation.Kind kind) {
+        if (annotations != null) {
+            for (IRecordTypeAnnotation ant : annotations) {
+                if (ant.getKind().equals(kind)) {
+                    return ant;
+                }
+            }
+        }
+        return null;
+    }
+
     @Override
     public String toString() {
         return append(new StringBuilder()).toString();
@@ -312,7 +325,10 @@
                 newTypes[i] = type.fieldTypes[i];
             }
         }
-        return new ARecordType(type.typeName, type.fieldNames, newTypes, type.isOpen);
+        Set<String> newAllPossibleAdditionalFieldNames =
+                allPossibleAdditionalFieldNames != null ? new HashSet<>(allPossibleAdditionalFieldNames) : null;
+        return new ARecordType(type.typeName, type.fieldNames, newTypes, type.isOpen,
+                newAllPossibleAdditionalFieldNames);
     }
 
     @Override
@@ -344,7 +360,8 @@
         }
         ARecordType rt = (ARecordType) obj;
         return (isOpen == rt.isOpen) && Arrays.deepEquals(fieldNames, rt.fieldNames)
-                && Arrays.deepEquals(fieldTypes, rt.fieldTypes);
+                && Arrays.deepEquals(fieldTypes, rt.fieldTypes)
+                && Objects.equals(allPossibleAdditionalFieldNames, rt.allPossibleAdditionalFieldNames);
     }
 
     @Override
diff --git a/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java b/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
index 5303870..dad6ee9 100644
--- a/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
+++ b/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
@@ -20,11 +20,15 @@
 package org.apache.asterix.dataflow.data.common;
 
 import java.util.ArrayList;
+import java.util.Arrays;
 import java.util.Collections;
 import java.util.HashSet;
+import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Set;
 
+import org.apache.asterix.common.annotations.IRecordTypeAnnotation;
+import org.apache.asterix.common.annotations.RecordFieldOrderAnnotation;
 import org.apache.asterix.om.types.AOrderedListType;
 import org.apache.asterix.om.types.ARecordType;
 import org.apache.asterix.om.types.AUnionType;
@@ -124,6 +128,42 @@
     }
 
     @Test
+    public void testRecordTypeFieldOrderHint() {
+        // Constructs input types.
+        ARecordType leftRecordType = new ARecordType(null, new String[] { "a", "b", "c" },
+                new IAType[] { BuiltinType.ASTRING, BuiltinType.ASTRING, BuiltinType.ASTRING, }, true,
+                new HashSet<>(Arrays.asList("d", "e")));
+        leftRecordType.getAnnotations()
+                .add(new RecordFieldOrderAnnotation(new LinkedHashSet<>(Arrays.asList("a", "b", "c", "d", "e"))));
+
+        ARecordType rightRecordType = new ARecordType(null, new String[] { "a", "c", "d" },
+                new IAType[] { BuiltinType.ASTRING, BuiltinType.ASTRING, BuiltinType.ASTRING, }, true,
+                new HashSet<>(Arrays.asList("e", "f")));
+        rightRecordType.getAnnotations()
+                .add(new RecordFieldOrderAnnotation(new LinkedHashSet<>(Arrays.asList("a", "c", "d", "e", "f"))));
+
+        // Resolves input types to a generalized type.
+        List<IAType> inputTypes = new ArrayList<>();
+        inputTypes.add(leftRecordType);
+        inputTypes.add(rightRecordType);
+        ARecordType resolvedType = (ARecordType) TypeResolverUtil.resolve(inputTypes);
+
+        // Constructs the expected type.
+        Set<String> possibleAdditionalFields = new HashSet<>(Arrays.asList("b", "d", "e", "f"));
+        ARecordType expectedType = new ARecordType(null, new String[] { "a", "c" },
+                new IAType[] { BuiltinType.ASTRING, BuiltinType.ASTRING }, true, possibleAdditionalFields);
+        expectedType.getAnnotations()
+                .add(new RecordFieldOrderAnnotation(new LinkedHashSet<>(Arrays.asList("a", "b", "c", "d", "e", "f"))));
+
+        // Compares the resolved type with the expected type.
+        Assert.assertEquals(expectedType, resolvedType);
+
+        IRecordTypeAnnotation expecedAnn = expectedType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        IRecordTypeAnnotation resolvedAnn = resolvedType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        Assert.assertEquals(expecedAnn, resolvedAnn);
+    }
+
+    @Test
     public void testOrderedListType() {
         // Constructs input types.
         ARecordType leftRecordType = new ARecordType("null", new String[] { "a", "b" },

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/13586
To unsubscribe, or for help writing mail filters, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Change-Id: I53c8b15f2d2299e7df670bcd2ed7880e3bcca344
Gerrit-Change-Number: 13586
Gerrit-PatchSet: 1
Gerrit-Owner: Dmitry Lychagin <dm...@couchbase.com>
Gerrit-MessageType: newchange

Change in asterixdb[master]: [NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types

Posted by AsterixDB Code Review <do...@asterix-gerrit.ics.uci.edu>.
From Dmitry Lychagin <dm...@couchbase.com>:

Dmitry Lychagin has posted comments on this change. ( https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/13586 )

Change subject: [NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types
......................................................................


Patch Set 1: Code-Review+1


-- 
To view, visit https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/13586
To unsubscribe, or for help writing mail filters, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Change-Id: I53c8b15f2d2299e7df670bcd2ed7880e3bcca344
Gerrit-Change-Number: 13586
Gerrit-PatchSet: 1
Gerrit-Owner: Dmitry Lychagin <dm...@couchbase.com>
Gerrit-Reviewer: Ali Alsuliman <al...@gmail.com>
Gerrit-Reviewer: Dmitry Lychagin <dm...@couchbase.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-CC: Anon. E. Moose #1000171
Gerrit-Comment-Date: Wed, 06 Oct 2021 21:56:47 +0000
Gerrit-HasComments: No
Gerrit-Has-Labels: Yes
Gerrit-MessageType: comment

Change in asterixdb[master]: [NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types

Posted by AsterixDB Code Review <do...@asterix-gerrit.ics.uci.edu>.
From Dmitry Lychagin <dm...@couchbase.com>:

Dmitry Lychagin has uploaded this change for review. ( https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/13586 )


Change subject: [NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types
......................................................................

[NO ISSUE][COMP] Introduce RecordFieldOrderAnnotation for record types

- user model changes: no
- storage format changes: no
- interface changes: no

Details:
- Introduce RecordFieldOrderAnnotation for record types
  which acts as a hint specifying field order
- Generate these annotations for types created by
  OpenRecordConstructorResultType type computer
- Modify TypeResolverUtil to propagate these annotations
  if they're available
- Add testcase

Change-Id: I53c8b15f2d2299e7df670bcd2ed7880e3bcca344
---
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
M asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
6 files changed, 173 insertions(+), 12 deletions(-)



  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/86/13586/1

diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
index 5edef2f..ec0917e 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/IRecordTypeAnnotation.java
@@ -19,9 +19,10 @@
 package org.apache.asterix.common.annotations;
 
 public interface IRecordTypeAnnotation {
-    public enum Kind {
-        RECORD_DATA_GEN
+    enum Kind {
+        RECORD_DATA_GEN,
+        RECORD_FIELD_ORDER
     }
 
-    public Kind getKind();
+    Kind getKind();
 }
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java
new file mode 100644
index 0000000..04cf589
--- /dev/null
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/annotations/RecordFieldOrderAnnotation.java
@@ -0,0 +1,62 @@
+/*
+ * 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.common.annotations;
+
+import java.io.Serializable;
+import java.util.LinkedHashSet;
+import java.util.Objects;
+
+/**
+ * Contains an ordered set of fields of a record
+ */
+public final class RecordFieldOrderAnnotation implements IRecordTypeAnnotation, Serializable {
+
+    private static final long serialVersionUID = 1L;
+
+    private final LinkedHashSet<String> fieldNames;
+
+    public RecordFieldOrderAnnotation(LinkedHashSet<String> fieldNames) {
+        this.fieldNames = Objects.requireNonNull(fieldNames);
+    }
+
+    public LinkedHashSet<String> getFieldNames() {
+        return fieldNames;
+    }
+
+    @Override
+    public Kind getKind() {
+        return Kind.RECORD_FIELD_ORDER;
+    }
+
+    @Override
+    public boolean equals(Object o) {
+        if (this == o)
+            return true;
+        if (o == null || getClass() != o.getClass())
+            return false;
+        RecordFieldOrderAnnotation that = (RecordFieldOrderAnnotation) o;
+        return fieldNames.equals(that.fieldNames);
+    }
+
+    @Override
+    public int hashCode() {
+        return Objects.hash(fieldNames);
+    }
+}
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
index 3cc9e3d..9d52a85 100644
--- a/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/dataflow/data/common/TypeResolverUtil.java
@@ -21,9 +21,12 @@
 
 import java.util.ArrayList;
 import java.util.HashSet;
+import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Set;
 
+import org.apache.asterix.common.annotations.IRecordTypeAnnotation;
+import org.apache.asterix.common.annotations.RecordFieldOrderAnnotation;
 import org.apache.asterix.om.typecomputer.impl.TypeComputeUtils;
 import org.apache.asterix.om.types.AOrderedListType;
 import org.apache.asterix.om.types.ARecordType;
@@ -168,9 +171,38 @@
                 generalizeRecordFields(leftType, rightType, allPossibleAdditionalFieldNames, fieldNames, fieldTypes);
         boolean rightAllMatched =
                 generalizeRecordFields(rightType, leftType, allPossibleAdditionalFieldNames, fieldNames, fieldTypes);
-        return new ARecordType("generalized-record-type", fieldNames.toArray(new String[fieldNames.size()]),
-                fieldTypes.toArray(new IAType[fieldTypes.size()]), !(canBeClosed && leftAllMatched && rightAllMatched),
-                knowsAdditonalFieldNames ? allPossibleAdditionalFieldNames : null);
+        boolean resultTypeIsOpen = !(canBeClosed && leftAllMatched && rightAllMatched);
+        String[] fieldNamesArray = fieldNames.toArray(new String[0]);
+        IAType[] fieldTypesArray = fieldTypes.toArray(new IAType[0]);
+        ARecordType resultType;
+        if (resultTypeIsOpen && knowsAdditonalFieldNames) {
+            resultType = new ARecordType("generalized-record-type", fieldNamesArray, fieldTypesArray, resultTypeIsOpen,
+                    allPossibleAdditionalFieldNames);
+            LinkedHashSet<String> resultFieldOrder = generalizeRecordFieldOrderHint(leftType, rightType);
+            if (resultFieldOrder != null) {
+                resultType.getAnnotations().add(new RecordFieldOrderAnnotation(resultFieldOrder));
+            }
+        } else {
+            resultType = new ARecordType("generalized-record-type", fieldNamesArray, fieldTypesArray, resultTypeIsOpen);
+        }
+        return resultType;
+    }
+
+    private static LinkedHashSet<String> generalizeRecordFieldOrderHint(ARecordType leftType, ARecordType rightType) {
+        IRecordTypeAnnotation leftFieldOrderHint =
+                leftType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        if (leftFieldOrderHint == null) {
+            return null;
+        }
+        IRecordTypeAnnotation rightFieldOrderHint =
+                rightType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        if (rightFieldOrderHint == null) {
+            return null;
+        }
+        LinkedHashSet<String> resultFieldOrder = new LinkedHashSet<>();
+        resultFieldOrder.addAll(((RecordFieldOrderAnnotation) leftFieldOrderHint).getFieldNames());
+        resultFieldOrder.addAll(((RecordFieldOrderAnnotation) rightFieldOrderHint).getFieldNames());
+        return resultFieldOrder;
     }
 
     // Generates closed fields and possible additional fields of a generalized type of two record types.
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
index 6deb17c..6b08230 100644
--- a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/typecomputer/impl/OpenRecordConstructorResultType.java
@@ -22,9 +22,11 @@
 import java.util.ArrayList;
 import java.util.HashSet;
 import java.util.Iterator;
+import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Set;
 
+import org.apache.asterix.common.annotations.RecordFieldOrderAnnotation;
 import org.apache.asterix.common.exceptions.CompilationException;
 import org.apache.asterix.common.exceptions.ErrorCode;
 import org.apache.asterix.om.typecomputer.base.IResultTypeComputer;
@@ -51,7 +53,7 @@
             IMetadataProvider<?, ?> metadataProvider) throws AlgebricksException {
         AbstractFunctionCallExpression f = (AbstractFunctionCallExpression) expression;
 
-        /**
+        /*
          * if type has been top-down propagated, use the enforced type
          */
         ARecordType type = (ARecordType) TypeCastUtils.getRequiredType(f);
@@ -66,6 +68,7 @@
         // but are additional possible field names. For example, a field "foo" with type
         // ANY cannot be in the closed part, but "foo" is a possible field name.
         Set<String> allPossibleAdditionalFieldNames = new HashSet<>();
+        LinkedHashSet<String> allPossibleFieldNamesOrdered = new LinkedHashSet<>();
         boolean canProvideAdditionFieldInfo = true;
         boolean isOpen = false;
         while (argIter.hasNext()) {
@@ -91,11 +94,17 @@
                 }
                 isOpen = true;
             }
+            allPossibleFieldNamesOrdered.add(fieldName);
         }
         String[] fieldNames = namesList.toArray(new String[0]);
         IAType[] fieldTypes = typesList.toArray(new IAType[0]);
-        return canProvideAdditionFieldInfo
-                ? new ARecordType(null, fieldNames, fieldTypes, isOpen, allPossibleAdditionalFieldNames)
-                : new ARecordType(null, fieldNames, fieldTypes, isOpen);
+        ARecordType resultType;
+        if (isOpen && canProvideAdditionFieldInfo) {
+            resultType = new ARecordType(null, fieldNames, fieldTypes, isOpen, allPossibleAdditionalFieldNames);
+            resultType.getAnnotations().add(new RecordFieldOrderAnnotation(allPossibleFieldNamesOrdered));
+        } else {
+            resultType = new ARecordType(null, fieldNames, fieldTypes, isOpen);
+        }
+        return resultType;
     }
 }
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
index a79b8de..e256e1b 100644
--- a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/types/ARecordType.java
@@ -22,8 +22,10 @@
 import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.HashMap;
+import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
+import java.util.Objects;
 import java.util.Set;
 
 import org.apache.asterix.common.annotations.IRecordTypeAnnotation;
@@ -145,6 +147,17 @@
         return annotations;
     }
 
+    public IRecordTypeAnnotation findAnnotation(IRecordTypeAnnotation.Kind kind) {
+        if (annotations != null) {
+            for (IRecordTypeAnnotation ant : annotations) {
+                if (ant.getKind().equals(kind)) {
+                    return ant;
+                }
+            }
+        }
+        return null;
+    }
+
     @Override
     public String toString() {
         return append(new StringBuilder()).toString();
@@ -312,7 +325,10 @@
                 newTypes[i] = type.fieldTypes[i];
             }
         }
-        return new ARecordType(type.typeName, type.fieldNames, newTypes, type.isOpen);
+        Set<String> newAllPossibleAdditionalFieldNames =
+                allPossibleAdditionalFieldNames != null ? new HashSet<>(allPossibleAdditionalFieldNames) : null;
+        return new ARecordType(type.typeName, type.fieldNames, newTypes, type.isOpen,
+                newAllPossibleAdditionalFieldNames);
     }
 
     @Override
@@ -344,7 +360,8 @@
         }
         ARecordType rt = (ARecordType) obj;
         return (isOpen == rt.isOpen) && Arrays.deepEquals(fieldNames, rt.fieldNames)
-                && Arrays.deepEquals(fieldTypes, rt.fieldTypes);
+                && Arrays.deepEquals(fieldTypes, rt.fieldTypes)
+                && Objects.equals(allPossibleAdditionalFieldNames, rt.allPossibleAdditionalFieldNames);
     }
 
     @Override
diff --git a/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java b/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
index 5303870..dad6ee9 100644
--- a/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
+++ b/asterixdb/asterix-om/src/test/java/org/apache/asterix/dataflow/data/common/TypeResolverUtilTest.java
@@ -20,11 +20,15 @@
 package org.apache.asterix.dataflow.data.common;
 
 import java.util.ArrayList;
+import java.util.Arrays;
 import java.util.Collections;
 import java.util.HashSet;
+import java.util.LinkedHashSet;
 import java.util.List;
 import java.util.Set;
 
+import org.apache.asterix.common.annotations.IRecordTypeAnnotation;
+import org.apache.asterix.common.annotations.RecordFieldOrderAnnotation;
 import org.apache.asterix.om.types.AOrderedListType;
 import org.apache.asterix.om.types.ARecordType;
 import org.apache.asterix.om.types.AUnionType;
@@ -124,6 +128,42 @@
     }
 
     @Test
+    public void testRecordTypeFieldOrderHint() {
+        // Constructs input types.
+        ARecordType leftRecordType = new ARecordType(null, new String[] { "a", "b", "c" },
+                new IAType[] { BuiltinType.ASTRING, BuiltinType.ASTRING, BuiltinType.ASTRING, }, true,
+                new HashSet<>(Arrays.asList("d", "e")));
+        leftRecordType.getAnnotations()
+                .add(new RecordFieldOrderAnnotation(new LinkedHashSet<>(Arrays.asList("a", "b", "c", "d", "e"))));
+
+        ARecordType rightRecordType = new ARecordType(null, new String[] { "a", "c", "d" },
+                new IAType[] { BuiltinType.ASTRING, BuiltinType.ASTRING, BuiltinType.ASTRING, }, true,
+                new HashSet<>(Arrays.asList("e", "f")));
+        rightRecordType.getAnnotations()
+                .add(new RecordFieldOrderAnnotation(new LinkedHashSet<>(Arrays.asList("a", "c", "d", "e", "f"))));
+
+        // Resolves input types to a generalized type.
+        List<IAType> inputTypes = new ArrayList<>();
+        inputTypes.add(leftRecordType);
+        inputTypes.add(rightRecordType);
+        ARecordType resolvedType = (ARecordType) TypeResolverUtil.resolve(inputTypes);
+
+        // Constructs the expected type.
+        Set<String> possibleAdditionalFields = new HashSet<>(Arrays.asList("b", "d", "e", "f"));
+        ARecordType expectedType = new ARecordType(null, new String[] { "a", "c" },
+                new IAType[] { BuiltinType.ASTRING, BuiltinType.ASTRING }, true, possibleAdditionalFields);
+        expectedType.getAnnotations()
+                .add(new RecordFieldOrderAnnotation(new LinkedHashSet<>(Arrays.asList("a", "b", "c", "d", "e", "f"))));
+
+        // Compares the resolved type with the expected type.
+        Assert.assertEquals(expectedType, resolvedType);
+
+        IRecordTypeAnnotation expecedAnn = expectedType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        IRecordTypeAnnotation resolvedAnn = resolvedType.findAnnotation(IRecordTypeAnnotation.Kind.RECORD_FIELD_ORDER);
+        Assert.assertEquals(expecedAnn, resolvedAnn);
+    }
+
+    @Test
     public void testOrderedListType() {
         // Constructs input types.
         ARecordType leftRecordType = new ARecordType("null", new String[] { "a", "b" },

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/c/asterixdb/+/13586
To unsubscribe, or for help writing mail filters, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Change-Id: I53c8b15f2d2299e7df670bcd2ed7880e3bcca344
Gerrit-Change-Number: 13586
Gerrit-PatchSet: 1
Gerrit-Owner: Dmitry Lychagin <dm...@couchbase.com>
Gerrit-MessageType: newchange