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 @@ -158,7 +158,7 @@ object TypeCoercion {
}
case (MapType(kt1, vt1, valueContainsNull1), MapType(kt2, vt2, valueContainsNull2)) =>
findTypeFunc(kt1, kt2)
.filter { kt => !Cast.forceNullable(kt1, kt) && !Cast.forceNullable(kt2, kt) }
.filter { kt => Cast.canCastMapKeyNullSafe(kt1, kt) && Cast.canCastMapKeyNullSafe(kt2, kt) }
.flatMap { kt =>
findTypeFunc(vt1, vt2).map { vt =>
MapType(kt, vt, valueContainsNull1 || valueContainsNull2 ||
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -77,8 +77,7 @@ object Cast {
resolvableNullability(fn || forceNullable(fromType, toType), tn)

case (MapType(fromKey, fromValue, fn), MapType(toKey, toValue, tn)) =>
canCast(fromKey, toKey) &&
(!forceNullable(fromKey, toKey)) &&
canCast(fromKey, toKey) && canCastMapKeyNullSafe(fromKey, toKey) &&
canCast(fromValue, toValue) &&
resolvableNullability(fn || forceNullable(fromValue, toValue), tn)

Expand All @@ -98,6 +97,11 @@ object Cast {
case _ => false
}

def canCastMapKeyNullSafe(fromType: DataType, toType: DataType): Boolean = {
// If the original map key type is NullType, it's OK as the map must be empty.
fromType == NullType || !forceNullable(fromType, toType)
}

/**
* Return true if we need to use the `timeZone` information casting `from` type to `to` type.
* The patterns matched reflect the current implementation in the Cast node.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -3487,6 +3487,12 @@ class SQLQuerySuite extends QueryTest with SharedSparkSession with AdaptiveSpark
}
}
}

test("SPARK-31166: UNION map<null, null> and other maps should not fail") {
checkAnswer(
sql("(SELECT map()) UNION ALL (SELECT map(1, 2))"),
Seq(Row(Map[Int, Int]()), Row(Map(1 -> 2))))
}
Copy link
Member

Choose a reason for hiding this comment

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

Nice catch! It seems #27542 broke some other queries merging map<null,null> and map<some type, some type>, e.g., map_concat;

// the current master
scala> sql("select map_concat(map(), map(1, 2))").show()
org.apache.spark.sql.AnalysisException: cannot resolve 'map_concat(map(), map(1, 2))' due to data type mismatch: input to function map_concat should all be the same type, but it's [map<null,null>, map<int,int>]; line 1 pos 7;
'Project [unresolvedalias(map_concat(map(), map(1, 2)), None)]
+- OneRowRelation

  at org.apache.spark.sql.catalyst.analysis.package$AnalysisErrorAt.failAnalysis(package.scala:42)

This PR can fix this query, too.

}

case class Foo(bar: Option[String])