Skip to content

Commit

Permalink
Revert "[Spark] Fix Spark-master compile errors (#3591)"
Browse files Browse the repository at this point in the history
This reverts commit b843ad6.
  • Loading branch information
vkorukanti authored Sep 23, 2024
1 parent a99f62b commit ebc420d
Show file tree
Hide file tree
Showing 44 changed files with 24 additions and 274 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -25,7 +25,6 @@ import org.apache.spark.annotation.Evolving
import org.apache.spark.sql.{functions, Column, DataFrame, Dataset, Row, SparkSession}
import org.apache.spark.sql.catalyst.encoders.AgnosticEncoders.PrimitiveBooleanEncoder
import org.apache.spark.sql.connect.delta.ImplicitProtoConversions._
import org.apache.spark.sql.internal.ColumnNodeToProtoConverter.toExpr

/**
* Main class for programmatically interacting with Delta tables.
Expand Down Expand Up @@ -138,7 +137,7 @@ class DeltaTable private[tables](
val delete = proto.DeleteFromTable
.newBuilder()
.setTarget(df.plan.getRoot)
condition.foreach(c => delete.setCondition(toExpr(c)))
condition.foreach(c => delete.setCondition(c.expr))
val relation = proto.DeltaRelation.newBuilder().setDeleteFromTable(delete).build()
val extension = com.google.protobuf.Any.pack(relation)
val sparkRelation = spark_proto.Relation.newBuilder().setExtension(extension).build()
Expand Down Expand Up @@ -189,15 +188,15 @@ class DeltaTable private[tables](
val assignments = set.toSeq.map { case (field, value) =>
proto.Assignment
.newBuilder()
.setField(toExpr(functions.expr(field)))
.setValue(toExpr(value))
.setField(functions.expr(field).expr)
.setValue(value.expr)
.build()
}
val update = proto.UpdateTable
.newBuilder()
.setTarget(df.plan.getRoot)
.addAllAssignments(assignments.asJava)
condition.foreach(c => update.setCondition(toExpr(c)))
condition.foreach(c => update.setCondition(c.expr))
val relation = proto.DeltaRelation.newBuilder().setUpdateTable(update).build()
val extension = com.google.protobuf.Any.pack(relation)
val sparkRelation = spark_proto.Relation.newBuilder().setExtension(extension).build()
Expand Down
35 changes: 0 additions & 35 deletions spark/src/main/scala-spark-3.5/shims/ColumnExtShim.scala

This file was deleted.

This file was deleted.

53 changes: 0 additions & 53 deletions spark/src/main/scala-spark-master/shims/ColumnExtShim.scala

This file was deleted.

This file was deleted.

Original file line number Diff line number Diff line change
Expand Up @@ -28,7 +28,6 @@ import org.apache.spark.sql.delta.util.AnalysisHelper
import org.apache.spark.annotation._
import org.apache.spark.internal.Logging
import org.apache.spark.sql._
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.catalyst.ExtendedAnalysisException
import org.apache.spark.sql.catalyst.analysis.UnresolvedAttribute
import org.apache.spark.sql.catalyst.expressions.{AttributeReference, NamedExpression}
Expand Down
1 change: 0 additions & 1 deletion spark/src/main/scala/io/delta/tables/DeltaTable.scala
Original file line number Diff line number Diff line change
Expand Up @@ -29,7 +29,6 @@ import org.apache.hadoop.fs.Path

import org.apache.spark.annotation._
import org.apache.spark.sql._
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.catalyst.TableIdentifier
import org.apache.spark.sql.types.StructType

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,7 +26,6 @@ import org.apache.spark.sql.delta.commands.{DeltaGenerateCommand, DescribeDeltaD
import org.apache.spark.sql.delta.util.AnalysisHelper
import org.apache.hadoop.fs.Path

import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.{functions, Column, DataFrame}
import org.apache.spark.sql.catalyst.TableIdentifier
import org.apache.spark.sql.catalyst.analysis.{UnresolvedAttribute, UnresolvedRelation}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,6 @@ import org.apache.hadoop.mapreduce.{Job, TaskType}
import org.apache.spark.TaskContext
import org.apache.spark.internal.MDC
import org.apache.spark.paths.SparkPath
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.{Column, DataFrame, Dataset, Row, SparkSession}
import org.apache.spark.sql.catalyst.TableIdentifier
import org.apache.spark.sql.catalyst.analysis.UnresolvedAttribute
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,12 +26,7 @@ import org.apache.spark.sql.delta.metering.DeltaLogging
import org.apache.spark.sql.delta.schema.SchemaUtils
import org.apache.spark.sql.delta.sources.{DeltaSourceUtils, DeltaSQLConf, DeltaStreamUtils}

import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.DataFrame
import org.apache.spark.sql.Dataset
import org.apache.spark.sql.Encoder
import org.apache.spark.sql.catalyst.encoders.ExpressionEncoder
import org.apache.spark.sql.{Column, DataFrame}
import org.apache.spark.sql.catalyst.expressions.EqualNullSafe
import org.apache.spark.sql.catalyst.util.CaseInsensitiveMap
import org.apache.spark.sql.catalyst.util.ResolveDefaultColumns._
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -58,7 +58,6 @@ import org.apache.spark.sql.connector.catalog.{Identifier, TableCatalog}
import org.apache.spark.sql.connector.catalog.CatalogV2Implicits._
import org.apache.spark.sql.connector.expressions.{FieldReference, IdentityTransform, Transform}
import org.apache.spark.sql.errors.QueryCompilationErrors
import org.apache.spark.sql.errors.QueryCompilationErrorsShim._
import org.apache.spark.sql.execution.command.CreateTableLikeCommand
import org.apache.spark.sql.execution.command.RunnableCommand
import org.apache.spark.sql.execution.datasources.HadoopFsRelation
Expand Down Expand Up @@ -574,7 +573,7 @@ class DeltaAnalysis(session: SparkSession)
val v1TableName = child.identifier.asTableIdentifier
namespace.foreach { ns =>
if (v1TableName.database.exists(!resolver(_, ns.head))) {
throw showColumnsWithConflictDatabasesError(ns, v1TableName)
throw QueryCompilationErrors.showColumnsWithConflictDatabasesError(ns, v1TableName)
}
}
ShowDeltaTableColumnsCommand(child)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -42,7 +42,6 @@ import org.apache.hadoop.conf.Configuration
import org.apache.hadoop.fs.{FileStatus, FileSystem, Path}

import org.apache.spark.sql._
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.catalyst.{FileSourceOptions, TableIdentifier}
import org.apache.spark.sql.catalyst.analysis.{Resolver, UnresolvedAttribute}
import org.apache.spark.sql.catalyst.catalog.{BucketSpec, CatalogStatistics, CatalogTable}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,11 +27,7 @@ import org.apache.spark.sql.delta.sources.DeltaSourceUtils.GENERATION_EXPRESSION
import org.apache.spark.sql.delta.sources.DeltaSQLConf
import org.apache.spark.sql.delta.util.AnalysisHelper

import org.apache.spark.sql.AnalysisException
import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.Dataset
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.{AnalysisException, Column, Dataset, SparkSession}
import org.apache.spark.sql.catalyst.analysis.Analyzer
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.catalyst.expressions.aggregate.AggregateExpression
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -23,11 +23,7 @@ import org.apache.spark.sql.delta.util.JsonUtils
import org.apache.hadoop.conf.Configuration
import org.apache.hadoop.fs.Path

import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.DataFrame
import org.apache.spark.sql.Dataset
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.{Column, DataFrame, Dataset, SparkSession}
import org.apache.spark.sql.catalyst.analysis.UnresolvedAttribute
import org.apache.spark.sql.catalyst.expressions.{Attribute, Expression}
import org.apache.spark.sql.catalyst.plans.logical.{LocalRelation, LogicalPlan}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -51,11 +51,7 @@ import org.apache.hadoop.fs.{FileStatus, Path}

import org.apache.spark.SparkException
import org.apache.spark.internal.{MDC, MessageWithContext}
import org.apache.spark.sql.AnalysisException
import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.DataFrame
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.{AnalysisException, Column, DataFrame, SparkSession}
import org.apache.spark.sql.catalyst.catalog.CatalogTable
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.catalyst.types.DataTypeUtils.toAttributes
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -43,7 +43,6 @@ import org.apache.hadoop.fs.{FileStatus, Path}

import org.apache.spark.internal.Logging
import org.apache.spark.paths.SparkPath
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.{Column, Encoder, SparkSession}
import org.apache.spark.sql.catalyst.ScalaReflection
import org.apache.spark.sql.catalyst.encoders.ExpressionEncoder
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -37,7 +37,6 @@ import org.apache.hadoop.fs.Path

import org.apache.spark.paths.SparkPath
import org.apache.spark.sql._
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.catalyst.expressions.{AttributeReference, Expression}
import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, Project}
import org.apache.spark.sql.execution.datasources.{HadoopFsRelation, LogicalRelation}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -28,12 +28,7 @@ import org.apache.spark.sql.delta.sources.DeltaSQLConf
import com.fasterxml.jackson.databind.annotation.JsonDeserialize

import org.apache.spark.SparkContext
import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.DataFrame
import org.apache.spark.sql.Dataset
import org.apache.spark.sql.Row
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.{Column, DataFrame, Dataset, Row, SparkSession}
import org.apache.spark.sql.catalyst.analysis.EliminateSubqueryAliases
import org.apache.spark.sql.catalyst.catalog.CatalogTable
import org.apache.spark.sql.catalyst.expressions.{Attribute, AttributeReference, EqualNullSafe, Expression, If, Literal, Not}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -29,12 +29,7 @@ import com.fasterxml.jackson.databind.annotation.JsonDeserialize
import org.apache.hadoop.fs.Path

import org.apache.spark.SparkContext
import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.DataFrame
import org.apache.spark.sql.Dataset
import org.apache.spark.sql.Row
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.{Column, DataFrame, Dataset, Row, SparkSession}
import org.apache.spark.sql.catalyst.analysis.UnresolvedAttribute
import org.apache.spark.sql.catalyst.catalog.CatalogTable
import org.apache.spark.sql.catalyst.expressions.{Attribute, AttributeReference, Expression, If, Literal}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -32,7 +32,6 @@ import org.apache.spark.sql.delta.skipping.clustering.temp.ClusterBySpec
import org.apache.spark.sql.delta.sources.DeltaSQLConf

import org.apache.spark.sql._
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.catalyst.analysis.UnresolvedAttribute
import org.apache.spark.sql.catalyst.catalog.CatalogTable
import org.apache.spark.sql.catalyst.expressions.{And, Attribute, Expression, Literal}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -40,11 +40,7 @@ import org.apache.spark.sql.delta.stats.StatisticsCollection
import org.apache.hadoop.fs.Path

import org.apache.spark.internal.MDC
import org.apache.spark.sql.AnalysisException
import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.Row
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.{AnalysisException, Column, Row, SparkSession}
import org.apache.spark.sql.catalyst.analysis.{Resolver, UnresolvedAttribute}
import org.apache.spark.sql.catalyst.catalog.CatalogUtils
import org.apache.spark.sql.catalyst.expressions._
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -34,13 +34,7 @@ import org.apache.spark.sql.util.ScalaExtensions.OptionExt

import org.apache.spark.internal.MDC
import org.apache.spark.rdd.RDD
import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.DataFrame
import org.apache.spark.sql.Dataset
import org.apache.spark.sql.Row
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.SQLContext
import org.apache.spark.sql.{Column, DataFrame, Dataset, Row, SparkSession, SQLContext}
import org.apache.spark.sql.catalyst.InternalRow
import org.apache.spark.sql.catalyst.expressions.{And, Attribute, AttributeReference, Expression, Literal}
import org.apache.spark.sql.catalyst.plans.logical.Statistics
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,10 +26,7 @@ import org.apache.spark.sql.delta.commands.merge.MergeOutputGeneration.{SOURCE_R
import org.apache.spark.sql.delta.files.TahoeBatchFileIndex
import org.apache.spark.sql.delta.util.SetAccumulator

import org.apache.spark.sql.Column
import org.apache.spark.sql.ColumnImplicitsShim._
import org.apache.spark.sql.Dataset
import org.apache.spark.sql.SparkSession
import org.apache.spark.sql.{Column, Dataset, SparkSession}
import org.apache.spark.sql.catalyst.expressions.{And, Expression, Literal, Or}
import org.apache.spark.sql.catalyst.plans.logical.DeltaMergeIntoClause
import org.apache.spark.sql.functions.{coalesce, col, count, input_file_name, lit, monotonically_increasing_id, sum}
Expand Down
Loading

0 comments on commit ebc420d

Please sign in to comment.