Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Refactor for Delta Uniform IcebergConversionTransaction to use current iceberg PartitionSpec #4174

Merged
merged 1 commit into from
Feb 20, 2025
Merged
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 @@ -32,7 +32,7 @@ import org.apache.spark.sql.delta.schema.SchemaUtils
import org.apache.spark.sql.delta.sources.DeltaSQLConf
import org.apache.commons.lang3.exception.ExceptionUtils
import org.apache.hadoop.conf.Configuration
import shadedForDelta.org.apache.iceberg.{AppendFiles, DeleteFiles, OverwriteFiles, PendingUpdate, RewriteFiles, Transaction => IcebergTransaction}
import shadedForDelta.org.apache.iceberg.{AppendFiles, DeleteFiles, OverwriteFiles, PartitionSpec, PendingUpdate, RewriteFiles, Transaction => IcebergTransaction}
import shadedForDelta.org.apache.iceberg.ExpireSnapshots
import shadedForDelta.org.apache.iceberg.mapping.MappingUtil
import shadedForDelta.org.apache.iceberg.mapping.NameMappingParser
Expand Down Expand Up @@ -64,7 +64,8 @@ class IcebergConversionTransaction(
protected val postCommitSnapshot: Snapshot,
protected val tableOp: IcebergTableOp = WRITE_TABLE,
protected val lastConvertedIcebergSnapshotId: Option[Long] = None,
protected val lastConvertedDeltaVersion: Option[Long] = None) extends DeltaLogging {
protected val lastConvertedDeltaVersion: Option[Long] = None
) extends DeltaLogging {

///////////////////////////
// Nested Helper Classes //
Expand Down Expand Up @@ -100,7 +101,7 @@ class IcebergConversionTransaction(
convertDeltaAddFileToIcebergDataFile(
add,
tablePath,
partitionSpec,
currentPartitionSpec,
logicalToPhysicalPartitionNames,
statsParser,
postCommitSnapshot
Expand Down Expand Up @@ -141,7 +142,7 @@ class IcebergConversionTransaction(
convertDeltaAddFileToIcebergDataFile(
add,
tablePath,
partitionSpec,
currentPartitionSpec,
logicalToPhysicalPartitionNames,
statsParser,
postCommitSnapshot
Expand All @@ -154,7 +155,7 @@ class IcebergConversionTransaction(
convertDeltaRemoveFileToIcebergDataFile(
remove,
tablePath,
partitionSpec,
currentPartitionSpec,
logicalToPhysicalPartitionNames,
postCommitSnapshot)
)
Expand All @@ -178,7 +179,7 @@ class IcebergConversionTransaction(
convertDeltaRemoveFileToIcebergDataFile(
f,
tablePath,
partitionSpec,
currentPartitionSpec,
logicalToPhysicalPartitionNames,
postCommitSnapshot)
}.toSet.asJava
Expand All @@ -188,7 +189,7 @@ class IcebergConversionTransaction(
convertDeltaAddFileToIcebergDataFile(
f,
tablePath,
partitionSpec,
currentPartitionSpec,
logicalToPhysicalPartitionNames,
statsParser,
postCommitSnapshot
Expand All @@ -212,8 +213,15 @@ class IcebergConversionTransaction(
protected val tablePath = postCommitSnapshot.deltaLog.dataPath
protected val icebergSchema =
convertDeltaSchemaToIcebergSchema(postCommitSnapshot.metadata.schema)
// Initial partition spec converted from Delta
protected val partitionSpec =
createPartitionSpec(icebergSchema, postCommitSnapshot.metadata.partitionColumns)

// Current partition spec from iceberg table
def currentPartitionSpec: PartitionSpec = {
Some(txn.table()).map(_.spec()).getOrElse(partitionSpec)
}

private val logicalToPhysicalPartitionNames =
getPartitionPhysicalNameMapping(postCommitSnapshot.metadata.partitionSchema)

Expand Down
Loading