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
Original file line number Diff line number Diff line change
Expand Up @@ -35,6 +35,7 @@ message Expression {
UnresolvedFunction unresolved_function = 3;
ExpressionString expression_string = 4;
UnresolvedStar unresolved_star = 5;
Alias alias = 6;
}

message Literal {
Expand Down Expand Up @@ -166,4 +167,9 @@ message Expression {
string name = 1;
DataType type = 2;
}

message Alias {
Copy link
Contributor

Choose a reason for hiding this comment

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

why wouldn't this be a common attribute of Expression instead?

Copy link
Contributor

Choose a reason for hiding this comment

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

Only a few places require Expression to be wrapped by an alias. I'd like to match catalyst and have an individual Alias expression.

Expression expr = 1;
string name = 2;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -40,6 +40,11 @@ package object dsl {
.build())
.build()
}

implicit class DslExpression(val expr: proto.Expression) {
def as(alias: String): proto.Expression = proto.Expression.newBuilder().setAlias(
proto.Expression.Alias.newBuilder().setName(alias).setExpr(expr)).build()
}
}

object plans { // scalastyle:ignore
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,7 +24,7 @@ import org.apache.spark.connect.proto
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.catalyst.analysis.{UnresolvedAlias, UnresolvedAttribute, UnresolvedFunction, UnresolvedRelation, UnresolvedStar}
import org.apache.spark.sql.catalyst.expressions
import org.apache.spark.sql.catalyst.expressions.{Attribute, AttributeReference, Expression}
import org.apache.spark.sql.catalyst.expressions.{Alias, Attribute, AttributeReference, Expression}
import org.apache.spark.sql.catalyst.plans.{logical, FullOuter, Inner, JoinType, LeftAnti, LeftOuter, LeftSemi, RightOuter}
import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, SubqueryAlias}
import org.apache.spark.sql.types._
Expand Down Expand Up @@ -132,6 +132,7 @@ class SparkConnectPlanner(plan: proto.Relation, session: SparkSession) {
transformUnresolvedExpression(exp)
case proto.Expression.ExprTypeCase.UNRESOLVED_FUNCTION =>
transformScalarFunction(exp.getUnresolvedFunction)
case proto.Expression.ExprTypeCase.ALIAS => transformAlias(exp.getAlias)
case _ => throw InvalidPlanInput()
}
}
Expand Down Expand Up @@ -208,6 +209,10 @@ class SparkConnectPlanner(plan: proto.Relation, session: SparkSession) {
}
}

private def transformAlias(alias: proto.Expression.Alias): Expression = {
Alias(transformExpression(alias.getExpr), alias.getName)()
}

private def transformUnion(u: proto.Union): LogicalPlan = {
assert(u.getInputsCount == 2, "Union must have 2 inputs")
val plan = logical.Union(transformRelation(u.getInputs(0)), transformRelation(u.getInputs(1)))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -81,6 +81,15 @@ class SparkConnectProtoSuite extends PlanTest with SparkConnectPlanTest {
}
}

test("column alias") {
val connectPlan = {
import org.apache.spark.sql.connect.dsl.expressions._
import org.apache.spark.sql.connect.dsl.plans._
transform(connectTestRelation.select("id".protoAttr.as("id2")))
}
val sparkPlan = sparkTestRelation.select($"id".as("id2"))
}
Copy link
Member

Choose a reason for hiding this comment

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

Actually this doesn't test :-). Let me make a quick followup.


test("Aggregate with more than 1 grouping expressions") {
val connectPlan = {
import org.apache.spark.sql.connect.dsl.expressions._
Expand Down