This is an automated email from the ASF dual-hosted git repository.
lzljs3620320 pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/paimon.git
The following commit(s) were added to refs/heads/master by this push:
new 820e384aa8 [core] Minor refactor append table compaction (#7244)
820e384aa8 is described below
commit 820e384aa8012176ff24cb7bbb7555e5121ff479
Author: yuzelin <[email protected]>
AuthorDate: Tue Feb 10 09:19:13 2026 +0800
[core] Minor refactor append table compaction (#7244)
---
.../apache/paimon/append/AppendCompactTask.java | 6 +++---
.../dataevolution/DataEvolutionCompactTask.java | 23 +++-------------------
.../paimon/flink/sink/AppendTableCompactSink.java | 2 --
.../flink/sink/DataEvolutionTableCompactSink.java | 3 ---
4 files changed, 6 insertions(+), 28 deletions(-)
diff --git
a/paimon-core/src/main/java/org/apache/paimon/append/AppendCompactTask.java
b/paimon-core/src/main/java/org/apache/paimon/append/AppendCompactTask.java
index 42c4548842..77d52b7c7d 100644
--- a/paimon-core/src/main/java/org/apache/paimon/append/AppendCompactTask.java
+++ b/paimon-core/src/main/java/org/apache/paimon/append/AppendCompactTask.java
@@ -43,9 +43,9 @@ import static
org.apache.paimon.table.BucketMode.UNAWARE_BUCKET;
/** Compaction task generated by {@link AppendCompactCoordinator}. */
public class AppendCompactTask {
- private final BinaryRow partition;
- private final List<DataFileMeta> compactBefore;
- private final List<DataFileMeta> compactAfter;
+ protected final BinaryRow partition;
+ protected final List<DataFileMeta> compactBefore;
+ protected final List<DataFileMeta> compactAfter;
public AppendCompactTask(BinaryRow partition, List<DataFileMeta> files) {
Preconditions.checkArgument(files != null);
diff --git
a/paimon-core/src/main/java/org/apache/paimon/append/dataevolution/DataEvolutionCompactTask.java
b/paimon-core/src/main/java/org/apache/paimon/append/dataevolution/DataEvolutionCompactTask.java
index f1f48ac56d..abbc03f17b 100644
---
a/paimon-core/src/main/java/org/apache/paimon/append/dataevolution/DataEvolutionCompactTask.java
+++
b/paimon-core/src/main/java/org/apache/paimon/append/dataevolution/DataEvolutionCompactTask.java
@@ -20,6 +20,7 @@ package org.apache.paimon.append.dataevolution;
import org.apache.paimon.AppendOnlyFileStore;
import org.apache.paimon.CoreOptions;
+import org.apache.paimon.append.AppendCompactTask;
import org.apache.paimon.data.BinaryRow;
import org.apache.paimon.data.InternalRow;
import org.apache.paimon.io.CompactIncrement;
@@ -36,7 +37,6 @@ import org.apache.paimon.types.RowType;
import org.apache.paimon.utils.FileStorePathFactory;
import org.apache.paimon.utils.RecordWriter;
-import java.util.ArrayList;
import java.util.Collections;
import java.util.List;
import java.util.Map;
@@ -46,36 +46,19 @@ import java.util.stream.Collectors;
import static org.apache.paimon.utils.Preconditions.checkArgument;
/** Data evolution table compaction task. */
-public class DataEvolutionCompactTask {
+public class DataEvolutionCompactTask extends AppendCompactTask {
private static final Map<String, String> DYNAMIC_WRITE_OPTIONS =
Collections.singletonMap(CoreOptions.TARGET_FILE_SIZE.key(),
"99999 G");
- private final BinaryRow partition;
- private final List<DataFileMeta> compactBefore;
- private final List<DataFileMeta> compactAfter;
private final boolean blobTask;
public DataEvolutionCompactTask(
BinaryRow partition, List<DataFileMeta> files, boolean blobTask) {
- this.partition = partition;
- this.compactBefore = new ArrayList<>(files);
- this.compactAfter = new ArrayList<>();
+ super(partition, files);
this.blobTask = blobTask;
}
- public BinaryRow partition() {
- return partition;
- }
-
- public List<DataFileMeta> compactBefore() {
- return compactBefore;
- }
-
- public List<DataFileMeta> compactAfter() {
- return compactAfter;
- }
-
public boolean isBlobTask() {
return blobTask;
}
diff --git
a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/AppendTableCompactSink.java
b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/AppendTableCompactSink.java
index 6b29577c8e..9648d5f946 100644
---
a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/AppendTableCompactSink.java
+++
b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/AppendTableCompactSink.java
@@ -29,12 +29,10 @@ import
org.apache.flink.streaming.api.operators.OneInputStreamOperatorFactory;
/** Compaction Sink for unaware-bucket table. */
public class AppendTableCompactSink extends FlinkSink<AppendCompactTask> {
- private final FileStoreTable table;
private final boolean isStreaming;
public AppendTableCompactSink(FileStoreTable table, boolean isStreaming) {
super(table, true);
- this.table = table;
this.isStreaming = isStreaming;
}
diff --git
a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/DataEvolutionTableCompactSink.java
b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/DataEvolutionTableCompactSink.java
index eb021be8dd..b66d0bd67a 100644
---
a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/DataEvolutionTableCompactSink.java
+++
b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/sink/DataEvolutionTableCompactSink.java
@@ -31,11 +31,8 @@ import static
org.apache.paimon.utils.Preconditions.checkArgument;
/** Compaction Sink for data-evolution table. */
public class DataEvolutionTableCompactSink extends
FlinkSink<DataEvolutionCompactTask> {
- private final FileStoreTable table;
-
public DataEvolutionTableCompactSink(FileStoreTable table) {
super(table, true);
- this.table = table;
}
public static DataStreamSink<?> sink(