diff --git a/core/src/main/java/org/apache/iceberg/util/SortOrderUtil.java b/core/src/main/java/org/apache/iceberg/util/SortOrderUtil.java
index 37e0c1fffab0..4d7a631ab559 100644
--- a/core/src/main/java/org/apache/iceberg/util/SortOrderUtil.java
+++ b/core/src/main/java/org/apache/iceberg/util/SortOrderUtil.java
@@ -46,6 +46,23 @@ public static SortOrder buildSortOrder(Table table, SortOrder sortOrder) {
return buildSortOrder(table.schema(), table.spec(), sortOrder);
}
+ /**
+ * Attempts to match a user-supplied {@link SortOrder} with an equivalent sort order from a {@link
+ * Table}.
+ *
+ * @param table the table to try and match the sort order against
+ * @param userSuppliedSortOrder the user supplied sort order to try and match with a table sort
+ * order
+ * @return the matching {@link SortOrder} from the table (with the orderId set) or {@link
+ * SortOrder#unsorted()} if no match is found.
+ */
+ public static SortOrder maybeFindTableSortOrder(Table table, SortOrder userSuppliedSortOrder) {
+ return table.sortOrders().values().stream()
+ .filter(sortOrder -> sortOrder.sameOrder(userSuppliedSortOrder))
+ .findFirst()
+ .orElseGet(SortOrder::unsorted);
+ }
+
/**
* Build a final sort order that satisfies the clustering required by the partition spec.
*
diff --git a/core/src/test/java/org/apache/iceberg/util/TestSortOrderUtil.java b/core/src/test/java/org/apache/iceberg/util/TestSortOrderUtil.java
index 02c81de93222..3757b70dd334 100644
--- a/core/src/test/java/org/apache/iceberg/util/TestSortOrderUtil.java
+++ b/core/src/test/java/org/apache/iceberg/util/TestSortOrderUtil.java
@@ -287,4 +287,68 @@ public void testSortOrderClusteringWithRedundantPartitionFieldsMissing() {
.as("Should add spec fields as prefix")
.isEqualTo(expected);
}
+
+ @Test
+ public void testFindSortOrderForTable() {
+ PartitionSpec spec = PartitionSpec.unpartitioned();
+ SortOrder order = SortOrder.builderFor(SCHEMA).withOrderId(1).asc("id", NULLS_LAST).build();
+ TestTables.TestTable table = TestTables.create(tableDir, "test", SCHEMA, spec, order, 2);
+
+ SortOrder tableSortOrder = table.sortOrder();
+
+ SortOrder actualOrder = SortOrderUtil.maybeFindTableSortOrder(table, tableSortOrder);
+
+ assertThat(actualOrder).as("Should find current table sort order").isEqualTo(table.sortOrder());
+ }
+
+ @Test
+ public void testFindSortOrderForTableWithoutFieldId() {
+ PartitionSpec spec = PartitionSpec.unpartitioned();
+ SortOrder order = SortOrder.builderFor(SCHEMA).withOrderId(1).asc("id", NULLS_LAST).build();
+ TestTables.TestTable table = TestTables.create(tableDir, "test", SCHEMA, spec, order, 2);
+
+ SortOrder userSuppliedOrder =
+ SortOrder.builderFor(table.schema()).asc("id", NULLS_LAST).build();
+
+ SortOrder actualOrder = SortOrderUtil.maybeFindTableSortOrder(table, userSuppliedOrder);
+
+ assertThat(actualOrder).as("Should find current table sort order").isEqualTo(table.sortOrder());
+ }
+
+ @Test
+ public void testFindSortOrderForTableThatIsNotCurrentOrder() {
+ PartitionSpec spec = PartitionSpec.unpartitioned();
+ SortOrder order = SortOrder.builderFor(SCHEMA).withOrderId(1).asc("id", NULLS_LAST).build();
+ TestTables.TestTable table = TestTables.create(tableDir, "test", SCHEMA, spec, order, 2);
+
+ table.replaceSortOrder().asc("data").desc("ts").commit();
+
+ SortOrder userSuppliedOrder =
+ SortOrder.builderFor(table.schema()).asc("id", NULLS_LAST).build();
+
+ SortOrder actualOrder = SortOrderUtil.maybeFindTableSortOrder(table, userSuppliedOrder);
+
+ assertThat(actualOrder)
+ .as("Should find first sorted table sort order")
+ .isEqualTo(table.sortOrders().get(1));
+ }
+
+ @Test
+ public void testReturnsEmptyForFindingNonMatchingSortOrder() {
+ PartitionSpec spec = PartitionSpec.unpartitioned();
+ SortOrder order = SortOrder.builderFor(SCHEMA).withOrderId(1).asc("id", NULLS_LAST).build();
+ TestTables.TestTable table = TestTables.create(tableDir, "test", SCHEMA, spec, order, 2);
+
+ table.replaceSortOrder().asc("data").desc("ts").commit();
+
+ SortOrder userSuppliedOrder =
+ SortOrder.builderFor(table.schema()).desc("id", NULLS_LAST).build();
+
+ SortOrder actualOrder = SortOrderUtil.maybeFindTableSortOrder(table, userSuppliedOrder);
+
+ assertThat(actualOrder)
+ .as(
+ "Should return unsorted order if user supplied order does not match any table sort order")
+ .isEqualTo(SortOrder.unsorted());
+ }
}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkReadConf.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkReadConf.java
index 2788e160d526..b20ad1c86f71 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkReadConf.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkReadConf.java
@@ -268,6 +268,14 @@ public boolean preserveDataGrouping() {
.parse();
}
+ public boolean preserveDataOrdering() {
+ return confParser
+ .booleanConf()
+ .sessionConf(SparkSQLProperties.PRESERVE_DATA_ORDERING)
+ .defaultValue(SparkSQLProperties.PRESERVE_DATA_ORDERING_DEFAULT)
+ .parse();
+ }
+
public boolean aggregatePushDownEnabled() {
return confParser
.booleanConf()
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkSQLProperties.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkSQLProperties.java
index 81139969f746..c5ff0609a051 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkSQLProperties.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkSQLProperties.java
@@ -43,6 +43,11 @@ private SparkSQLProperties() {}
"spark.sql.iceberg.planning.preserve-data-grouping";
public static final boolean PRESERVE_DATA_GROUPING_DEFAULT = false;
+ // Controls whether to preserve data ordering and report it to Spark
+ public static final String PRESERVE_DATA_ORDERING =
+ "spark.sql.iceberg.planning.preserve-data-ordering";
+ public static final boolean PRESERVE_DATA_ORDERING_DEFAULT = false;
+
// Controls whether to push down aggregate (MAX/MIN/COUNT) to Iceberg
public static final String AGGREGATE_PUSH_DOWN_ENABLED =
"spark.sql.iceberg.aggregate-push-down.enabled";
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteConf.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteConf.java
index 96131e0e56dd..f85fb0dfb9ff 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteConf.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteConf.java
@@ -42,6 +42,7 @@
import org.apache.iceberg.FileFormat;
import org.apache.iceberg.IsolationLevel;
import org.apache.iceberg.SnapshotSummary;
+import org.apache.iceberg.SortOrder;
import org.apache.iceberg.Table;
import org.apache.iceberg.TableProperties;
import org.apache.iceberg.TableUtil;
@@ -164,6 +165,22 @@ public int outputSpecId() {
return outputSpecId;
}
+ public SortOrder outputSortOrder() {
+ int outputSortOrderId =
+ confParser
+ .intConf()
+ .option(SparkWriteOptions.OUTPUT_SORT_ORDER_ID)
+ .defaultValue(SortOrder.unsorted().orderId())
+ .parse();
+
+ Preconditions.checkArgument(
+ table.sortOrders().containsKey(outputSortOrderId),
+ "Output sort order id %s is not a valid sort order id for table",
+ outputSortOrderId);
+
+ return table.sortOrders().get(outputSortOrderId);
+ }
+
public FileFormat dataFileFormat() {
String valueAsString =
confParser
@@ -284,6 +301,21 @@ public SparkWriteRequirements writeRequirements() {
table, distributionMode(), fanoutWriterEnabled(), dataAdvisoryPartitionSize());
}
+ public SparkWriteRequirements rewriteFilesWriteRequirements() {
+ Preconditions.checkNotNull(
+ rewrittenFileSetId(), "Can only use rewrite files write requirements during rewrite job!");
+
+ SortOrder outputSortOrder = outputSortOrder();
+ if (outputSortOrder.isSorted()) {
+ LOG.info(
+ "Found explicit sort order {} set in job configuration. Going to apply that to the sort-order-id of the rewritten files",
+ Spark3Util.describe(outputSortOrder));
+ return writeRequirements().withTableSortOrder(outputSortOrder);
+ }
+
+ return writeRequirements();
+ }
+
@VisibleForTesting
DistributionMode distributionMode() {
String modeName =
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteOptions.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteOptions.java
index 33db70bae587..1be02feaf0c0 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteOptions.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteOptions.java
@@ -54,6 +54,7 @@ private SparkWriteOptions() {}
public static final String REWRITTEN_FILE_SCAN_TASK_SET_ID = "rewritten-file-scan-task-set-id";
public static final String OUTPUT_SPEC_ID = "output-spec-id";
+ public static final String OUTPUT_SORT_ORDER_ID = "output-sort-order-id";
public static final String OVERWRITE_MODE = "overwrite-mode";
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteRequirements.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteRequirements.java
index 833e0e44e391..dd4bc863912f 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteRequirements.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteRequirements.java
@@ -26,18 +26,32 @@
/** A set of requirements such as distribution and ordering reported to Spark during writes. */
public class SparkWriteRequirements {
+ public static final long NO_ADVISORY_PARTITION_SIZE = 0;
public static final SparkWriteRequirements EMPTY =
- new SparkWriteRequirements(Distributions.unspecified(), new SortOrder[0], 0);
+ new SparkWriteRequirements(
+ Distributions.unspecified(),
+ new SortOrder[0],
+ org.apache.iceberg.SortOrder.unsorted(),
+ NO_ADVISORY_PARTITION_SIZE);
private final Distribution distribution;
private final SortOrder[] ordering;
+ private final org.apache.iceberg.SortOrder icebergOrdering;
private final long advisoryPartitionSize;
SparkWriteRequirements(
- Distribution distribution, SortOrder[] ordering, long advisoryPartitionSize) {
+ Distribution distribution,
+ SortOrder[] ordering,
+ org.apache.iceberg.SortOrder icebergOrdering,
+ long advisoryPartitionSize) {
this.distribution = distribution;
this.ordering = ordering;
- this.advisoryPartitionSize = advisoryPartitionSize;
+ this.icebergOrdering = icebergOrdering;
+ // Spark prohibits requesting a particular advisory partition size without distribution
+ this.advisoryPartitionSize =
+ distribution instanceof UnspecifiedDistribution
+ ? NO_ADVISORY_PARTITION_SIZE
+ : advisoryPartitionSize;
}
public Distribution distribution() {
@@ -48,12 +62,19 @@ public SortOrder[] ordering() {
return ordering;
}
+ public org.apache.iceberg.SortOrder icebergOrdering() {
+ return icebergOrdering;
+ }
+
public boolean hasOrdering() {
return ordering.length != 0;
}
public long advisoryPartitionSize() {
- // Spark prohibits requesting a particular advisory partition size without distribution
- return distribution instanceof UnspecifiedDistribution ? 0 : advisoryPartitionSize;
+ return advisoryPartitionSize;
+ }
+
+ public SparkWriteRequirements withTableSortOrder(org.apache.iceberg.SortOrder sortOrder) {
+ return new SparkWriteRequirements(distribution, ordering, sortOrder, advisoryPartitionSize);
}
}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteUtil.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteUtil.java
index 0d68a0d8cdd0..535674aba977 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteUtil.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/SparkWriteUtil.java
@@ -55,13 +55,23 @@ public class SparkWriteUtil {
private static final Expression[] PARTITION_FILE_CLUSTERING =
clusterBy(SPEC_ID, PARTITION, FILE_PATH);
- private static final SortOrder[] EMPTY_ORDERING = new SortOrder[0];
- private static final SortOrder[] EXISTING_ROW_ORDERING = orderBy(FILE_PATH, ROW_POSITION);
- private static final SortOrder[] PARTITION_ORDERING = orderBy(SPEC_ID, PARTITION);
- private static final SortOrder[] PARTITION_FILE_ORDERING = orderBy(SPEC_ID, PARTITION, FILE_PATH);
- private static final SortOrder[] POSITION_DELETE_ORDERING =
+ private static final SortOrder[] EMPTY_SPARK_ORDERING = new SortOrder[0];
+ private static final SortOrder[] EXISTING_ROW_SPARK_ORDERING = orderBy(FILE_PATH, ROW_POSITION);
+ private static final SortOrder[] PARTITION_SPARK_ORDERING = orderBy(SPEC_ID, PARTITION);
+ private static final SortOrder[] PARTITION_FILE_SPARK_ORDERING =
+ orderBy(SPEC_ID, PARTITION, FILE_PATH);
+ private static final SortOrder[] POSITION_DELETE_SPARK_ORDERING =
orderBy(SPEC_ID, PARTITION, FILE_PATH, ROW_POSITION);
+ private static final SparkAndIcebergOrdering EXISTING_ROW_ORDERING =
+ SparkAndIcebergOrdering.unsorted().prependOrder(EXISTING_ROW_SPARK_ORDERING);
+ private static final SparkAndIcebergOrdering PARTITION_ORDERING =
+ SparkAndIcebergOrdering.unsorted().prependOrder(PARTITION_SPARK_ORDERING);
+ private static final SparkAndIcebergOrdering PARTITION_FILE_ORDERING =
+ SparkAndIcebergOrdering.unsorted().prependOrder(PARTITION_FILE_SPARK_ORDERING);
+ private static final SparkAndIcebergOrdering POSITION_DELETE_ORDERING =
+ SparkAndIcebergOrdering.unsorted().prependOrder(POSITION_DELETE_SPARK_ORDERING);
+
private SparkWriteUtil() {}
/** Builds requirements for batch and micro-batch writes such as append or overwrite. */
@@ -69,8 +79,9 @@ public static SparkWriteRequirements writeRequirements(
Table table, DistributionMode mode, boolean fanoutEnabled, long advisoryPartitionSize) {
Distribution distribution = writeDistribution(table, mode);
- SortOrder[] ordering = writeOrdering(table, fanoutEnabled);
- return new SparkWriteRequirements(distribution, ordering, advisoryPartitionSize);
+ SparkAndIcebergOrdering ordering = writeOrdering(table, fanoutEnabled);
+ return new SparkWriteRequirements(
+ distribution, ordering.sparkOrder(), ordering.icebergOrder(), advisoryPartitionSize);
}
private static Distribution writeDistribution(Table table, DistributionMode mode) {
@@ -82,7 +93,7 @@ private static Distribution writeDistribution(Table table, DistributionMode mode
return Distributions.clustered(clustering(table));
case RANGE:
- return Distributions.ordered(ordering(table));
+ return Distributions.ordered(ordering(table).sparkOrder());
default:
throw new IllegalArgumentException("Unsupported distribution mode: " + mode);
@@ -99,8 +110,9 @@ public static SparkWriteRequirements copyOnWriteRequirements(
if (command == DELETE || command == UPDATE) {
Distribution distribution = copyOnWriteDeleteUpdateDistribution(table, mode);
- SortOrder[] ordering = writeOrdering(table, fanoutEnabled);
- return new SparkWriteRequirements(distribution, ordering, advisoryPartitionSize);
+ SparkAndIcebergOrdering ordering = writeOrdering(table, fanoutEnabled);
+ return new SparkWriteRequirements(
+ distribution, ordering.sparkOrder(), ordering.icebergOrder(), advisoryPartitionSize);
} else {
return writeRequirements(table, mode, fanoutEnabled, advisoryPartitionSize);
}
@@ -122,9 +134,9 @@ private static Distribution copyOnWriteDeleteUpdateDistribution(
case RANGE:
if (table.spec().isPartitioned() || table.sortOrder().isSorted()) {
- return Distributions.ordered(ordering(table));
+ return Distributions.ordered(ordering(table).sparkOrder());
} else {
- return Distributions.ordered(EXISTING_ROW_ORDERING);
+ return Distributions.ordered(EXISTING_ROW_ORDERING.sparkOrder());
}
default:
@@ -142,12 +154,15 @@ public static SparkWriteRequirements positionDeltaRequirements(
if (command == UPDATE || command == MERGE) {
Distribution distribution = positionDeltaUpdateMergeDistribution(table, mode);
- SortOrder[] ordering = positionDeltaUpdateMergeOrdering(table, fanoutEnabled);
- return new SparkWriteRequirements(distribution, ordering, advisoryPartitionSize);
+ SparkAndIcebergOrdering ordering = positionDeltaUpdateMergeOrdering(table, fanoutEnabled);
+ return new SparkWriteRequirements(
+ distribution, ordering.sparkOrder(), ordering.icebergOrder(), advisoryPartitionSize);
} else {
Distribution distribution = positionDeltaDeleteDistribution(table, mode);
- SortOrder[] ordering = fanoutEnabled ? EMPTY_ORDERING : POSITION_DELETE_ORDERING;
- return new SparkWriteRequirements(distribution, ordering, advisoryPartitionSize);
+ SparkAndIcebergOrdering ordering =
+ fanoutEnabled ? SparkAndIcebergOrdering.unsorted() : POSITION_DELETE_ORDERING;
+ return new SparkWriteRequirements(
+ distribution, ordering.sparkOrder(), ordering.icebergOrder(), advisoryPartitionSize);
}
}
@@ -167,9 +182,15 @@ private static Distribution positionDeltaUpdateMergeDistribution(
case RANGE:
if (table.spec().isUnpartitioned()) {
- return Distributions.ordered(concat(PARTITION_FILE_ORDERING, ordering(table)));
+ return Distributions.ordered(
+ SparkAndIcebergOrdering.forTable(table)
+ .prependOrder(PARTITION_FILE_SPARK_ORDERING)
+ .sparkOrder());
} else {
- return Distributions.ordered(concat(PARTITION_ORDERING, ordering(table)));
+ return Distributions.ordered(
+ SparkAndIcebergOrdering.forTable(table)
+ .prependOrder(PARTITION_SPARK_ORDERING)
+ .sparkOrder());
}
default:
@@ -177,11 +198,12 @@ private static Distribution positionDeltaUpdateMergeDistribution(
}
}
- private static SortOrder[] positionDeltaUpdateMergeOrdering(Table table, boolean fanoutEnabled) {
+ private static SparkAndIcebergOrdering positionDeltaUpdateMergeOrdering(
+ Table table, boolean fanoutEnabled) {
if (fanoutEnabled && table.sortOrder().isUnsorted()) {
- return EMPTY_ORDERING;
+ return SparkAndIcebergOrdering.unsorted();
} else {
- return concat(POSITION_DELETE_ORDERING, ordering(table));
+ return SparkAndIcebergOrdering.forTable(table).prependOrder(POSITION_DELETE_SPARK_ORDERING);
}
}
@@ -199,9 +221,9 @@ private static Distribution positionDeltaDeleteDistribution(Table table, Distrib
case RANGE:
if (table.spec().isUnpartitioned()) {
- return Distributions.ordered(PARTITION_FILE_ORDERING);
+ return Distributions.ordered(PARTITION_FILE_ORDERING.sparkOrder());
} else {
- return Distributions.ordered(PARTITION_ORDERING);
+ return Distributions.ordered(PARTITION_ORDERING.sparkOrder());
}
default:
@@ -213,9 +235,9 @@ private static Distribution positionDeltaDeleteDistribution(Table table, Distrib
// - there is a defined table sort order, so it is clear how the data should be ordered
// - the table is partitioned and fanout writers are disabled,
// so records for one partition must be co-located within a task
- private static SortOrder[] writeOrdering(Table table, boolean fanoutEnabled) {
+ private static SparkAndIcebergOrdering writeOrdering(Table table, boolean fanoutEnabled) {
if (fanoutEnabled && table.sortOrder().isUnsorted()) {
- return EMPTY_ORDERING;
+ return SparkAndIcebergOrdering.unsorted();
} else {
return ordering(table);
}
@@ -225,8 +247,8 @@ private static Expression[] clustering(Table table) {
return Spark3Util.toTransforms(table.spec());
}
- private static SortOrder[] ordering(Table table) {
- return Spark3Util.toOrdering(SortOrderUtil.buildSortOrder(table));
+ private static SparkAndIcebergOrdering ordering(Table table) {
+ return SparkAndIcebergOrdering.forTable(table);
}
private static Expression[] concat(Expression[] clustering, Expression... otherClustering) {
@@ -256,4 +278,39 @@ private static SortOrder[] orderBy(Expression... exprs) {
private static SortOrder sort(Expression expr) {
return Expressions.sort(expr, SortDirection.ASCENDING);
}
+
+ private static class SparkAndIcebergOrdering {
+ private static final SparkAndIcebergOrdering UNSORTED =
+ new SparkAndIcebergOrdering(org.apache.iceberg.SortOrder.unsorted(), EMPTY_SPARK_ORDERING);
+
+ private final org.apache.iceberg.SortOrder icebergSortOrder;
+ private final SortOrder[] sparkSortOrder;
+
+ private SparkAndIcebergOrdering(
+ org.apache.iceberg.SortOrder icebergSortOrder, SortOrder[] sparkSortOrder) {
+ this.icebergSortOrder = icebergSortOrder;
+ this.sparkSortOrder = sparkSortOrder;
+ }
+
+ public static SparkAndIcebergOrdering forTable(Table table) {
+ return new SparkAndIcebergOrdering(
+ table.sortOrder(), Spark3Util.toOrdering(SortOrderUtil.buildSortOrder(table)));
+ }
+
+ public static SparkAndIcebergOrdering unsorted() {
+ return UNSORTED;
+ }
+
+ public SparkAndIcebergOrdering prependOrder(SortOrder[] ordering) {
+ return new SparkAndIcebergOrdering(icebergSortOrder, concat(ordering, sparkSortOrder));
+ }
+
+ public org.apache.iceberg.SortOrder icebergOrder() {
+ return icebergSortOrder;
+ }
+
+ public SortOrder[] sparkOrder() {
+ return sparkSortOrder;
+ }
+ }
}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/actions/SparkShufflingFileRewriteRunner.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/actions/SparkShufflingFileRewriteRunner.java
index 569eb252cba5..1ba4c7e2fac2 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/actions/SparkShufflingFileRewriteRunner.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/actions/SparkShufflingFileRewriteRunner.java
@@ -47,10 +47,14 @@
import org.apache.spark.sql.connector.expressions.SortOrder;
import org.apache.spark.sql.connector.write.RequiresDistributionAndOrdering;
import org.apache.spark.sql.execution.datasources.v2.DistributionAndOrderingUtils$;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
import scala.Option;
abstract class SparkShufflingFileRewriteRunner extends SparkDataFileRewriteRunner {
+ private static final Logger LOG = LoggerFactory.getLogger(SparkShufflingFileRewriteRunner.class);
+
/**
* The number of shuffle partitions to use for each output file. By default, this file rewriter
* assumes each shuffle partition would become a separate output file. Attempting to generate
@@ -119,6 +123,17 @@ public void doRewrite(String groupId, RewriteFileGroup fileGroup) {
spec(fileGroup.outputSpecId()),
fileGroup.expectedOutputFiles()));
+ org.apache.iceberg.SortOrder sortOrderInJobSpec = sortOrder();
+
+ org.apache.iceberg.SortOrder maybeMatchingTableSortOrder =
+ SortOrderUtil.maybeFindTableSortOrder(table(), sortOrder());
+
+ if (sortOrderInJobSpec.isSorted() && maybeMatchingTableSortOrder.isUnsorted()) {
+ LOG.warn(
+ "Sort order specified for job {} doesn't match any table sort orders, so going to not mark rewritten files as sorted in the manifest files",
+ Spark3Util.describe(sortOrderInJobSpec));
+ }
+
sortedDF
.write()
.format("iceberg")
@@ -126,6 +141,7 @@ public void doRewrite(String groupId, RewriteFileGroup fileGroup) {
.option(SparkWriteOptions.TARGET_FILE_SIZE_BYTES, fileGroup.maxOutputFileSize())
.option(SparkWriteOptions.USE_TABLE_DISTRIBUTION_AND_ORDERING, "false")
.option(SparkWriteOptions.OUTPUT_SPEC_ID, fileGroup.outputSpecId())
+ .option(SparkWriteOptions.OUTPUT_SORT_ORDER_ID, maybeMatchingTableSortOrder.orderId())
.mode("append")
.save(groupId);
}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/InternalRowComparator.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/InternalRowComparator.java
new file mode 100644
index 000000000000..d1ffcea10149
--- /dev/null
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/InternalRowComparator.java
@@ -0,0 +1,72 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.iceberg.spark.source;
+
+import java.util.Comparator;
+import org.apache.iceberg.Schema;
+import org.apache.iceberg.SortOrder;
+import org.apache.iceberg.SortOrderComparators;
+import org.apache.iceberg.StructLike;
+import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
+import org.apache.spark.sql.catalyst.InternalRow;
+import org.apache.spark.sql.types.StructType;
+
+/**
+ * A comparator for Spark {@link InternalRow} objects based on an Iceberg {@link SortOrder}.
+ *
+ *
This comparator adapts Spark's InternalRow to Iceberg's StructLike interface and delegates to
+ * Iceberg's existing {@link SortOrderComparators} infrastructure, which provides full support for:
+ *
+ *
This class is NOT thread-safe.
+ */
+class InternalRowComparator implements Comparator {
+ private final Comparator delegate;
+ private final InternalRowWrapper leftWrapper;
+ private final InternalRowWrapper rightWrapper;
+
+ /**
+ * Creates a comparator for the given sort order and schemas.
+ *
+ * @param sortOrder the Iceberg sort order to use for comparison
+ * @param sparkSchema the Spark schema of the rows to compare
+ * @param icebergSchema the Iceberg schema of the rows to compare
+ */
+ InternalRowComparator(SortOrder sortOrder, StructType sparkSchema, Schema icebergSchema) {
+ Preconditions.checkArgument(
+ sortOrder.isSorted(), "Cannot create comparator for unsorted order");
+ Preconditions.checkNotNull(sparkSchema, "Spark schema cannot be null");
+ Preconditions.checkNotNull(icebergSchema, "Iceberg schema cannot be null");
+
+ this.delegate = SortOrderComparators.forSchema(icebergSchema, sortOrder);
+ this.leftWrapper = new InternalRowWrapper(sparkSchema, icebergSchema.asStruct());
+ this.rightWrapper = new InternalRowWrapper(sparkSchema, icebergSchema.asStruct());
+ }
+
+ @Override
+ public int compare(InternalRow row1, InternalRow row2) {
+ return delegate.compare(leftWrapper.wrap(row1), rightWrapper.wrap(row2));
+ }
+}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/MergingPartitionReader.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/MergingPartitionReader.java
new file mode 100644
index 000000000000..e4997c3eb94a
--- /dev/null
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/MergingPartitionReader.java
@@ -0,0 +1,150 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.iceberg.spark.source;
+
+import java.io.IOException;
+import java.util.Comparator;
+import java.util.List;
+import java.util.stream.Collectors;
+import org.apache.iceberg.Schema;
+import org.apache.iceberg.SortOrder;
+import org.apache.iceberg.io.CloseableIterable;
+import org.apache.iceberg.io.CloseableIterator;
+import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
+import org.apache.iceberg.util.SortedMerge;
+import org.apache.spark.sql.catalyst.InternalRow;
+import org.apache.spark.sql.connector.read.PartitionReader;
+import org.apache.spark.sql.types.StructType;
+
+/**
+ * A {@link PartitionReader} that performs a k-way merge of multiple sorted readers.
+ *
+ *
This reader takes multiple {@link PartitionReader}s (one per file), each producing sorted data
+ * according to the same {@link SortOrder}, and merges them into a single sorted stream using
+ * Iceberg's {@link SortedMerge} utility.
+ *
+ *
The merge is performed using a priority queue (heap) to efficiently select the next row from
+ * among all readers, maintaining the sort order with O(log k) comparisons per row, where k is the
+ * number of files being merged.
+ *
+ * @param the type of InternalRow being read
+ */
+class MergingPartitionReader implements PartitionReader {
+ private final List> readers;
+ private final CloseableIterator mergedIterator;
+ private T current = null;
+ private boolean closed = false;
+
+ MergingPartitionReader(
+ List> readers,
+ SortOrder sortOrder,
+ StructType sparkSchema,
+ Schema icebergSchema) {
+ Preconditions.checkNotNull(readers, "Readers cannot be null");
+ Preconditions.checkArgument(!readers.isEmpty(), "Readers cannot be empty");
+ Preconditions.checkNotNull(sortOrder, "Sort order cannot be null");
+ Preconditions.checkArgument(sortOrder.isSorted(), "Sort order must be sorted");
+
+ this.readers = readers;
+
+ Comparator comparator =
+ (Comparator) new InternalRowComparator(sortOrder, sparkSchema, icebergSchema);
+
+ List> iterables =
+ readers.stream().map(this::readerToIterable).collect(Collectors.toList());
+
+ SortedMerge sortedMerge = new SortedMerge<>(comparator, iterables);
+ this.mergedIterator = sortedMerge.iterator();
+ }
+
+ /** Converts a PartitionReader to a CloseableIterable for use with SortedMerge. */
+ private CloseableIterable readerToIterable(PartitionReader reader) {
+ return new CloseableIterable() {
+ @Override
+ public CloseableIterator iterator() {
+ return new CloseableIterator() {
+ private boolean advanced = false;
+ private boolean hasNext = false;
+
+ @Override
+ public boolean hasNext() {
+ if (!advanced) {
+ try {
+ hasNext = reader.next();
+ advanced = true;
+ } catch (IOException e) {
+ throw new RuntimeException("Failed to advance reader", e);
+ }
+ }
+ return hasNext;
+ }
+
+ @Override
+ public T next() {
+ if (!advanced) {
+ hasNext();
+ }
+ advanced = false;
+ // Spark readers reuse InternalRow objects for performance (see
+ // SparkParquetReaders.java:547)
+ // Return a copy of the row to avoid corruption.
+ return (T) reader.get().copy();
+ }
+
+ @Override
+ public void close() throws IOException {
+ reader.close();
+ }
+ };
+ }
+
+ @Override
+ public void close() throws IOException {
+ reader.close();
+ }
+ };
+ }
+
+ @Override
+ public boolean next() throws IOException {
+ if (mergedIterator.hasNext()) {
+ this.current = mergedIterator.next();
+ return true;
+ }
+ return false;
+ }
+
+ @Override
+ public T get() {
+ return current;
+ }
+
+ @Override
+ public void close() throws IOException {
+ if (closed) {
+ return;
+ }
+
+ try {
+ mergedIterator.close();
+ } finally {
+ closed = true;
+ }
+ }
+}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/MergingSortedRowDataReader.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/MergingSortedRowDataReader.java
new file mode 100644
index 000000000000..180857b0f231
--- /dev/null
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/MergingSortedRowDataReader.java
@@ -0,0 +1,139 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.iceberg.spark.source;
+
+import java.io.IOException;
+import java.util.Arrays;
+import java.util.List;
+import java.util.stream.Collectors;
+import org.apache.iceberg.BaseScanTaskGroup;
+import org.apache.iceberg.FileScanTask;
+import org.apache.iceberg.ScanTaskGroup;
+import org.apache.iceberg.Schema;
+import org.apache.iceberg.SortOrder;
+import org.apache.iceberg.Table;
+import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
+import org.apache.iceberg.spark.SparkSchemaUtil;
+import org.apache.iceberg.util.SnapshotUtil;
+import org.apache.spark.sql.catalyst.InternalRow;
+import org.apache.spark.sql.connector.metric.CustomTaskMetric;
+import org.apache.spark.sql.connector.read.PartitionReader;
+import org.apache.spark.sql.types.StructType;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * A {@link PartitionReader} that reads multiple sorted files and merges them into a single sorted
+ * stream.
+ *
+ *
This reader is used when {@code preserve-data-ordering} is enabled and the task group contains
+ * multiple files that all have the same sort order. It creates one {@link RowDataReader} per file
+ * and uses {@link MergingPartitionReader} to perform a k-way merge.
+ */
+class MergingSortedRowDataReader implements PartitionReader {
+ private static final Logger LOG = LoggerFactory.getLogger(MergingSortedRowDataReader.class);
+
+ private final MergingPartitionReader mergingReader;
+ private final List fileReaders;
+
+ MergingSortedRowDataReader(SparkInputPartition partition, int reportableSortOrderId) {
+ Table table = partition.table();
+ ScanTaskGroup taskGroup = partition.taskGroup();
+ Schema tableSchema = SnapshotUtil.schemaFor(table, partition.branch());
+ Schema expectedSchema = partition.expectedSchema();
+
+ Preconditions.checkArgument(
+ reportableSortOrderId > 0, "Invalid sort order ID: %s", reportableSortOrderId);
+ Preconditions.checkArgument(
+ taskGroup.tasks().size() > 1,
+ "Merging reader requires multiple files, got %s",
+ taskGroup.tasks().size());
+
+ LOG.info(
+ "Creating merging reader for {} files with sort order ID {} in table {}",
+ taskGroup.tasks().size(),
+ reportableSortOrderId,
+ table.name());
+
+ SortOrder sortOrder = table.sortOrders().get(reportableSortOrderId);
+ Preconditions.checkNotNull(
+ sortOrder,
+ "Cannot find sort order with ID %s in table %s",
+ reportableSortOrderId,
+ table.name());
+
+ this.fileReaders =
+ taskGroup.tasks().stream()
+ .map(
+ task -> {
+ ScanTaskGroup singleTaskGroup =
+ new BaseScanTaskGroup<>(java.util.Collections.singletonList(task));
+
+ return new RowDataReader(
+ table,
+ singleTaskGroup,
+ tableSchema,
+ expectedSchema,
+ partition.isCaseSensitive(),
+ partition.cacheDeleteFilesOnExecutors());
+ })
+ .collect(Collectors.toList());
+
+ List> readers =
+ fileReaders.stream()
+ .map(reader -> (PartitionReader) reader)
+ .collect(Collectors.toList());
+
+ StructType sparkSchema = SparkSchemaUtil.convert(expectedSchema);
+ this.mergingReader =
+ new MergingPartitionReader<>(readers, sortOrder, sparkSchema, expectedSchema);
+ }
+
+ @Override
+ public boolean next() throws IOException {
+ return mergingReader.next();
+ }
+
+ @Override
+ public InternalRow get() {
+ return mergingReader.get();
+ }
+
+ @Override
+ public void close() throws IOException {
+ mergingReader.close();
+ }
+
+ public CustomTaskMetric[] currentMetricsValues() {
+ long totalSplits = fileReaders.size();
+
+ long totalDeletes =
+ fileReaders.stream()
+ .flatMap(reader -> Arrays.stream(reader.currentMetricsValues()))
+ .filter(
+ metric -> metric instanceof org.apache.iceberg.spark.source.metrics.TaskNumDeletes)
+ .mapToLong(CustomTaskMetric::value)
+ .sum();
+
+ return new CustomTaskMetric[] {
+ new org.apache.iceberg.spark.source.metrics.TaskNumSplits(totalSplits),
+ new org.apache.iceberg.spark.source.metrics.TaskNumDeletes(totalDeletes)
+ };
+ }
+}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkBatch.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkBatch.java
index 0626d0b43985..eb812338e149 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkBatch.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkBatch.java
@@ -91,16 +91,24 @@ public InputPartition[] planInputPartitions() {
InputPartition[] partitions = new InputPartition[taskGroups.size()];
for (int index = 0; index < taskGroups.size(); index++) {
+ ScanTaskGroup> taskGroup = taskGroups.get(index);
+
+ Integer reportableSortOrderId = null;
+ if (readConf.preserveDataOrdering()) {
+ reportableSortOrderId = table.sortOrder().orderId();
+ }
+
partitions[index] =
new SparkInputPartition(
groupingKeyType,
- taskGroups.get(index),
+ taskGroup,
tableBroadcast,
branch,
expectedSchemaString,
caseSensitive,
locations != null ? locations[index] : SparkPlanningUtil.NO_LOCATION_PREFERENCE,
- cacheDeleteFilesOnExecutors);
+ cacheDeleteFilesOnExecutors,
+ reportableSortOrderId);
}
return partitions;
@@ -160,6 +168,12 @@ private boolean useParquetBatchReads() {
private boolean supportsParquetBatchReads(ScanTask task) {
if (task instanceof ScanTaskGroup) {
ScanTaskGroup> taskGroup = (ScanTaskGroup>) task;
+
+ // Vectorized readers cannot merge sorted data from multiple files
+ if (readConf.preserveDataOrdering() && taskGroup.tasks().size() > 1) {
+ return false;
+ }
+
return taskGroup.tasks().stream().allMatch(this::supportsParquetBatchReads);
} else if (task.isFileScanTask() && !task.isDataTask()) {
@@ -200,6 +214,12 @@ private boolean useOrcBatchReads() {
private boolean supportsOrcBatchReads(ScanTask task) {
if (task instanceof ScanTaskGroup) {
ScanTaskGroup> taskGroup = (ScanTaskGroup>) task;
+
+ // Vectorized readers cannot merge sorted data from multiple files
+ if (readConf.preserveDataOrdering() && taskGroup.tasks().size() > 1) {
+ return false;
+ }
+
return taskGroup.tasks().stream().allMatch(this::supportsOrcBatchReads);
} else if (task.isFileScanTask() && !task.isDataTask()) {
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkInputPartition.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkInputPartition.java
index 99b1d78a86b0..e2e4037aaac0 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkInputPartition.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkInputPartition.java
@@ -39,6 +39,7 @@ class SparkInputPartition implements InputPartition, HasPartitionKey, Serializab
private final boolean caseSensitive;
private final transient String[] preferredLocations;
private final boolean cacheDeleteFilesOnExecutors;
+ private final Integer reportableSortOrderId;
private transient Schema expectedSchema = null;
@@ -50,7 +51,8 @@ class SparkInputPartition implements InputPartition, HasPartitionKey, Serializab
String expectedSchemaString,
boolean caseSensitive,
String[] preferredLocations,
- boolean cacheDeleteFilesOnExecutors) {
+ boolean cacheDeleteFilesOnExecutors,
+ Integer reportableSortOrderId) {
this.groupingKeyType = groupingKeyType;
this.taskGroup = taskGroup;
this.tableBroadcast = tableBroadcast;
@@ -59,6 +61,7 @@ class SparkInputPartition implements InputPartition, HasPartitionKey, Serializab
this.caseSensitive = caseSensitive;
this.preferredLocations = preferredLocations;
this.cacheDeleteFilesOnExecutors = cacheDeleteFilesOnExecutors;
+ this.reportableSortOrderId = reportableSortOrderId;
}
@Override
@@ -103,4 +106,8 @@ public Schema expectedSchema() {
return expectedSchema;
}
+
+ public Integer reportableSortOrderId() {
+ return reportableSortOrderId;
+ }
}
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkMicroBatchStream.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkMicroBatchStream.java
index 60dd1f318ca5..b5d7e1b2473e 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkMicroBatchStream.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkMicroBatchStream.java
@@ -172,7 +172,8 @@ public InputPartition[] planInputPartitions(Offset start, Offset end) {
expectedSchema,
caseSensitive,
locations != null ? locations[index] : SparkPlanningUtil.NO_LOCATION_PREFERENCE,
- cacheDeleteFilesOnExecutors);
+ cacheDeleteFilesOnExecutors,
+ null);
}
return partitions;
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPartitioningAwareScan.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPartitioningAwareScan.java
index c9726518ee4e..9ac86d3c0730 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPartitioningAwareScan.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPartitioningAwareScan.java
@@ -48,7 +48,9 @@
import org.apache.iceberg.util.StructLikeSet;
import org.apache.iceberg.util.TableScanUtil;
import org.apache.spark.sql.SparkSession;
+import org.apache.spark.sql.connector.expressions.SortOrder;
import org.apache.spark.sql.connector.expressions.Transform;
+import org.apache.spark.sql.connector.read.SupportsReportOrdering;
import org.apache.spark.sql.connector.read.SupportsReportPartitioning;
import org.apache.spark.sql.connector.read.partitioning.KeyGroupedPartitioning;
import org.apache.spark.sql.connector.read.partitioning.Partitioning;
@@ -57,12 +59,13 @@
import org.slf4j.LoggerFactory;
abstract class SparkPartitioningAwareScan extends SparkScan
- implements SupportsReportPartitioning {
+ implements SupportsReportPartitioning, SupportsReportOrdering {
private static final Logger LOG = LoggerFactory.getLogger(SparkPartitioningAwareScan.class);
private final Scan, ? extends ScanTask, ? extends ScanTaskGroup>> scan;
private final boolean preserveDataGrouping;
+ private final boolean preserveDataOrdering;
private Set specs = null; // lazy cache of scanned specs
private List tasks = null; // lazy cache of uncombined tasks
@@ -82,6 +85,7 @@ abstract class SparkPartitioningAwareScan extends S
this.scan = scan;
this.preserveDataGrouping = readConf.preserveDataGrouping();
+ this.preserveDataOrdering = readConf.preserveDataOrdering();
if (scan == null) {
this.specs = Collections.emptySet();
@@ -114,6 +118,57 @@ public Partitioning outputPartitioning() {
}
}
+ @Override
+ public SortOrder[] outputOrdering() {
+ if (!preserveDataOrdering) {
+ return new SortOrder[0];
+ }
+
+ if (groupingKeyType().fields().isEmpty()) {
+ LOG.info("Not reporting ordering for unpartitioned table {}", table().name());
+ return new SortOrder[0];
+ }
+
+ org.apache.iceberg.SortOrder currentSortOrder = table().sortOrder();
+ if (currentSortOrder.isUnsorted()) {
+ return new SortOrder[0];
+ }
+
+ if (!allFilesHaveSortOrder(currentSortOrder.orderId())) {
+ LOG.info(
+ "Not all files have current table sort order {}, not reporting ordering",
+ currentSortOrder.orderId());
+ return new SortOrder[0];
+ }
+
+ SortOrder[] ordering = Spark3Util.toOrdering(currentSortOrder);
+ LOG.info(
+ "Reporting sort order {} for table {}: {}",
+ currentSortOrder.orderId(),
+ table().name(),
+ ordering);
+
+ return ordering;
+ }
+
+ private boolean allFilesHaveSortOrder(int expectedSortOrderId) {
+ for (ScanTaskGroup taskGroup : taskGroups()) {
+ for (T task : taskGroup.tasks()) {
+ if (!(task instanceof org.apache.iceberg.FileScanTask)) {
+ continue;
+ }
+
+ org.apache.iceberg.FileScanTask fileTask = (org.apache.iceberg.FileScanTask) task;
+ Integer fileSortOrderId = fileTask.file().sortOrderId();
+
+ if (fileSortOrderId == null || fileSortOrderId != expectedSortOrderId) {
+ return false;
+ }
+ }
+ }
+ return true;
+ }
+
@Override
protected StructType groupingKeyType() {
if (groupingKeyType == null) {
diff --git a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPositionDeltaWrite.java b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPositionDeltaWrite.java
index d072397dc6a3..e0c842e9a6d7 100644
--- a/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPositionDeltaWrite.java
+++ b/spark/v4.1/spark/src/main/java/org/apache/iceberg/spark/source/SparkPositionDeltaWrite.java
@@ -182,7 +182,8 @@ public DeltaWriterFactory createBatchWriterFactory(PhysicalWriteInfo info) {
broadcastRewritableDeletes(),
command,
context,
- writeProperties);
+ writeProperties,
+ writeRequirements.icebergOrdering());
}
private Broadcast