viirya commented on a change in pull request #29587:
URL: https://github.com/apache/spark/pull/29587#discussion_r491637817



##########
File path: 
sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/ResolveUnion.scala
##########
@@ -17,29 +17,168 @@
 
 package org.apache.spark.sql.catalyst.analysis
 
+import scala.collection.mutable
+
 import org.apache.spark.sql.AnalysisException
-import org.apache.spark.sql.catalyst.expressions.{Alias, Literal}
+import org.apache.spark.sql.catalyst.expressions.{Alias, Attribute, 
CreateNamedStruct, Expression, GetStructField, If, IsNull, KnownNotNull, 
Literal, NamedExpression, WithFields}
 import org.apache.spark.sql.catalyst.optimizer.CombineUnions
 import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, Project, 
Union}
 import org.apache.spark.sql.catalyst.rules.Rule
 import org.apache.spark.sql.internal.SQLConf
+import org.apache.spark.sql.types._
 import org.apache.spark.sql.util.SchemaUtils
+import org.apache.spark.unsafe.types.UTF8String
 
 /**
  * Resolves different children of Union to a common set of columns.
  */
 object ResolveUnion extends Rule[LogicalPlan] {
-  private def unionTwoSides(
+  /**
+   * This method sorts columns in a struct expression based on column names.
+   */
+  private def sortStructFields(expr: Expression): Expression = {
+    assert(expr.dataType.isInstanceOf[StructType])
+
+    val existingExprs = 
expr.dataType.asInstanceOf[StructType].fieldNames.zipWithIndex.map {
+      case (name, i) => (name, GetStructField(KnownNotNull(expr), 
i).asInstanceOf[Expression])
+    }.sortBy(_._1).flatMap(pair => Seq(Literal(pair._1), pair._2))
+
+    val newExpr = CreateNamedStruct(existingExprs)
+    if (expr.nullable) {
+      If(IsNull(expr), Literal(null, newExpr.dataType), newExpr)
+    } else {
+      newExpr
+    }
+  }
+
+  private def sortStructFields(fieldExprs: Seq[Expression]): Seq[Expression] = 
{
+    fieldExprs.grouped(2).map { e =>
+      Seq(e.head, e.last)
+    }.toSeq.sortBy { pair =>
+      assert(pair(0).isInstanceOf[Literal])
+      pair(0).eval().asInstanceOf[UTF8String].toString
+    }.flatten
+  }
+
+  /**
+   * This helper method sorts fields in a `WithFields` expression by field 
name.
+   */
+  private def sortStructFieldsInWithFields(expr: Expression): Expression = 
expr transformUp {
+    case w: WithFields if w.resolved =>
+      w.evalExpr match {
+        case i @ If(IsNull(_), _, CreateNamedStruct(fieldExprs)) =>
+          val sorted = sortStructFields(fieldExprs)
+          val newStruct = CreateNamedStruct(sorted)
+          i.copy(trueValue = Literal(null, newStruct.dataType), falseValue = 
newStruct)
+        case CreateNamedStruct(fieldExprs) =>
+          val sorted = sortStructFields(fieldExprs)
+          val newStruct = CreateNamedStruct(sorted)
+          newStruct
+        case other =>
+          throw new AnalysisException(s"`WithFields` has incorrect eval 
expression: $other")
+      }
+  }
+
+  /**
+   * Adds missing fields recursively into given `col` expression, based on the 
target `StructType`.
+   * This is called by `compareAndAddFields` when we find two struct columns 
with same name but
+   * different nested fields. This method will find out the missing nested 
fields from `col` to
+   * `target` struct and add these missing nested fields. Currently we don't 
support finding out
+   * missing nested fields of struct nested in array or struct nested in map.
+   */
+  private def addFields(col: NamedExpression, target: StructType): Expression 
= {
+    assert(col.dataType.isInstanceOf[StructType], "Only support StructType.")
+
+    val resolver = SQLConf.get.resolver
+    val missingFields =
+      StructType.findMissingFields(col.dataType.asInstanceOf[StructType], 
target, resolver)
+
+    // We need to sort columns in result, because we might add another column 
in other side.
+    // E.g., we want to union two structs "a int, b long" and "a int, c 
string".
+    // If we don't sort, we will have "a int, b long, c string" and
+    // "a int, c string, b long", which are not compatible.
+    if (missingFields.isEmpty) {
+      sortStructFields(col)

Review comment:
       @maropu I got you point when I was fixing the performance issue. Yeah, 
we should. I fixed it in latest commit. Thanks.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscr...@spark.apache.org
For additional commands, e-mail: reviews-h...@spark.apache.org

Reply via email to