Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
6 changes: 6 additions & 0 deletions common/utils/src/main/resources/error/error-conditions.json
Original file line number Diff line number Diff line change
Expand Up @@ -1693,6 +1693,12 @@
],
"sqlState" : "42846"
},
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I did not find we have similar non legacy error. Legacy errors are usually too specific, e.g. unsupported function.

"EXPRESSION_TRANSLATION_TO_V2_IS_NOT_SUPPORTED" : {
"message" : [
"Expression <expr> cannot be translated to v2 expression."
],
"sqlState" : "0A000"
},
"EXPRESSION_TYPE_IS_NOT_ORDERABLE" : {
"message" : [
"Column expression <expr> cannot be sorted because its type <exprType> is not orderable."
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,55 @@
/*
* 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.spark.sql.connector.expressions;

import org.apache.spark.annotation.Evolving;
import org.apache.spark.sql.internal.connector.ExpressionWithToString;

/**
* Get array item expression.
*
* @since 4.1.0
*/

@Evolving
public class GetArrayItem extends ExpressionWithToString {

private final Expression childArray;
private final Expression ordinal;
private final boolean failOnError;

/**
* Creates GetArrayItem expression.
* @param childArray Array that is source to get element from. Child of this expression.
* @param ordinal Ordinal of element. Zero-based indexing.
* @param failOnError Whether expression should throw exception for index out of bound or to
* return null.
*/
public GetArrayItem(Expression childArray, Expression ordinal, boolean failOnError) {
this.childArray = childArray;
this.ordinal = ordinal;
this.failOnError = failOnError;
}

public Expression childArray() { return this.childArray; }
public Expression ordinal() { return this.ordinal; }
public boolean failOnError() { return this.failOnError; }

@Override
public Expression[] children() { return new Expression[]{ childArray, ordinal }; }
}
Original file line number Diff line number Diff line change
Expand Up @@ -29,6 +29,7 @@
import org.apache.spark.sql.connector.expressions.Extract;
import org.apache.spark.sql.connector.expressions.NamedReference;
import org.apache.spark.sql.connector.expressions.GeneralScalarExpression;
import org.apache.spark.sql.connector.expressions.GetArrayItem;
import org.apache.spark.sql.connector.expressions.Literal;
import org.apache.spark.sql.connector.expressions.NullOrdering;
import org.apache.spark.sql.connector.expressions.SortDirection;
Expand Down Expand Up @@ -84,6 +85,8 @@ public String build(Expression expr) {
} else if (expr instanceof SortOrder sortOrder) {
return visitSortOrder(
build(sortOrder.expression()), sortOrder.direction(), sortOrder.nullOrdering());
} else if (expr instanceof GetArrayItem getArrayItem) {
return visitGetArrayItem(getArrayItem);
} else if (expr instanceof GeneralScalarExpression e) {
String name = e.name();
return switch (name) {
Expand Down Expand Up @@ -348,6 +351,13 @@ protected String visitTrim(String direction, String[] inputs) {
}
}

protected String visitGetArrayItem(GetArrayItem getArrayItem) {
throw new SparkUnsupportedOperationException(
"EXPRESSION_TRANSLATION_TO_V2_IS_NOT_SUPPORTED",
Map.of("expr", getArrayItem.toString())
);
}

protected String visitExtract(Extract extract) {
return visitExtract(extract.field(), build(extract.source()));
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,7 +24,7 @@ import org.apache.spark.sql.catalyst.expressions.aggregate.{AggregateExpression,
import org.apache.spark.sql.catalyst.expressions.objects.{Invoke, StaticInvoke}
import org.apache.spark.sql.catalyst.optimizer.ConstantFolding
import org.apache.spark.sql.connector.catalog.functions.ScalarFunction
import org.apache.spark.sql.connector.expressions.{Cast => V2Cast, Expression => V2Expression, Extract => V2Extract, FieldReference, GeneralScalarExpression, LiteralValue, NullOrdering, SortDirection, SortValue, UserDefinedScalarFunc}
import org.apache.spark.sql.connector.expressions.{Cast => V2Cast, Expression => V2Expression, Extract => V2Extract, FieldReference, GeneralScalarExpression, GetArrayItem => V2GetArrayItem, LiteralValue, NullOrdering, SortDirection, SortValue, UserDefinedScalarFunc}
import org.apache.spark.sql.connector.expressions.aggregate.{AggregateFunc, Avg, Count, CountStar, GeneralAggregateFunc, Max, Min, Sum, UserDefinedAggregateFunc}
import org.apache.spark.sql.connector.expressions.filter.{AlwaysFalse, AlwaysTrue, And => V2And, Not => V2Not, Or => V2Or, Predicate => V2Predicate}
import org.apache.spark.sql.internal.SQLConf
Expand Down Expand Up @@ -326,6 +326,13 @@ class V2ExpressionBuilder(e: Expression, isPredicate: Boolean = false) extends L
case _: Sha2 => generateExpressionWithName("SHA2", expr, isPredicate)
case _: StringLPad => generateExpressionWithName("LPAD", expr, isPredicate)
case _: StringRPad => generateExpressionWithName("RPAD", expr, isPredicate)
case GetArrayItem(child, ordinal, failOnError) =>
(generateExpression(child), generateExpression(ordinal)) match {
case (Some(v2ArrayChild), Some(v2Ordinal)) =>
Some(new V2GetArrayItem(v2ArrayChild, v2Ordinal, failOnError))
case _ =>
None
}
// TODO supports other expressions
case ApplyFunctionExpression(function, children) =>
val childrenExpressions = children.flatMap(generateExpression(_))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@

package org.apache.spark.sql.internal.connector

import org.apache.spark.sql.connector.expressions.GetArrayItem
import org.apache.spark.sql.connector.util.V2ExpressionSQLBuilder

/**
Expand All @@ -35,4 +36,8 @@ class ToStringSQLBuilder extends V2ExpressionSQLBuilder with Serializable {
val distinct = if (isDistinct) "DISTINCT " else ""
s"""$funcName($distinct${inputs.mkString(", ")})"""
}

override protected def visitGetArrayItem(getArrayItem: GetArrayItem): String = {
s"${getArrayItem.childArray.toString}[${getArrayItem.ordinal.toString}]"
}
}