Skip to content

Commit

Permalink
variant predefine
Browse files Browse the repository at this point in the history
  • Loading branch information
eldenmoon committed Aug 20, 2024
1 parent 777dbfb commit 612ab0c
Show file tree
Hide file tree
Showing 9 changed files with 336 additions and 8 deletions.
16 changes: 16 additions & 0 deletions be/src/runtime/types.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,7 @@
#include <utility>

#include "olap/olap_define.h"
#include "runtime/define_primitive_type.h"
#include "runtime/primitive_type.h"

namespace doris {
Expand Down Expand Up @@ -108,6 +109,21 @@ TypeDescriptor::TypeDescriptor(const std::vector<TTypeNode>& types, int* idx)
contains_nulls.push_back(node.contains_nulls[1]);
break;
}
case TTypeNodeType::VARIANT: {
// complex variant type
DCHECK(!node.__isset.scalar_type);
DCHECK_LT(*idx, types.size() - 1);
DCHECK(!node.__isset.contains_nulls);
type = TYPE_VARIANT;
contains_nulls.reserve(node.struct_fields.size());
for (size_t i = 0; i < node.struct_fields.size(); i++) {
++(*idx);
children.push_back(TypeDescriptor(types, idx));
field_names.push_back(node.struct_fields[i].name);
contains_nulls.push_back(node.struct_fields[i].contains_null);
}
break;
}
default:
DCHECK(false) << node.type;
}
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,123 @@
// 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.doris.catalog;


import org.apache.doris.thrift.TTypeDesc;
import org.apache.doris.thrift.TTypeNode;
import org.apache.doris.thrift.TTypeNodeType;

import com.google.common.base.Joiner;
import com.google.common.base.Preconditions;
import com.google.common.collect.Lists;
import com.google.common.collect.Maps;
import com.google.gson.annotations.SerializedName;

import java.util.ArrayList;
import java.util.HashMap;

public class ComplexVariantType extends Type {

@SerializedName(value = "fieldMap")
private final HashMap<String, StructField> fieldMap = Maps.newHashMap();

@SerializedName(value = "fields")
private final ArrayList<StructField> predefinedFields;

public ComplexVariantType() {
this.predefinedFields = Lists.newArrayList();
}

public ComplexVariantType(ArrayList<StructField> fields) {
Preconditions.checkNotNull(fields);
this.predefinedFields = fields;
for (int i = 0; i < this.predefinedFields.size(); ++i) {
this.predefinedFields.get(i).setPosition(i);
fieldMap.put(this.predefinedFields.get(i).getName().toLowerCase(), this.predefinedFields.get(i));
}
}

public ArrayList<StructField> getPredefinedFields() {
return predefinedFields;
}

@Override
protected String toSql(int depth) {
if (depth >= MAX_NESTING_DEPTH) {
return "variant<...>";
}
ArrayList<String> fieldsSql = Lists.newArrayList();
for (StructField f : predefinedFields) {
fieldsSql.add(f.toSql(depth + 1));
}
return String.format("variant<%s>", Joiner.on(",").join(fieldsSql));
}

@Override
protected String prettyPrint(int lpad) {
return null;
}

@Override
public void toThrift(TTypeDesc container) {
// not use ScalarType's toThrift for compatibility, because VariantType is not extends ScalarType previously
TTypeNode node = new TTypeNode();
container.types.add(node);
node.setType(TTypeNodeType.VARIANT);
// predefined fields
node.setStructFields(new ArrayList<>());
for (StructField field : predefinedFields) {
field.toThrift(container, node);
}
}

@Override
public PrimitiveType getPrimitiveType() {
return PrimitiveType.VARIANT;
}


@Override
public boolean matchesType(Type t) {
return t.isVariantType() || t.isStringType();
}

@Override
public boolean supportSubType(Type subType) {
for (Type supportedType : Type.getComplexVariantSubTypes()) {
if (subType.getPrimitiveType() == supportedType.getPrimitiveType()) {
return true;
}
}
return false;
}

@Override
public boolean equals(Object other) {
if (!(other instanceof ComplexVariantType)) {
return false;
}
ComplexVariantType otherVariantType = (ComplexVariantType) other;
return otherVariantType.getPredefinedFields().equals(predefinedFields);
}

@Override
public int getSlotSize() {
return PrimitiveType.VARIANT.getSlotSize();
}
}
42 changes: 41 additions & 1 deletion fe/fe-common/src/main/java/org/apache/doris/catalog/Type.java
Original file line number Diff line number Diff line change
Expand Up @@ -135,6 +135,8 @@ public abstract class Type {
private static final ArrayList<Type> arraySubTypes;
private static final ArrayList<Type> mapSubTypes;
private static final ArrayList<Type> structSubTypes;

private static final ArrayList<Type> complexVariantSubTypes;
private static final ArrayList<ScalarType> trivialTypes;

static {
Expand Down Expand Up @@ -306,6 +308,29 @@ public abstract class Type {
structSubTypes.add(ARRAY);
structSubTypes.add(MAP);
structSubTypes.add(STRUCT);

complexVariantSubTypes = Lists.newArrayList();
complexVariantSubTypes.add(BOOLEAN);
complexVariantSubTypes.addAll(integerTypes);
complexVariantSubTypes.add(FLOAT);
complexVariantSubTypes.add(DOUBLE);
complexVariantSubTypes.add(DECIMALV2);
complexVariantSubTypes.add(DECIMAL32); // same DEFAULT_DECIMALV3
complexVariantSubTypes.add(DECIMAL64);
complexVariantSubTypes.add(DECIMAL128);
complexVariantSubTypes.add(DECIMAL256);
complexVariantSubTypes.add(TIME);
complexVariantSubTypes.add(TIMEV2);
complexVariantSubTypes.add(DATE);
complexVariantSubTypes.add(DATETIME);
complexVariantSubTypes.add(DATEV2);
complexVariantSubTypes.add(DATETIMEV2);
complexVariantSubTypes.add(IPV4);
complexVariantSubTypes.add(IPV6);
complexVariantSubTypes.add(CHAR);
complexVariantSubTypes.add(VARCHAR);
complexVariantSubTypes.add(STRING);
complexVariantSubTypes.add(NULL);
}

public static final Set<Class> DATE_SUPPORTED_JAVA_TYPE = Sets.newHashSet(LocalDate.class, java.util.Date.class,
Expand Down Expand Up @@ -375,6 +400,10 @@ public static ArrayList<Type> getStructSubTypes() {
return structSubTypes;
}

public static ArrayList<Type> getComplexVariantSubTypes() {
return complexVariantSubTypes;
}

/**
* Return true if this is complex type and support subType
*/
Expand Down Expand Up @@ -556,7 +585,7 @@ public boolean isJsonbType() {
}

public boolean isVariantType() {
return isScalarType(PrimitiveType.VARIANT);
return isScalarType(PrimitiveType.VARIANT) || isComplexVariant();
}

// only metric types have the following constraint:
Expand Down Expand Up @@ -688,6 +717,10 @@ public boolean isStructType() {
return this instanceof StructType;
}

public boolean isComplexVariant() {
return this instanceof ComplexVariantType;
}

public boolean isAnyType() {
return this instanceof AnyType;
}
Expand Down Expand Up @@ -981,6 +1014,13 @@ private boolean exceedsMaxNestingDepth(int d) {
} else if (isMapType()) {
MapType mapType = (MapType) this;
return mapType.getValueType().exceedsMaxNestingDepth(d + 1);
} else if (isComplexVariant()) {
ComplexVariantType complexVariantType = (ComplexVariantType) this;
for (StructField f : complexVariantType.getPredefinedFields()) {
if (f.getType().exceedsMaxNestingDepth(d + 1)) {
return true;
}
}
} else {
Preconditions.checkState(isScalarType() || isAggStateType());
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1000,6 +1000,7 @@ dataType
: complex=ARRAY LT dataType GT #complexDataType
| complex=MAP LT dataType COMMA dataType GT #complexDataType
| complex=STRUCT LT complexColTypeList GT #complexDataType
| VARIANT LT complexColTypeList GT #variantPredefined
| AGG_STATE LT functionNameIdentifier
LEFT_PAREN dataTypes+=dataTypeWithNullable
(COMMA dataTypes+=dataTypeWithNullable)* RIGHT_PAREN GT #aggStateDataType
Expand Down Expand Up @@ -1047,7 +1048,7 @@ complexColTypeList
;

complexColType
: identifier COLON dataType commentSpec?
: qualifiedName COLON dataType commentSpec?
;

commentSpec
Expand Down
9 changes: 5 additions & 4 deletions fe/fe-core/src/main/java/org/apache/doris/catalog/Column.java
Original file line number Diff line number Diff line change
Expand Up @@ -330,8 +330,9 @@ public void createChildrenColumn(Type type, Column column) {
v.setIsAllowNull(((MapType) type).getIsValueContainsNull());
column.addChildrenColumn(k);
column.addChildrenColumn(v);
} else if (type.isStructType()) {
ArrayList<StructField> fields = ((StructType) type).getFields();
} else if (type.isStructType() || type.isComplexVariant()) {
ArrayList<StructField> fields = type.isStructType()
? ((StructType) type).getFields() : ((ComplexVariantType) type).getPredefinedFields();
for (StructField field : fields) {
Column c = new Column(field.getName(), field.getType());
c.setIsAllowNull(field.getContainsNull());
Expand Down Expand Up @@ -657,7 +658,7 @@ private void toChildrenThrift(Column column, TColumn tColumn) {
tColumn.setChildrenColumn(new ArrayList<>());
setChildrenTColumn(k, tColumn);
setChildrenTColumn(v, tColumn);
} else if (column.type.isStructType()) {
} else if (column.type.isStructType() || column.type.isComplexVariant()) {
List<Column> childrenColumns = column.getChildren();
tColumn.setChildrenColumn(new ArrayList<>());
for (Column children : childrenColumns) {
Expand Down Expand Up @@ -802,7 +803,7 @@ public OlapFile.ColumnPB toPb(Set<String> bfColumns, List<Index> indexes) throws
builder.addChildrenColumns(k.toPb(Sets.newHashSet(), Lists.newArrayList()));
Column v = this.getChildren().get(1);
builder.addChildrenColumns(v.toPb(Sets.newHashSet(), Lists.newArrayList()));
} else if (this.type.isStructType()) {
} else if (this.type.isStructType() || this.type.isComplexVariant()) {
List<Column> childrenColumns = this.getChildren();
for (Column c : childrenColumns) {
builder.addChildrenColumns(c.toPb(Sets.newHashSet(), Lists.newArrayList()));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -202,6 +202,7 @@
import org.apache.doris.nereids.DorisParser.UpdateContext;
import org.apache.doris.nereids.DorisParser.UserIdentifyContext;
import org.apache.doris.nereids.DorisParser.UserVariableContext;
import org.apache.doris.nereids.DorisParser.VariantPredefinedContext;
import org.apache.doris.nereids.DorisParser.WhereClauseContext;
import org.apache.doris.nereids.DorisParser.WindowFrameContext;
import org.apache.doris.nereids.DorisParser.WindowSpecContext;
Expand Down Expand Up @@ -458,6 +459,7 @@
import org.apache.doris.nereids.types.ArrayType;
import org.apache.doris.nereids.types.BigIntType;
import org.apache.doris.nereids.types.BooleanType;
import org.apache.doris.nereids.types.ComplexVariantType;
import org.apache.doris.nereids.types.DataType;
import org.apache.doris.nereids.types.LargeIntType;
import org.apache.doris.nereids.types.MapType;
Expand Down Expand Up @@ -2703,6 +2705,8 @@ public ColumnDefinition visitColumnDef(ColumnDefContext ctx) {
? visitPrimitiveDataType(((PrimitiveDataTypeContext) ctx.type))
: ctx.type instanceof ComplexDataTypeContext
? visitComplexDataType((ComplexDataTypeContext) ctx.type)
: ctx.type instanceof VariantPredefinedContext
? visitVariantPredefined((VariantPredefinedContext) ctx.type)
: visitAggStateDataType((AggStateDataTypeContext) ctx.type);
colType = colType.conversion();
boolean isKey = ctx.KEY() != null;
Expand Down Expand Up @@ -3531,6 +3535,11 @@ public DataType visitPrimitiveDataType(PrimitiveDataTypeContext ctx) {
});
}

@Override
public DataType visitVariantPredefined(VariantPredefinedContext ctx) {
return new ComplexVariantType(visitComplexColTypeList(ctx.complexColTypeList()));
}

@Override
public DataType visitComplexDataType(ComplexDataTypeContext ctx) {
return ParserUtils.withOrigin(ctx, () -> {
Expand Down Expand Up @@ -3561,7 +3570,7 @@ public StructField visitComplexColType(ComplexColTypeContext ctx) {
} else {
comment = "";
}
return new StructField(ctx.identifier().getText(), typedVisit(ctx.dataType()), true, comment);
return new StructField(ctx.qualifiedName().getText(), typedVisit(ctx.dataType()), true, comment);
}

private String parseConstant(ConstantContext context) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -466,6 +466,20 @@ private void validateDataType(Type catalogType) {
}
}
}
} else if (catalogType.isComplexVariant()) {
ArrayList<org.apache.doris.catalog.StructField> predefinedFields =
((org.apache.doris.catalog.ComplexVariantType) catalogType).getPredefinedFields();
Set<String> fieldNames = new HashSet<>();
for (org.apache.doris.catalog.StructField field : predefinedFields) {
Type fieldType = field.getType();
if (fieldType instanceof ScalarType) {
validateNestedType(catalogType, (ScalarType) fieldType);
if (!fieldNames.add(field.getName())) {
throw new AnalysisException("Duplicate field name " + field.getName()
+ " in struct " + catalogType.toSql());
}
}
}
}
}

Expand Down
Loading

0 comments on commit 612ab0c

Please sign in to comment.