This is an automated email from the ASF dual-hosted git repository.

czweng pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-paimon.git


The following commit(s) were added to refs/heads/master by this push:
     new f44345bb7 Revert "[core] Introduce file.compression (#914)"
f44345bb7 is described below

commit f44345bb7a2d3cbec4515b5ec315cad27594be5e
Author: tsreaper <[email protected]>
AuthorDate: Mon Apr 17 19:31:48 2023 +0800

    Revert "[core] Introduce file.compression (#914)"
    
    This reverts commit 9965bf00
---
 .../apache/paimon/format/FormatWriterFactory.java  |  8 ++++++-
 .../paimon/format/FileStatsExtractorTestBase.java  |  2 +-
 .../main/java/org/apache/paimon/CoreOptions.java   | 12 ----------
 .../org/apache/paimon/append/AppendOnlyWriter.java |  8 ++-----
 .../paimon/io/KeyValueFileWriterFactory.java       | 27 +++-------------------
 .../org/apache/paimon/io/RowDataFileWriter.java    | 12 ++--------
 .../apache/paimon/io/RowDataRollingFileWriter.java |  6 ++---
 .../org/apache/paimon/manifest/ManifestFile.java   | 11 +++------
 .../org/apache/paimon/manifest/ManifestList.java   |  4 +---
 .../paimon/operation/AppendOnlyFileStoreWrite.java |  9 ++------
 .../paimon/operation/KeyValueFileStoreWrite.java   | 12 ++--------
 .../java/org/apache/paimon/FileFormatTest.java     |  9 ++------
 .../apache/paimon/append/AppendOnlyWriterTest.java |  3 +--
 .../apache/paimon/format/FileFormatSuffixTest.java |  4 +---
 .../apache/paimon/format/FlushingFileFormat.java   |  7 +-----
 .../apache/paimon/io/RollingFileWriterTest.java    |  3 +--
 .../apache/paimon/format/BulkFileFormatTest.java   |  2 +-
 .../paimon/format/orc/OrcWriterFactoryTest.java    |  4 ++--
 .../format/parquet/ParquetReadWriteTest.java       |  2 +-
 19 files changed, 35 insertions(+), 110 deletions(-)

diff --git 
a/paimon-common/src/main/java/org/apache/paimon/format/FormatWriterFactory.java 
b/paimon-common/src/main/java/org/apache/paimon/format/FormatWriterFactory.java
index 517cc00cb..0ac921ee6 100644
--- 
a/paimon-common/src/main/java/org/apache/paimon/format/FormatWriterFactory.java
+++ 
b/paimon-common/src/main/java/org/apache/paimon/format/FormatWriterFactory.java
@@ -20,6 +20,8 @@ package org.apache.paimon.format;
 
 import org.apache.paimon.fs.PositionOutputStream;
 
+import javax.annotation.Nullable;
+
 import java.io.IOException;
 
 /** A factory to create {@link FormatWriter} for file. */
@@ -33,5 +35,9 @@ public interface FormatWriterFactory {
      * @throws IOException Thrown if the writer cannot be opened, or if the 
output stream throws an
      *     exception.
      */
-    FormatWriter create(PositionOutputStream out, String compression) throws 
IOException;
+    FormatWriter create(PositionOutputStream out, @Nullable String 
compression) throws IOException;
+
+    default FormatWriter create(PositionOutputStream out) throws IOException {
+        return create(out, null);
+    }
 }
diff --git 
a/paimon-common/src/test/java/org/apache/paimon/format/FileStatsExtractorTestBase.java
 
b/paimon-common/src/test/java/org/apache/paimon/format/FileStatsExtractorTestBase.java
index 6839670de..727cc1b69 100644
--- 
a/paimon-common/src/test/java/org/apache/paimon/format/FileStatsExtractorTestBase.java
+++ 
b/paimon-common/src/test/java/org/apache/paimon/format/FileStatsExtractorTestBase.java
@@ -74,7 +74,7 @@ public abstract class FileStatsExtractorTestBase {
         FormatWriterFactory writerFactory = 
format.createWriterFactory(rowType);
         Path path = new Path(tempDir.toString() + "/test");
         PositionOutputStream out = new LocalFileIO().newOutputStream(path, 
false);
-        FormatWriter writer = writerFactory.create(out, "LZ4");
+        FormatWriter writer = writerFactory.create(out);
 
         List<GenericRow> data = createData(rowType);
         for (GenericRow row : data) {
diff --git a/paimon-core/src/main/java/org/apache/paimon/CoreOptions.java 
b/paimon-core/src/main/java/org/apache/paimon/CoreOptions.java
index b1554e5d2..c27d8b26a 100644
--- a/paimon-core/src/main/java/org/apache/paimon/CoreOptions.java
+++ b/paimon-core/src/main/java/org/apache/paimon/CoreOptions.java
@@ -113,14 +113,6 @@ public class CoreOptions implements Serializable {
                                     + "could be NONE, ZLIB, SNAPPY, LZO, LZ4, 
for parquet file format, the compression value could be "
                                     + "UNCOMPRESSED, SNAPPY, GZIP, LZO, 
BROTLI, LZ4, ZSTD.");
 
-    public static final ConfigOption<String> FILE_COMPRESSION =
-            key("file.compression")
-                    .stringType()
-                    .defaultValue("LZ4")
-                    .withDescription(
-                            "Default file compression format, can be 
overridden by "
-                                    + FILE_COMPRESSION_PER_LEVEL.key());
-
     public static final ConfigOption<FileFormatType> MANIFEST_FORMAT =
             key("manifest.format")
                     .enumType(FileFormatType.class)
@@ -681,10 +673,6 @@ public class CoreOptions implements Serializable {
                 .collect(Collectors.toMap(e -> Integer.valueOf(e.getKey()), 
Map.Entry::getValue));
     }
 
-    public String fileCompression() {
-        return options.get(FILE_COMPRESSION);
-    }
-
     public int snapshotNumRetainMin() {
         return options.get(SNAPSHOT_NUM_RETAINED_MIN);
     }
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/append/AppendOnlyWriter.java 
b/paimon-core/src/main/java/org/apache/paimon/append/AppendOnlyWriter.java
index b40c038c5..9c2601c99 100644
--- a/paimon-core/src/main/java/org/apache/paimon/append/AppendOnlyWriter.java
+++ b/paimon-core/src/main/java/org/apache/paimon/append/AppendOnlyWriter.java
@@ -61,7 +61,6 @@ public class AppendOnlyWriter implements 
RecordWriter<InternalRow> {
     private final List<DataFileMeta> compactBefore;
     private final List<DataFileMeta> compactAfter;
     private final LongCounter seqNumCounter;
-    private final String fileCompression;
 
     private RowDataRollingFileWriter writer;
 
@@ -75,8 +74,7 @@ public class AppendOnlyWriter implements 
RecordWriter<InternalRow> {
             CompactManager compactManager,
             boolean forceCompact,
             DataFilePathFactory pathFactory,
-            @Nullable CommitIncrement increment,
-            String fileCompression) {
+            @Nullable CommitIncrement increment) {
         this.fileIO = fileIO;
         this.schemaId = schemaId;
         this.fileFormat = fileFormat;
@@ -89,7 +87,6 @@ public class AppendOnlyWriter implements 
RecordWriter<InternalRow> {
         this.compactBefore = new ArrayList<>();
         this.compactAfter = new ArrayList<>();
         this.seqNumCounter = new LongCounter(maxSequenceNumber + 1);
-        this.fileCompression = fileCompression;
 
         this.writer = createRollingRowWriter();
 
@@ -172,8 +169,7 @@ public class AppendOnlyWriter implements 
RecordWriter<InternalRow> {
                 targetFileSize,
                 writeSchema,
                 pathFactory,
-                seqNumCounter,
-                fileCompression);
+                seqNumCounter);
     }
 
     private void trySyncLatestCompaction(boolean blocking)
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/io/KeyValueFileWriterFactory.java 
b/paimon-core/src/main/java/org/apache/paimon/io/KeyValueFileWriterFactory.java
index 2ea50eede..c0bd2f0f9 100644
--- 
a/paimon-core/src/main/java/org/apache/paimon/io/KeyValueFileWriterFactory.java
+++ 
b/paimon-core/src/main/java/org/apache/paimon/io/KeyValueFileWriterFactory.java
@@ -18,7 +18,6 @@
 
 package org.apache.paimon.io;
 
-import org.apache.paimon.CoreOptions;
 import org.apache.paimon.KeyValue;
 import org.apache.paimon.KeyValueSerializer;
 import org.apache.paimon.annotation.VisibleForTesting;
@@ -47,7 +46,6 @@ public class KeyValueFileWriterFactory {
     private final DataFilePathFactory pathFactory;
     private final long suggestedFileSize;
     private final Map<Integer, String> levelCompressions;
-    private final String fileCompression;
 
     private KeyValueFileWriterFactory(
             FileIO fileIO,
@@ -58,8 +56,7 @@ public class KeyValueFileWriterFactory {
             @Nullable FileStatsExtractor fileStatsExtractor,
             DataFilePathFactory pathFactory,
             long suggestedFileSize,
-            Map<Integer, String> levelCompressions,
-            String fileCompression) {
+            Map<Integer, String> levelCompressions) {
         this.fileIO = fileIO;
         this.schemaId = schemaId;
         this.keyType = keyType;
@@ -69,7 +66,6 @@ public class KeyValueFileWriterFactory {
         this.pathFactory = pathFactory;
         this.suggestedFileSize = suggestedFileSize;
         this.levelCompressions = levelCompressions;
-        this.fileCompression = fileCompression;
     }
 
     public RowType keyType() {
@@ -92,11 +88,7 @@ public class KeyValueFileWriterFactory {
     }
 
     private String getCompression(int level) {
-        if (null == levelCompressions) {
-            return fileCompression;
-        } else {
-            return levelCompressions.getOrDefault(level, fileCompression);
-        }
+        return null == levelCompressions ? null : levelCompressions.get(level);
     }
 
     public RollingFileWriter<KeyValue, DataFileMeta> 
createRollingChangelogFileWriter(int level) {
@@ -168,18 +160,6 @@ public class KeyValueFileWriterFactory {
 
         public KeyValueFileWriterFactory build(
                 BinaryRow partition, int bucket, Map<Integer, String> 
levelCompressions) {
-            return build(
-                    partition,
-                    bucket,
-                    levelCompressions,
-                    CoreOptions.FILE_COMPRESSION.defaultValue());
-        }
-
-        public KeyValueFileWriterFactory build(
-                BinaryRow partition,
-                int bucket,
-                Map<Integer, String> levelCompressions,
-                String fileCompression) {
             RowType recordType = KeyValue.schema(keyType, valueType);
             return new KeyValueFileWriterFactory(
                     fileIO,
@@ -190,8 +170,7 @@ public class KeyValueFileWriterFactory {
                     fileFormat.createStatsExtractor(recordType).orElse(null),
                     pathFactory.createDataFilePathFactory(partition, bucket),
                     suggestedFileSize,
-                    levelCompressions,
-                    fileCompression);
+                    levelCompressions);
         }
     }
 }
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/io/RowDataFileWriter.java 
b/paimon-core/src/main/java/org/apache/paimon/io/RowDataFileWriter.java
index 471e044b9..22afd08a6 100644
--- a/paimon-core/src/main/java/org/apache/paimon/io/RowDataFileWriter.java
+++ b/paimon-core/src/main/java/org/apache/paimon/io/RowDataFileWriter.java
@@ -51,16 +51,8 @@ public class RowDataFileWriter extends 
StatsCollectingSingleFileWriter<InternalR
             RowType writeSchema,
             @Nullable FileStatsExtractor fileStatsExtractor,
             long schemaId,
-            LongCounter seqNumCounter,
-            String fileCompression) {
-        super(
-                fileIO,
-                factory,
-                path,
-                Function.identity(),
-                writeSchema,
-                fileStatsExtractor,
-                fileCompression);
+            LongCounter seqNumCounter) {
+        super(fileIO, factory, path, Function.identity(), writeSchema, 
fileStatsExtractor, null);
         this.schemaId = schemaId;
         this.seqNumCounter = seqNumCounter;
         this.statsArraySerializer = new FieldStatsArraySerializer(writeSchema);
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/io/RowDataRollingFileWriter.java 
b/paimon-core/src/main/java/org/apache/paimon/io/RowDataRollingFileWriter.java
index 7b17ed167..c5e549ac4 100644
--- 
a/paimon-core/src/main/java/org/apache/paimon/io/RowDataRollingFileWriter.java
+++ 
b/paimon-core/src/main/java/org/apache/paimon/io/RowDataRollingFileWriter.java
@@ -35,8 +35,7 @@ public class RowDataRollingFileWriter extends 
RollingFileWriter<InternalRow, Dat
             long targetFileSize,
             RowType writeSchema,
             DataFilePathFactory pathFactory,
-            LongCounter seqNumCounter,
-            String fileCompression) {
+            LongCounter seqNumCounter) {
         super(
                 () ->
                         new RowDataFileWriter(
@@ -46,8 +45,7 @@ public class RowDataRollingFileWriter extends 
RollingFileWriter<InternalRow, Dat
                                 writeSchema,
                                 
fileFormat.createStatsExtractor(writeSchema).orElse(null),
                                 schemaId,
-                                seqNumCounter,
-                                fileCompression),
+                                seqNumCounter),
                 targetFileSize);
     }
 }
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestFile.java 
b/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestFile.java
index fbaec0dae..e5ea62794 100644
--- a/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestFile.java
+++ b/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestFile.java
@@ -18,7 +18,6 @@
 
 package org.apache.paimon.manifest;
 
-import org.apache.paimon.CoreOptions;
 import org.apache.paimon.annotation.VisibleForTesting;
 import org.apache.paimon.format.FieldStatsCollector;
 import org.apache.paimon.format.FileFormat;
@@ -83,11 +82,7 @@ public class ManifestFile extends ObjectsFile<ManifestEntry> 
{
     public List<ManifestFileMeta> write(List<ManifestEntry> entries) {
         RollingFileWriter<ManifestEntry, ManifestFileMeta> writer =
                 new RollingFileWriter<>(
-                        () ->
-                                new ManifestEntryWriter(
-                                        writerFactory,
-                                        pathFactory.newPath(),
-                                        
CoreOptions.FILE_COMPRESSION.defaultValue()),
+                        () -> new ManifestEntryWriter(writerFactory, 
pathFactory.newPath()),
                         suggestedFileSize);
         try {
             writer.write(entries);
@@ -107,8 +102,8 @@ public class ManifestFile extends 
ObjectsFile<ManifestEntry> {
         private long numDeletedFiles = 0;
         private long schemaId = Long.MIN_VALUE;
 
-        ManifestEntryWriter(FormatWriterFactory factory, Path path, String 
fileCompression) {
-            super(ManifestFile.this.fileIO, factory, path, serializer::toRow, 
fileCompression);
+        ManifestEntryWriter(FormatWriterFactory factory, Path path) {
+            super(ManifestFile.this.fileIO, factory, path, serializer::toRow, 
null);
 
             this.partitionStatsCollector = new 
FieldStatsCollector(partitionType);
             this.partitionStatsSerializer = new 
FieldStatsArraySerializer(partitionType);
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestList.java 
b/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestList.java
index c11536713..86529e31c 100644
--- a/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestList.java
+++ b/paimon-core/src/main/java/org/apache/paimon/manifest/ManifestList.java
@@ -18,7 +18,6 @@
 
 package org.apache.paimon.manifest;
 
-import org.apache.paimon.CoreOptions;
 import org.apache.paimon.format.FileFormat;
 import org.apache.paimon.format.FormatReaderFactory;
 import org.apache.paimon.format.FormatWriter;
@@ -65,8 +64,7 @@ public class ManifestList extends 
ObjectsFile<ManifestFileMeta> {
         Path path = pathFactory.newPath();
         try {
             try (PositionOutputStream out = fileIO.newOutputStream(path, 
false)) {
-                FormatWriter writer =
-                        writerFactory.create(out, 
CoreOptions.FILE_COMPRESSION.defaultValue());
+                FormatWriter writer = writerFactory.create(out);
                 try {
                     for (ManifestFileMeta record : metas) {
                         writer.addElement(serializer.toRow(record));
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/operation/AppendOnlyFileStoreWrite.java
 
b/paimon-core/src/main/java/org/apache/paimon/operation/AppendOnlyFileStoreWrite.java
index f5f071d23..52f0fc2e9 100644
--- 
a/paimon-core/src/main/java/org/apache/paimon/operation/AppendOnlyFileStoreWrite.java
+++ 
b/paimon-core/src/main/java/org/apache/paimon/operation/AppendOnlyFileStoreWrite.java
@@ -64,7 +64,6 @@ public class AppendOnlyFileStoreWrite extends 
AbstractFileStoreWrite<InternalRow
     private final boolean commitForceCompact;
     private final boolean skipCompaction;
     private final boolean assertDisorder;
-    private final String fileCompression;
 
     public AppendOnlyFileStoreWrite(
             FileIO fileIO,
@@ -89,7 +88,6 @@ public class AppendOnlyFileStoreWrite extends 
AbstractFileStoreWrite<InternalRow
         this.commitForceCompact = options.commitForceCompact();
         this.skipCompaction = options.writeOnly();
         this.assertDisorder = 
options.toConfiguration().get(APPEND_ONLY_ASSERT_DISORDER);
-        this.fileCompression = options.fileCompression();
     }
 
     @Override
@@ -114,7 +112,6 @@ public class AppendOnlyFileStoreWrite extends 
AbstractFileStoreWrite<InternalRow
                                 targetFileSize,
                                 compactRewriter(partition, bucket),
                                 assertDisorder);
-
         return new AppendOnlyWriter(
                 fileIO,
                 schemaId,
@@ -125,8 +122,7 @@ public class AppendOnlyFileStoreWrite extends 
AbstractFileStoreWrite<InternalRow
                 compactManager,
                 commitForceCompact,
                 factory,
-                restoreIncrement,
-                fileCompression);
+                restoreIncrement);
     }
 
     private AppendOnlyCompactManager.CompactRewriter compactRewriter(
@@ -143,8 +139,7 @@ public class AppendOnlyFileStoreWrite extends 
AbstractFileStoreWrite<InternalRow
                             targetFileSize,
                             rowType,
                             pathFactory.createDataFilePathFactory(partition, 
bucket),
-                            new 
LongCounter(toCompact.get(0).minSequenceNumber()),
-                            fileCompression);
+                            new 
LongCounter(toCompact.get(0).minSequenceNumber()));
             rewriter.write(
                     new RecordReaderIterator<>(
                             read.createReader(
diff --git 
a/paimon-core/src/main/java/org/apache/paimon/operation/KeyValueFileStoreWrite.java
 
b/paimon-core/src/main/java/org/apache/paimon/operation/KeyValueFileStoreWrite.java
index 72af8fb57..aade98209 100644
--- 
a/paimon-core/src/main/java/org/apache/paimon/operation/KeyValueFileStoreWrite.java
+++ 
b/paimon-core/src/main/java/org/apache/paimon/operation/KeyValueFileStoreWrite.java
@@ -135,11 +135,7 @@ public class KeyValueFileStoreWrite extends 
MemoryFileStoreWrite<KeyValue> {
         }
 
         KeyValueFileWriterFactory writerFactory =
-                writerFactoryBuilder.build(
-                        partition,
-                        bucket,
-                        options.fileCompressionPerLevel(),
-                        options.fileCompression());
+                writerFactoryBuilder.build(partition, bucket, 
options.fileCompressionPerLevel());
         Comparator<InternalRow> keyComparator = keyComparatorSupplier.get();
         Levels levels = new Levels(keyComparator, restoreFiles, 
options.numLevels());
         UniversalCompaction universalCompaction =
@@ -198,11 +194,7 @@ public class KeyValueFileStoreWrite extends 
MemoryFileStoreWrite<KeyValue> {
             BinaryRow partition, int bucket, Comparator<InternalRow> 
keyComparator, Levels levels) {
         KeyValueFileReaderFactory readerFactory = 
readerFactoryBuilder.build(partition, bucket);
         KeyValueFileWriterFactory writerFactory =
-                writerFactoryBuilder.build(
-                        partition,
-                        bucket,
-                        options.fileCompressionPerLevel(),
-                        options.fileCompression());
+                writerFactoryBuilder.build(partition, bucket, 
options.fileCompressionPerLevel());
         switch (options.changelogProducer()) {
             case FULL_COMPACTION:
                 return new FullChangelogMergeTreeCompactRewriter(
diff --git a/paimon-core/src/test/java/org/apache/paimon/FileFormatTest.java 
b/paimon-core/src/test/java/org/apache/paimon/FileFormatTest.java
index 8554dc025..83a869ffb 100644
--- a/paimon-core/src/test/java/org/apache/paimon/FileFormatTest.java
+++ b/paimon-core/src/test/java/org/apache/paimon/FileFormatTest.java
@@ -59,9 +59,7 @@ public class FileFormatTest {
         expected.add(GenericRow.of(2, 22));
         expected.add(GenericRow.of(3, 33));
         PositionOutputStream out = LocalFileIO.create().newOutputStream(path, 
false);
-        FormatWriter writer =
-                avro.createWriterFactory(rowType)
-                        .create(out, 
CoreOptions.FILE_COMPRESSION.defaultValue());
+        FormatWriter writer = avro.createWriterFactory(rowType).create(out);
         for (InternalRow row : expected) {
             writer.addElement(row);
         }
@@ -85,10 +83,7 @@ public class FileFormatTest {
         Path path = new Path(tempDir.toUri().toString(), "1.avro");
         Assertions.assertThrows(
                 RuntimeException.class,
-                () ->
-                        writerFactory.create(
-                                LocalFileIO.create().newOutputStream(path, 
false),
-                                CoreOptions.FILE_COMPRESSION.defaultValue()),
+                () -> 
writerFactory.create(LocalFileIO.create().newOutputStream(path, false)),
                 "Unrecognized codec: _unsupported");
     }
 
diff --git 
a/paimon-core/src/test/java/org/apache/paimon/append/AppendOnlyWriterTest.java 
b/paimon-core/src/test/java/org/apache/paimon/append/AppendOnlyWriterTest.java
index 9c2913c7e..5ac63cb9d 100644
--- 
a/paimon-core/src/test/java/org/apache/paimon/append/AppendOnlyWriterTest.java
+++ 
b/paimon-core/src/test/java/org/apache/paimon/append/AppendOnlyWriterTest.java
@@ -316,8 +316,7 @@ public class AppendOnlyWriterTest {
                         compactManager,
                         forceCompact,
                         pathFactory,
-                        null,
-                        CoreOptions.FILE_COMPRESSION.defaultValue());
+                        null);
         return Pair.of(writer, compactManager.allFiles());
     }
 
diff --git 
a/paimon-core/src/test/java/org/apache/paimon/format/FileFormatSuffixTest.java 
b/paimon-core/src/test/java/org/apache/paimon/format/FileFormatSuffixTest.java
index 40e814844..60eb362c6 100644
--- 
a/paimon-core/src/test/java/org/apache/paimon/format/FileFormatSuffixTest.java
+++ 
b/paimon-core/src/test/java/org/apache/paimon/format/FileFormatSuffixTest.java
@@ -18,7 +18,6 @@
 
 package org.apache.paimon.format;
 
-import org.apache.paimon.CoreOptions;
 import org.apache.paimon.append.AppendOnlyCompactManager;
 import org.apache.paimon.append.AppendOnlyWriter;
 import org.apache.paimon.data.BinaryString;
@@ -73,8 +72,7 @@ public class FileFormatSuffixTest extends 
KeyValueFileReadWriteTest {
                                 null, toCompact, 4, 10, 10, null, false), // 
not used
                         false,
                         dataFilePathFactory,
-                        null,
-                        CoreOptions.FILE_COMPRESSION.defaultValue());
+                        null);
         appendOnlyWriter.write(
                 GenericRow.of(1, BinaryString.fromString("aaa"), 
BinaryString.fromString("1")));
         CommitIncrement increment = appendOnlyWriter.prepareCommit(true);
diff --git 
a/paimon-core/src/test/java/org/apache/paimon/format/FlushingFileFormat.java 
b/paimon-core/src/test/java/org/apache/paimon/format/FlushingFileFormat.java
index d2f1829cf..548bd8b51 100644
--- a/paimon-core/src/test/java/org/apache/paimon/format/FlushingFileFormat.java
+++ b/paimon-core/src/test/java/org/apache/paimon/format/FlushingFileFormat.java
@@ -18,7 +18,6 @@
 
 package org.apache.paimon.format;
 
-import org.apache.paimon.CoreOptions;
 import org.apache.paimon.data.InternalRow;
 import org.apache.paimon.options.Options;
 import org.apache.paimon.predicate.Predicate;
@@ -48,11 +47,7 @@ public class FlushingFileFormat extends FileFormat {
     @Override
     public FormatWriterFactory createWriterFactory(RowType type) {
         return (PositionOutputStream, level) -> {
-            FormatWriter wrapped =
-                    format.createWriterFactory(type)
-                            .create(
-                                    PositionOutputStream,
-                                    
CoreOptions.FILE_COMPRESSION.defaultValue());
+            FormatWriter wrapped = 
format.createWriterFactory(type).create(PositionOutputStream);
             return new FormatWriter() {
                 @Override
                 public void addElement(InternalRow rowData) throws IOException 
{
diff --git 
a/paimon-core/src/test/java/org/apache/paimon/io/RollingFileWriterTest.java 
b/paimon-core/src/test/java/org/apache/paimon/io/RollingFileWriterTest.java
index f19f036d2..cfd3300f5 100644
--- a/paimon-core/src/test/java/org/apache/paimon/io/RollingFileWriterTest.java
+++ b/paimon-core/src/test/java/org/apache/paimon/io/RollingFileWriterTest.java
@@ -75,8 +75,7 @@ public class RollingFileWriterTest {
                                         SCHEMA,
                                         
fileFormat.createStatsExtractor(SCHEMA).orElse(null),
                                         0L,
-                                        new LongCounter(0),
-                                        
CoreOptions.FILE_COMPRESSION.defaultValue()),
+                                        new LongCounter(0)),
                         TARGET_FILE_SIZE);
     }
 
diff --git 
a/paimon-format/src/test/java/org/apache/paimon/format/BulkFileFormatTest.java 
b/paimon-format/src/test/java/org/apache/paimon/format/BulkFileFormatTest.java
index ccd9d2c7d..bf2d02803 100644
--- 
a/paimon-format/src/test/java/org/apache/paimon/format/BulkFileFormatTest.java
+++ 
b/paimon-format/src/test/java/org/apache/paimon/format/BulkFileFormatTest.java
@@ -71,7 +71,7 @@ public class BulkFileFormatTest {
         expected.add(GenericRow.of(2, 2));
         expected.add(GenericRow.of(3, 3));
         PositionOutputStream out = new LocalFileIO().newOutputStream(path, 
false);
-        FormatWriter writer = 
fileFormat.createWriterFactory(rowType).create(out, "LZ4");
+        FormatWriter writer = 
fileFormat.createWriterFactory(rowType).create(out);
         for (InternalRow row : expected) {
             writer.addElement(row);
         }
diff --git 
a/paimon-format/src/test/java/org/apache/paimon/format/orc/OrcWriterFactoryTest.java
 
b/paimon-format/src/test/java/org/apache/paimon/format/orc/OrcWriterFactoryTest.java
index 2511d7ed7..11c60a381 100644
--- 
a/paimon-format/src/test/java/org/apache/paimon/format/orc/OrcWriterFactoryTest.java
+++ 
b/paimon-format/src/test/java/org/apache/paimon/format/orc/OrcWriterFactoryTest.java
@@ -50,8 +50,8 @@ class OrcWriterFactoryTest {
                                 "struct<_col0:string,_col1:int>",
                                 new DataType[] {DataTypes.STRING(), 
DataTypes.INT()}),
                         memoryManager);
-        factory.create(new 
LocalPositionOutputStream(tmpDir.resolve("file1").toFile()), "LZ4");
-        factory.create(new 
LocalPositionOutputStream(tmpDir.resolve("file2").toFile()), "LZ4");
+        factory.create(new 
LocalPositionOutputStream(tmpDir.resolve("file1").toFile()));
+        factory.create(new 
LocalPositionOutputStream(tmpDir.resolve("file2").toFile()));
 
         List<Path> addedWriterPath = memoryManager.getAddedWriterPath();
         assertThat(addedWriterPath).hasSize(2);
diff --git 
a/paimon-format/src/test/java/org/apache/paimon/format/parquet/ParquetReadWriteTest.java
 
b/paimon-format/src/test/java/org/apache/paimon/format/parquet/ParquetReadWriteTest.java
index 260264229..245e13f7f 100644
--- 
a/paimon-format/src/test/java/org/apache/paimon/format/parquet/ParquetReadWriteTest.java
+++ 
b/paimon-format/src/test/java/org/apache/paimon/format/parquet/ParquetReadWriteTest.java
@@ -297,7 +297,7 @@ public class ParquetReadWriteTest {
         conf.setInteger("parquet.block.size", rowGroupSize);
         ParquetWriterFactory factory =
                 new ParquetWriterFactory(new RowDataParquetBuilder(ROW_TYPE, 
conf));
-        FormatWriter writer = factory.create(new 
LocalFileIO().newOutputStream(path, false), "LZ4");
+        FormatWriter writer = factory.create(new 
LocalFileIO().newOutputStream(path, false));
         for (InternalRow row : rows) {
             writer.addElement(row);
         }

Reply via email to