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 @@ -148,7 +148,7 @@ class Analyzer(
private def assignAliases(exprs: Seq[NamedExpression]) = {
exprs.zipWithIndex.map {
case (expr, i) =>
expr transform {
expr transformUp {
case u @ UnresolvedAlias(child) => child match {
case ne: NamedExpression => ne
case e if !e.resolved => u
Expand Down
17 changes: 10 additions & 7 deletions sql/core/src/main/scala/org/apache/spark/sql/Column.scala
Original file line number Diff line number Diff line change
Expand Up @@ -132,6 +132,15 @@ class Column(protected[sql] val expr: Expression) extends Logging {
case explode: Explode => MultiAlias(explode, Nil)
case jt: JsonTuple => MultiAlias(jt, Nil)

// If we have a top level Cast, there is a chance to give it a better alias, if there is a
// NamedExpression under this Cast.
case c: Cast => c.transformUp {
case Cast(ne: NamedExpression, to) => UnresolvedAlias(Cast(ne, to))
} match {
case ne: NamedExpression => ne
case other => Alias(expr, expr.prettyString)()
}

case expr: Expression => Alias(expr, expr.prettyString)()
}

Expand Down Expand Up @@ -931,13 +940,7 @@ class Column(protected[sql] val expr: Expression) extends Logging {
* @group expr_ops
* @since 1.3.0
*/
def cast(to: DataType): Column = withExpr {
expr match {
// keeps the name of expression if possible when do cast.
case ne: NamedExpression => UnresolvedAlias(Cast(expr, to))
case _ => Cast(expr, to)
}
}
def cast(to: DataType): Column = withExpr { Cast(expr, to) }

/**
* Casts the column to a different data type, using the canonical string representation
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -991,6 +991,7 @@ class DataFrameSuite extends QueryTest with SharedSQLContext {
test("SPARK-10743: keep the name of expression if possible when do cast") {
val df = (1 to 10).map(Tuple1.apply).toDF("i").as("src")
assert(df.select($"src.i".cast(StringType)).columns.head === "i")
assert(df.select($"src.i".cast(StringType).cast(IntegerType)).columns.head === "i")
}

test("SPARK-11301: fix case sensitivity for filter on partitioned columns") {
Expand Down Expand Up @@ -1163,4 +1164,10 @@ class DataFrameSuite extends QueryTest with SharedSQLContext {
val primitiveUDF = udf((i: Int) => i * 2)
checkAnswer(df.select(primitiveUDF($"age")), Row(44) :: Row(null) :: Nil)
}

test("SPARK-12841: cast in filter") {
checkAnswer(
Seq(1 -> "a").toDF("i", "j").filter($"i".cast(StringType) === "1"),
Row(1, "a"))
}
}