This is an automated email from the ASF dual-hosted git repository.
baiyangtx pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/amoro.git
The following commit(s) were added to refs/heads/master by this push:
new bb9146927 [AMORO-3117]: Integration of non-Iceberg related format in a
pluggable way (#3107)
bb9146927 is described below
commit bb9146927bb614ea9e42b2e6c52c409cb509a9a5
Author: baiyangtx <[email protected]>
AuthorDate: Wed Aug 28 17:20:08 2024 +0800
[AMORO-3117]: Integration of non-Iceberg related format in a pluggable way
(#3107)
* pluggable formats integrations
---------
Co-authored-by: ZhouJinsong <[email protected]>
---
README.md | 6 +
.../org/apache/amoro/api/process/AmoroProcess.java | 1 +
.../apache/amoro/api/process/OptimizingState.java | 1 +
.../org/apache/amoro/api/process/ProcessState.java | 1 +
.../org/apache/amoro/api/process/TableProcess.java | 1 +
.../amoro/api/process/TableProcessState.java | 1 +
amoro-ams/amoro-ams-server/pom.xml | 62 ++++++--
.../dashboard/MixedAndIcebergTableDescriptor.java | 172 +++++++++++++++++----
.../server/dashboard/ServerTableDescriptor.java | 40 +++--
.../reverser/IcebergTableMetaExtract.java | 1 +
.../dashboard/controller/TableController.java | 30 ++--
.../server/dashboard/model/HiveTableInfo.java | 1 +
.../server/dashboard/model/TableOperation.java | 2 +
.../server/dashboard/model/TableStatistics.java | 1 +
.../server/dashboard/utils/OptimizingUtil.java | 2 +-
.../server/dashboard/utils/TableStatCollector.java | 3 +-
.../amoro/server/optimizing/MetricsSummary.java | 4 +-
...he.amoro.table.descriptor.FormatTableDescriptor | 4 +-
.../amoro/server/catalog/TestServerCatalog.java | 4 +-
.../TestIcebergServerTableDescriptor.java | 7 +
amoro-core/pom.xml | 27 ----
.../amoro/hive/AuthenticatedHiveClientPool.java | 64 ++++----
.../org/apache/amoro}/process/OptimizingStage.java | 2 +-
.../org/apache/amoro}/process/ProcessStatus.java | 2 +-
.../apache/amoro/process/ProcessTaskStatus.java | 19 +--
.../amoro/table/descriptor}/AMSColumnInfo.java | 24 +--
.../amoro/table/descriptor}/AMSPartitionField.java | 15 +-
.../table/descriptor}/AmoroSnapshotsOfTable.java | 6 +-
.../amoro/table/descriptor}/ConsumerInfo.java | 2 +-
.../apache/amoro/table/descriptor}/DDLInfo.java | 2 +-
.../amoro/table/descriptor}/DDLReverser.java | 3 +-
.../amoro/table/descriptor}/FilesStatistics.java | 2 +-
.../table/descriptor}/FilesStatisticsBuilder.java | 4 +-
.../table/descriptor}/FormatTableDescriptor.java | 22 ++-
.../table/descriptor}/MetadataChangeHandler.java | 2 +-
.../amoro/table/descriptor}/OperationType.java | 2 +-
.../table/descriptor}/OptimizingProcessInfo.java | 71 +--------
.../table/descriptor}/OptimizingTaskInfo.java | 17 +-
.../amoro/table/descriptor}/PartitionBaseInfo.java | 6 +-
.../table/descriptor}/PartitionFileBaseInfo.java | 19 ++-
.../amoro/table/descriptor}/ServerTableMeta.java | 2 +-
.../descriptor}/SparkMetadataChangeHandler.java | 2 +-
.../amoro/table/descriptor}/TableMetaExtract.java | 4 +-
.../amoro/table/descriptor}/TableSummary.java | 2 +-
.../amoro/table/descriptor}/TagOrBranchInfo.java | 21 +--
.../PaimonSnapshot.java => utils/CommonUtil.java} | 46 +++---
.../services/org.apache.amoro.FormatCatalogFactory | 3 -
.../java/org/apache/amoro/hive/HMSMockServer.java | 31 ++--
.../test/java/org/apache/amoro/hive/TestHMS.java | 0
.../descriptor}/TestServerTableDescriptor.java | 71 +++++++--
.../ppr/PartitionExpressionForMetastore.java | 0
.../AuthorizationPreEventListener.java | 0
.../src/test/resources/hive-schema-3.1.0.derby.sql | 0
amoro-hudi-format/pom.xml | 62 ++++++++
.../amoro/formats/hudi/HudiCatalogFactory.java | 0
.../amoro/formats/hudi/HudiHadoopCatalog.java | 0
.../apache/amoro/formats/hudi/HudiHiveCatalog.java | 0
.../apache/amoro/formats/hudi/HudiSnapshot.java | 0
.../org/apache/amoro/formats/hudi/HudiTable.java | 30 ++--
.../amoro/formats/hudi}/HudiTableDescriptor.java | 91 +++++------
.../apache/amoro/formats/hudi}/HudiTableUtil.java | 2 +-
.../services/org.apache.amoro.FormatCatalogFactory | 1 -
...he.amoro.table.descriptor.FormatTableDescriptor | 4 +-
.../amoro-mixed-format-flink-common/pom.xml | 15 ++
.../flink/catalog/FlinkAmoroCatalogITCase.java | 4 +-
.../v3.2/amoro-mixed-format-spark-3.2/pom.xml | 7 +
.../v3.3/amoro-mixed-format-spark-3.3/pom.xml | 6 +
amoro-paimon-format/pom.xml | 60 +++++++
.../apache/amoro/formats/paimon/PaimonCatalog.java | 0
.../amoro/formats/paimon/PaimonCatalogFactory.java | 0
.../amoro/formats/paimon/PaimonSnapshot.java | 0
.../apache/amoro/formats/paimon/PaimonTable.java | 0
.../formats/paimon}/PaimonTableDescriptor.java | 64 ++++----
.../formats/paimon}/PaimonTableMetaExtract.java | 3 +-
.../formats/paimon}/PaimonTypeToSparkType.java | 2 +-
.../services/org.apache.amoro.FormatCatalogFactory | 3 +-
...he.amoro.table.descriptor.FormatTableDescriptor | 4 +-
.../paimon}/PaimonHadoopCatalogTestHelper.java | 4 +-
.../paimon}/PaimonHiveCatalogTestHelper.java | 3 +-
.../formats/paimon}/TestPaimonAmoroCatalog.java | 4 +-
.../paimon}/TestPaimonHiveAmoroCatalog.java | 3 +-
.../paimon}/TestPaimonServerTableDescriptor.java | 29 ++--
pom.xml | 14 ++
83 files changed, 720 insertions(+), 532 deletions(-)
diff --git a/README.md b/README.md
index 9f1f8276b..bc23104bb 100644
--- a/README.md
+++ b/README.md
@@ -96,6 +96,9 @@ Amoro support multiple processing engines for Mixed format as
below:
Amoro contains modules as below:
- `amoro-core` contains core abstractions and common implementation for other
modules
+- `amoro-iceberg-format` contains integration of Apache Iceberg format
+- `amoro-hudi-format` contains integration of Apache Hudi format
+- `amoro-paimon-format` contains integration of Apache Paimon format
- `amoro-ams` is amoro management service module
- `amoro-ams-api` contains ams thrift api and common interfaces
- `amoro-ams-dashboard` is the dashboard frontend for ams
@@ -122,6 +125,9 @@ Amoro is built using Maven with JDK 8 and JDK 17(only for
`amoro-mixed-format/am
* Specify Spark version for Spark optimizer(the default is 3.3.3): `mvn clean
package -DskipTests -Dspark-optimizer.spark-version=3.3.3`
* Build `amoro-mixed-format-trino` module under JDK 17: `mvn clean package
-DskipTests -Pformat-mixed-format-trino,build-mixed-format-trino -pl
'amoro-mixed-format/amoro-mixed-format-trino' -am`.
* Build all modules: `mvn clean package -DskipTests
-Ptoolchain,build-mixed-format-trino`, besides you need config `toolchains.xml`
in `${user.home}/.m2/` dir with content below.
+* Build a distribution package with all formats integrated: `mvn clean package
-Psupport-all-formats`
+ * Build a distribution package with Apache Paimon format: `mvn clean package
-Psupport-paimon-format`
+ * Build a distribution package with Apache Hudi format: `mvn clean package
-Psupport-hudi-format`
```
<?xml version="1.0" encoding="UTF-8"?>
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/AmoroProcess.java
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/AmoroProcess.java
index 968015209..39cb53e15 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/AmoroProcess.java
+++
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/AmoroProcess.java
@@ -19,6 +19,7 @@
package org.apache.amoro.api.process;
import org.apache.amoro.api.Action;
+import org.apache.amoro.process.ProcessStatus;
import java.util.Map;
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/OptimizingState.java
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/OptimizingState.java
index abdfdf3d8..a44c8be71 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/OptimizingState.java
+++
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/OptimizingState.java
@@ -21,6 +21,7 @@ package org.apache.amoro.api.process;
import org.apache.amoro.api.Action;
import org.apache.amoro.api.ServerTableIdentifier;
import org.apache.amoro.api.StateField;
+import org.apache.amoro.process.OptimizingStage;
/** The state of the optimizing process. */
public abstract class OptimizingState extends TableProcessState {
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessState.java
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessState.java
index d901dde1a..ee7b851a7 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessState.java
+++
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessState.java
@@ -19,6 +19,7 @@
package org.apache.amoro.api.process;
import org.apache.amoro.api.Action;
+import org.apache.amoro.process.ProcessStatus;
import java.util.Map;
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcess.java
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcess.java
index 29fbae592..a15154528 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcess.java
+++
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcess.java
@@ -19,6 +19,7 @@
package org.apache.amoro.api.process;
import org.apache.amoro.api.TableRuntime;
+import org.apache.amoro.process.ProcessStatus;
/**
* An abstract table process to handle table state.
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcessState.java
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcessState.java
index 94b036000..96a63ffe3 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcessState.java
+++
b/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/TableProcessState.java
@@ -21,6 +21,7 @@ package org.apache.amoro.api.process;
import org.apache.amoro.api.Action;
import org.apache.amoro.api.ServerTableIdentifier;
import org.apache.amoro.api.StateField;
+import org.apache.amoro.process.ProcessStatus;
import java.util.Map;
diff --git a/amoro-ams/amoro-ams-server/pom.xml
b/amoro-ams/amoro-ams-server/pom.xml
index a13bb67c7..5261a8fb2 100644
--- a/amoro-ams/amoro-ams-server/pom.xml
+++ b/amoro-ams/amoro-ams-server/pom.xml
@@ -339,19 +339,6 @@
<scope>runtime</scope>
</dependency>
- <dependency>
- <groupId>org.apache.paimon</groupId>
- <artifactId>paimon-s3</artifactId>
- <scope>runtime</scope>
- </dependency>
-
- <dependency>
- <groupId>org.apache.paimon</groupId>
- <artifactId>paimon-spark-common</artifactId>
- <version>${paimon.version}</version>
- <scope>runtime</scope>
- </dependency>
-
<!-- test dependencies -->
<dependency>
<groupId>org.apache.amoro</groupId>
@@ -385,6 +372,21 @@
<type>test-jar</type>
</dependency>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ <version>${project.version}</version>
+ <scope>test</scope>
+ </dependency>
+
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ <version>${project.version}</version>
+ <scope>test</scope>
+ <type>test-jar</type>
+ </dependency>
+
<dependency>
<groupId>org.apache.curator</groupId>
<artifactId>curator-test</artifactId>
@@ -470,4 +472,38 @@
</plugin>
</plugins>
</build>
+
+ <profiles>
+ <profile>
+ <id>support-paimon-format</id>
+ <dependencies>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ </dependency>
+ </dependencies>
+ </profile>
+ <profile>
+ <id>support-hudi-format</id>
+ <dependencies>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-hudi-format</artifactId>
+ </dependency>
+ </dependencies>
+ </profile>
+ <profile>
+ <id>support-all-formats</id>
+ <dependencies>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ </dependency>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-hudi-format</artifactId>
+ </dependency>
+ </dependencies>
+ </profile>
+ </profiles>
</project>
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/MixedAndIcebergTableDescriptor.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/MixedAndIcebergTableDescriptor.java
index 1db9ad37b..fe7920dcd 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/MixedAndIcebergTableDescriptor.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/MixedAndIcebergTableDescriptor.java
@@ -23,28 +23,17 @@ import org.apache.amoro.TableFormat;
import org.apache.amoro.api.CommitMetaProducer;
import org.apache.amoro.data.DataFileType;
import org.apache.amoro.data.FileNameRules;
-import org.apache.amoro.server.dashboard.component.reverser.DDLReverser;
+import org.apache.amoro.process.ProcessStatus;
+import org.apache.amoro.process.ProcessTaskStatus;
import
org.apache.amoro.server.dashboard.component.reverser.IcebergTableMetaExtract;
-import org.apache.amoro.server.dashboard.model.AMSColumnInfo;
-import org.apache.amoro.server.dashboard.model.AMSPartitionField;
-import org.apache.amoro.server.dashboard.model.AmoroSnapshotsOfTable;
-import org.apache.amoro.server.dashboard.model.ConsumerInfo;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
-import org.apache.amoro.server.dashboard.model.FilesStatistics;
-import org.apache.amoro.server.dashboard.model.OperationType;
-import org.apache.amoro.server.dashboard.model.OptimizingProcessInfo;
-import org.apache.amoro.server.dashboard.model.OptimizingTaskInfo;
-import org.apache.amoro.server.dashboard.model.PartitionBaseInfo;
-import org.apache.amoro.server.dashboard.model.PartitionFileBaseInfo;
-import org.apache.amoro.server.dashboard.model.ServerTableMeta;
import org.apache.amoro.server.dashboard.model.TableBasicInfo;
import org.apache.amoro.server.dashboard.model.TableStatistics;
-import org.apache.amoro.server.dashboard.model.TableSummary;
-import org.apache.amoro.server.dashboard.model.TagOrBranchInfo;
import org.apache.amoro.server.dashboard.utils.AmsUtil;
import org.apache.amoro.server.dashboard.utils.TableStatCollector;
+import org.apache.amoro.server.optimizing.MetricsSummary;
import org.apache.amoro.server.optimizing.OptimizingProcessMeta;
import org.apache.amoro.server.optimizing.OptimizingTaskMeta;
+import org.apache.amoro.server.optimizing.TaskRuntime;
import org.apache.amoro.server.persistence.PersistentBase;
import org.apache.amoro.server.persistence.mapper.OptimizingMapper;
import org.apache.amoro.shade.guava32.com.google.common.collect.ImmutableList;
@@ -52,23 +41,43 @@ import
org.apache.amoro.shade.guava32.com.google.common.collect.Iterables;
import org.apache.amoro.shade.guava32.com.google.common.collect.Maps;
import org.apache.amoro.table.KeyedTable;
import org.apache.amoro.table.MixedTable;
+import org.apache.amoro.table.PrimaryKeySpec;
import org.apache.amoro.table.TableIdentifier;
import org.apache.amoro.table.TableProperties;
import org.apache.amoro.table.UnkeyedTable;
+import org.apache.amoro.table.descriptor.AMSColumnInfo;
+import org.apache.amoro.table.descriptor.AMSPartitionField;
+import org.apache.amoro.table.descriptor.AmoroSnapshotsOfTable;
+import org.apache.amoro.table.descriptor.ConsumerInfo;
+import org.apache.amoro.table.descriptor.DDLInfo;
+import org.apache.amoro.table.descriptor.DDLReverser;
+import org.apache.amoro.table.descriptor.FilesStatistics;
+import org.apache.amoro.table.descriptor.FormatTableDescriptor;
+import org.apache.amoro.table.descriptor.OperationType;
+import org.apache.amoro.table.descriptor.OptimizingProcessInfo;
+import org.apache.amoro.table.descriptor.OptimizingTaskInfo;
+import org.apache.amoro.table.descriptor.PartitionBaseInfo;
+import org.apache.amoro.table.descriptor.PartitionFileBaseInfo;
+import org.apache.amoro.table.descriptor.ServerTableMeta;
+import org.apache.amoro.table.descriptor.TableSummary;
+import org.apache.amoro.table.descriptor.TagOrBranchInfo;
import org.apache.amoro.utils.MixedDataFiles;
import org.apache.amoro.utils.MixedTableUtil;
import org.apache.commons.collections.CollectionUtils;
+import org.apache.commons.lang3.tuple.Pair;
import org.apache.iceberg.ContentFile;
import org.apache.iceberg.HasTableOperations;
import org.apache.iceberg.IcebergFindFiles;
+import org.apache.iceberg.PartitionField;
import org.apache.iceberg.PartitionSpec;
+import org.apache.iceberg.Schema;
import org.apache.iceberg.Snapshot;
import org.apache.iceberg.SnapshotRef;
import org.apache.iceberg.SnapshotSummary;
import org.apache.iceberg.Table;
import org.apache.iceberg.data.GenericRecord;
import org.apache.iceberg.io.CloseableIterable;
-import org.apache.iceberg.util.Pair;
+import org.apache.iceberg.types.Types;
import org.apache.iceberg.util.PropertyUtil;
import org.apache.iceberg.util.SnapshotUtil;
import org.slf4j.Logger;
@@ -92,10 +101,11 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
private static final Logger LOG =
LoggerFactory.getLogger(MixedAndIcebergTableDescriptor.class);
- private final ExecutorService executorService;
+ private ExecutorService executorService;
- public MixedAndIcebergTableDescriptor(ExecutorService executorService) {
- this.executorService = executorService;
+ @Override
+ public void withIoExecutor(ExecutorService ioExecutor) {
+ this.executorService = ioExecutor;
}
@Override
@@ -257,8 +267,8 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
private void collectSnapshots(
List<AmoroSnapshotsOfTable> snapshotsOfTables, Pair<Table, Long>
tableAndSnapshotId) {
- Table table = tableAndSnapshotId.first();
- Long snapshotId = tableAndSnapshotId.second();
+ Table table = tableAndSnapshotId.getLeft();
+ Long snapshotId = tableAndSnapshotId.getRight();
if (snapshotId != null) {
SnapshotUtil.ancestorsOf(snapshotId, table::snapshot)
.forEach(
@@ -351,7 +361,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
result.add(
new PartitionFileBaseInfo(
snapshotId,
- DataFileType.ofContentId(f.content().id()),
+ DataFileType.ofContentId(f.content().id()).name(),
snapshotTime,
MixedTableUtil.getMixedTablePartitionSpecById(mixedTable, f.specId())
.partitionToPath(f.partition()),
@@ -365,7 +375,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
result.add(
new PartitionFileBaseInfo(
snapshotId,
- DataFileType.ofContentId(f.content().id()),
+ DataFileType.ofContentId(f.content().id()).name(),
snapshotTime,
MixedTableUtil.getMixedTablePartitionSpecById(mixedTable, f.specId())
.partitionToPath(f.partition()),
@@ -379,7 +389,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
result.add(
new PartitionFileBaseInfo(
snapshotId,
- DataFileType.ofContentId(f.content().id()),
+ DataFileType.ofContentId(f.content().id()).name(),
snapshotTime,
MixedTableUtil.getMixedTablePartitionSpecById(mixedTable, f.specId())
.partitionToPath(f.partition()),
@@ -393,7 +403,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
result.add(
new PartitionFileBaseInfo(
snapshotId,
- DataFileType.ofContentId(f.content().id()),
+ DataFileType.ofContentId(f.content().id()).name(),
snapshotTime,
MixedTableUtil.getMixedTablePartitionSpecById(mixedTable, f.specId())
.partitionToPath(f.partition()),
@@ -516,7 +526,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
return Pair.of(
processMetaList.stream()
- .map(p -> OptimizingProcessInfo.build(p,
optimizingTasks.get(p.getProcessId())))
+ .map(p -> buildOptimizingProcessInfo(p,
optimizingTasks.get(p.getProcessId())))
.collect(Collectors.toList()),
total);
}
@@ -540,7 +550,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
String.valueOf(taskMeta.getProcessId()),
taskMeta.getTaskId(),
taskMeta.getPartitionData(),
- taskMeta.getStatus(),
+ ProcessTaskStatus.valueOf(taskMeta.getStatus().name()),
taskMeta.getRetryNum(),
taskMeta.getOptimizerToken(),
taskMeta.getThreadId(),
@@ -601,7 +611,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
}
return new PartitionFileBaseInfo(
String.valueOf(snapshotId),
- dataFileType,
+ dataFileType.name(),
commitTime,
partitionPath,
contentFile.specId(),
@@ -668,11 +678,11 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
fillTableProperties(serverTableMeta, table.properties());
serverTableMeta.setPartitionColumnList(
table.spec().fields().stream()
- .map(item ->
AMSPartitionField.buildFromPartitionSpec(table.spec().schema(), item))
+ .map(item ->
buildPartitionFieldFromPartitionSpec(table.spec().schema(), item))
.collect(Collectors.toList()));
serverTableMeta.setSchema(
table.schema().columns().stream()
- .map(AMSColumnInfo::buildFromNestedField)
+
.map(MixedAndIcebergTableDescriptor::buildColumnInfoFromNestedField)
.collect(Collectors.toList()));
serverTableMeta.setFilter(null);
@@ -682,7 +692,7 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
if (kt.primaryKeySpec() != null) {
serverTableMeta.setPkList(
kt.primaryKeySpec().fields().stream()
- .map(item ->
AMSColumnInfo.buildFromPartitionSpec(table.spec().schema(), item))
+ .map(item ->
buildColumnInfoFromPartitionSpec(table.spec().schema(), item))
.collect(Collectors.toList()));
}
}
@@ -729,11 +739,109 @@ public class MixedAndIcebergTableDescriptor extends
PersistentBase
snapshotRefs.forEach(
(name, snapshotRef) -> {
if (predicate.test(snapshotRef)) {
- result.add(new TagOrBranchInfo(name, snapshotRef));
+ result.add(buildTagOrBranchInfo(name, snapshotRef));
}
});
result.sort(Comparator.comparing(TagOrBranchInfo::getName));
return result;
}
}
+
+ private static AMSColumnInfo
buildColumnInfoFromNestedField(Types.NestedField field) {
+ if (field == null) {
+ return null;
+ }
+ return new AMSColumnInfo.Builder()
+ .field(field.name())
+ .type(field.type().toString())
+ .required(field.isRequired())
+ .comment(field.doc())
+ .build();
+ }
+
+ /** Construct ColumnInfo based on schema and primary key field. */
+ private static AMSColumnInfo buildColumnInfoFromPartitionSpec(
+ Schema schema, PrimaryKeySpec.PrimaryKeyField pkf) {
+ return buildColumnInfoFromNestedField(schema.findField(pkf.fieldName()));
+ }
+
+ private static AMSPartitionField buildPartitionFieldFromPartitionSpec(
+ Schema schema, PartitionField pf) {
+ return new AMSPartitionField.Builder()
+ .field(pf.name())
+ .sourceField(schema.findColumnName(pf.sourceId()))
+ .transform(pf.transform().toString())
+ .fieldId(pf.fieldId())
+ .sourceFieldId(pf.sourceId())
+ .build();
+ }
+
+ private static TagOrBranchInfo buildTagOrBranchInfo(String name, SnapshotRef
snapshotRef) {
+ String type = null;
+ if (snapshotRef.isTag()) {
+ type = TagOrBranchInfo.TAG;
+ } else if (snapshotRef.isBranch()) {
+ type = TagOrBranchInfo.BRANCH;
+ } else {
+ throw new RuntimeException("Invalid snapshot ref: " + snapshotRef);
+ }
+ return new TagOrBranchInfo(
+ name,
+ snapshotRef.snapshotId(),
+ snapshotRef.minSnapshotsToKeep(),
+ snapshotRef.maxSnapshotAgeMs(),
+ snapshotRef.maxRefAgeMs(),
+ type);
+ }
+
+ private static OptimizingProcessInfo buildOptimizingProcessInfo(
+ OptimizingProcessMeta meta, List<OptimizingTaskMeta>
optimizingTaskStats) {
+ if (meta == null) {
+ return null;
+ }
+ OptimizingProcessInfo result = new OptimizingProcessInfo();
+
+ if (optimizingTaskStats != null) {
+ int successTasks = 0;
+ int runningTasks = 0;
+ for (OptimizingTaskMeta optimizingTaskStat : optimizingTaskStats) {
+ TaskRuntime.Status status = optimizingTaskStat.getStatus();
+ switch (status) {
+ case SUCCESS:
+ successTasks++;
+ break;
+ case SCHEDULED:
+ case ACKED:
+ runningTasks++;
+ break;
+ }
+ }
+ result.setTotalTasks(optimizingTaskStats.size());
+ result.setSuccessTasks(successTasks);
+ result.setRunningTasks(runningTasks);
+ }
+ MetricsSummary summary = meta.getSummary();
+ if (summary != null) {
+ result.setInputFiles(summary.getInputFilesStatistics());
+ result.setOutputFiles(summary.getOutputFilesStatistics());
+ }
+
+ result.setTableId(meta.getTableId());
+ result.setCatalogName(meta.getCatalogName());
+ result.setDbName(meta.getDbName());
+ result.setTableName(meta.getTableName());
+
+ result.setProcessId(String.valueOf(meta.getProcessId()));
+ result.setStartTime(meta.getPlanTime());
+ result.setOptimizingType(meta.getOptimizingType().name());
+ result.setStatus(ProcessStatus.valueOf(meta.getStatus().name()));
+ result.setFailReason(meta.getFailReason());
+ result.setDuration(
+ meta.getEndTime() > 0
+ ? meta.getEndTime() - meta.getPlanTime()
+ : System.currentTimeMillis() - meta.getPlanTime());
+ result.setFinishTime(meta.getEndTime());
+ result.setSummary(meta.getSummary().summaryAsMap(true));
+ return result;
+ }
}
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/ServerTableDescriptor.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/ServerTableDescriptor.java
index 21f77331d..9c3d1c11a 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/ServerTableDescriptor.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/ServerTableDescriptor.java
@@ -23,24 +23,26 @@ import org.apache.amoro.TableFormat;
import org.apache.amoro.api.TableIdentifier;
import org.apache.amoro.api.config.Configurations;
import org.apache.amoro.server.catalog.ServerCatalog;
-import org.apache.amoro.server.dashboard.model.AmoroSnapshotsOfTable;
-import org.apache.amoro.server.dashboard.model.ConsumerInfo;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
-import org.apache.amoro.server.dashboard.model.OperationType;
-import org.apache.amoro.server.dashboard.model.OptimizingProcessInfo;
-import org.apache.amoro.server.dashboard.model.OptimizingTaskInfo;
-import org.apache.amoro.server.dashboard.model.PartitionBaseInfo;
-import org.apache.amoro.server.dashboard.model.PartitionFileBaseInfo;
-import org.apache.amoro.server.dashboard.model.ServerTableMeta;
-import org.apache.amoro.server.dashboard.model.TagOrBranchInfo;
import org.apache.amoro.server.persistence.PersistentBase;
import org.apache.amoro.server.table.TableService;
-import org.apache.iceberg.util.Pair;
+import org.apache.amoro.table.descriptor.AmoroSnapshotsOfTable;
+import org.apache.amoro.table.descriptor.ConsumerInfo;
+import org.apache.amoro.table.descriptor.DDLInfo;
+import org.apache.amoro.table.descriptor.FormatTableDescriptor;
+import org.apache.amoro.table.descriptor.OperationType;
+import org.apache.amoro.table.descriptor.OptimizingProcessInfo;
+import org.apache.amoro.table.descriptor.OptimizingTaskInfo;
+import org.apache.amoro.table.descriptor.PartitionBaseInfo;
+import org.apache.amoro.table.descriptor.PartitionFileBaseInfo;
+import org.apache.amoro.table.descriptor.ServerTableMeta;
+import org.apache.amoro.table.descriptor.TagOrBranchInfo;
+import org.apache.commons.lang3.tuple.Pair;
import org.apache.iceberg.util.ThreadPools;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
+import java.util.ServiceLoader;
import java.util.concurrent.ExecutorService;
public class ServerTableDescriptor extends PersistentBase {
@@ -54,17 +56,13 @@ public class ServerTableDescriptor extends PersistentBase {
// All table formats will jointly reuse the work thread pool named
iceberg-worker-pool-%d
ExecutorService executorService = ThreadPools.getWorkerPool();
-
- FormatTableDescriptor[] formatTableDescriptors =
- new FormatTableDescriptor[] {
- new MixedAndIcebergTableDescriptor(executorService),
- new PaimonTableDescriptor(executorService),
- new HudiTableDescriptor(executorService)
- };
- for (FormatTableDescriptor formatTableDescriptor : formatTableDescriptors)
{
- for (TableFormat format : formatTableDescriptor.supportFormat()) {
- formatDescriptorMap.put(format, formatTableDescriptor);
+ ServiceLoader<FormatTableDescriptor> tableDescriptorLoader =
+ ServiceLoader.load(FormatTableDescriptor.class);
+ for (FormatTableDescriptor descriptor : tableDescriptorLoader) {
+ for (TableFormat format : descriptor.supportFormat()) {
+ formatDescriptorMap.put(format, descriptor);
}
+ descriptor.withIoExecutor(executorService);
}
}
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/IcebergTableMetaExtract.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/IcebergTableMetaExtract.java
index aa023c2fb..ad8b44e7e 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/IcebergTableMetaExtract.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/IcebergTableMetaExtract.java
@@ -19,6 +19,7 @@
package org.apache.amoro.server.dashboard.component.reverser;
import org.apache.amoro.shade.guava32.com.google.common.collect.Lists;
+import org.apache.amoro.table.descriptor.TableMetaExtract;
import org.apache.iceberg.HasTableOperations;
import org.apache.iceberg.Schema;
import org.apache.iceberg.Table;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/controller/TableController.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/controller/TableController.java
index 39cf9cd71..3ff3473a7 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/controller/TableController.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/controller/TableController.java
@@ -37,21 +37,9 @@ import org.apache.amoro.properties.HiveTableProperties;
import org.apache.amoro.server.catalog.ServerCatalog;
import org.apache.amoro.server.dashboard.ServerTableDescriptor;
import org.apache.amoro.server.dashboard.ServerTableProperties;
-import org.apache.amoro.server.dashboard.model.AMSColumnInfo;
-import org.apache.amoro.server.dashboard.model.AmoroSnapshotsOfTable;
-import org.apache.amoro.server.dashboard.model.ConsumerInfo;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
import org.apache.amoro.server.dashboard.model.HiveTableInfo;
-import org.apache.amoro.server.dashboard.model.OperationType;
-import org.apache.amoro.server.dashboard.model.OptimizingProcessInfo;
-import org.apache.amoro.server.dashboard.model.OptimizingTaskInfo;
-import org.apache.amoro.server.dashboard.model.PartitionBaseInfo;
-import org.apache.amoro.server.dashboard.model.PartitionFileBaseInfo;
-import org.apache.amoro.server.dashboard.model.ServerTableMeta;
import org.apache.amoro.server.dashboard.model.TableMeta;
import org.apache.amoro.server.dashboard.model.TableOperation;
-import org.apache.amoro.server.dashboard.model.TableSummary;
-import org.apache.amoro.server.dashboard.model.TagOrBranchInfo;
import org.apache.amoro.server.dashboard.model.UpgradeHiveMeta;
import org.apache.amoro.server.dashboard.model.UpgradeRunningInfo;
import org.apache.amoro.server.dashboard.model.UpgradeStatus;
@@ -68,12 +56,24 @@ import
org.apache.amoro.shade.guava32.com.google.common.util.concurrent.ThreadFa
import org.apache.amoro.table.TableIdentifier;
import org.apache.amoro.table.TableMetaStore;
import org.apache.amoro.table.TableProperties;
+import org.apache.amoro.table.descriptor.AMSColumnInfo;
+import org.apache.amoro.table.descriptor.AmoroSnapshotsOfTable;
+import org.apache.amoro.table.descriptor.ConsumerInfo;
+import org.apache.amoro.table.descriptor.DDLInfo;
+import org.apache.amoro.table.descriptor.OperationType;
+import org.apache.amoro.table.descriptor.OptimizingProcessInfo;
+import org.apache.amoro.table.descriptor.OptimizingTaskInfo;
+import org.apache.amoro.table.descriptor.PartitionBaseInfo;
+import org.apache.amoro.table.descriptor.PartitionFileBaseInfo;
+import org.apache.amoro.table.descriptor.ServerTableMeta;
+import org.apache.amoro.table.descriptor.TableSummary;
+import org.apache.amoro.table.descriptor.TagOrBranchInfo;
import org.apache.amoro.utils.CatalogUtil;
import org.apache.commons.lang3.StringUtils;
+import org.apache.commons.lang3.tuple.Pair;
import org.apache.hadoop.hive.metastore.api.FieldSchema;
import org.apache.hadoop.hive.metastore.api.Table;
import org.apache.iceberg.SnapshotRef;
-import org.apache.iceberg.util.Pair;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@@ -317,8 +317,8 @@ public class TableController {
Pair<List<OptimizingProcessInfo>, Integer> optimizingProcessesInfo =
tableDescriptor.getOptimizingProcessesInfo(
tableIdentifier.buildTableIdentifier(), limit, offset);
- List<OptimizingProcessInfo> result = optimizingProcessesInfo.first();
- int total = optimizingProcessesInfo.second();
+ List<OptimizingProcessInfo> result = optimizingProcessesInfo.getLeft();
+ int total = optimizingProcessesInfo.getRight();
ctx.json(OkResponse.of(PageResult.of(result, total)));
}
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/HiveTableInfo.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/HiveTableInfo.java
index 8b1369533..d8408bee6 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/HiveTableInfo.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/HiveTableInfo.java
@@ -19,6 +19,7 @@
package org.apache.amoro.server.dashboard.model;
import org.apache.amoro.table.TableIdentifier;
+import org.apache.amoro.table.descriptor.AMSColumnInfo;
import java.util.List;
import java.util.Map;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableOperation.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableOperation.java
index aa92c2ba6..46a35d1a3 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableOperation.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableOperation.java
@@ -18,6 +18,8 @@
package org.apache.amoro.server.dashboard.model;
+import org.apache.amoro.table.descriptor.DDLInfo;
+
public class TableOperation {
long ts;
String operation;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableStatistics.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableStatistics.java
index 60967dbbc..65af9c949 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableStatistics.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableStatistics.java
@@ -19,6 +19,7 @@
package org.apache.amoro.server.dashboard.model;
import org.apache.amoro.table.TableIdentifier;
+import org.apache.amoro.table.descriptor.FilesStatistics;
import java.util.Map;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/OptimizingUtil.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/OptimizingUtil.java
index ef6c70a29..6bd53c0ca 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/OptimizingUtil.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/OptimizingUtil.java
@@ -18,13 +18,13 @@
package org.apache.amoro.server.dashboard.utils;
-import org.apache.amoro.server.dashboard.model.FilesStatistics;
import org.apache.amoro.server.dashboard.model.TableOptimizingInfo;
import org.apache.amoro.server.optimizing.MetricsSummary;
import org.apache.amoro.server.optimizing.OptimizingProcess;
import org.apache.amoro.server.optimizing.OptimizingStatus;
import org.apache.amoro.server.optimizing.plan.OptimizingEvaluator;
import org.apache.amoro.server.table.TableRuntime;
+import org.apache.amoro.table.descriptor.FilesStatistics;
import org.apache.iceberg.ContentFile;
public class OptimizingUtil {
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/TableStatCollector.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/TableStatCollector.java
index 4b64c3bdb..c98886e12 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/TableStatCollector.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/TableStatCollector.java
@@ -19,13 +19,14 @@
package org.apache.amoro.server.dashboard.utils;
import org.apache.amoro.Constants;
-import org.apache.amoro.server.dashboard.model.FilesStatistics;
import org.apache.amoro.server.dashboard.model.SnapshotInfo;
import org.apache.amoro.server.dashboard.model.TableStatistics;
import org.apache.amoro.table.KeyedTable;
import org.apache.amoro.table.MixedTable;
import org.apache.amoro.table.TableIdentifier;
import org.apache.amoro.table.UnkeyedTable;
+import org.apache.amoro.table.descriptor.FilesStatistics;
+import org.apache.amoro.table.descriptor.FilesStatisticsBuilder;
import org.apache.iceberg.Snapshot;
import org.apache.iceberg.SnapshotSummary;
import org.apache.iceberg.Table;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/optimizing/MetricsSummary.java
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/optimizing/MetricsSummary.java
index 7ca1fe0e1..283f72673 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/optimizing/MetricsSummary.java
+++
b/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/optimizing/MetricsSummary.java
@@ -21,10 +21,10 @@ package org.apache.amoro.server.optimizing;
import static org.apache.amoro.server.dashboard.utils.AmsUtil.byteToXB;
import org.apache.amoro.optimizing.RewriteFilesInput;
-import org.apache.amoro.server.dashboard.model.FilesStatistics;
-import org.apache.amoro.server.dashboard.utils.FilesStatisticsBuilder;
import org.apache.amoro.shade.guava32.com.google.common.base.MoreObjects;
import
org.apache.amoro.shade.jackson2.com.fasterxml.jackson.annotation.JsonIgnoreProperties;
+import org.apache.amoro.table.descriptor.FilesStatistics;
+import org.apache.amoro.table.descriptor.FilesStatisticsBuilder;
import org.apache.iceberg.ContentFile;
import org.apache.iceberg.DataFile;
import org.apache.iceberg.FileContent;
diff --git
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
b/amoro-ams/amoro-ams-server/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
similarity index 88%
copy from
amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
copy to
amoro-ams/amoro-ams-server/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
index 72d74f17d..18dd68113 100644
---
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
+++
b/amoro-ams/amoro-ams-server/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
@@ -15,6 +15,4 @@
# See the License for the specific language governing permissions and
# limitations under the License.
#
-
-org.apache.amoro.formats.paimon.PaimonCatalogFactory
-org.apache.amoro.formats.hudi.HudiCatalogFactory
\ No newline at end of file
+org.apache.amoro.server.dashboard.MixedAndIcebergTableDescriptor
\ No newline at end of file
diff --git
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/catalog/TestServerCatalog.java
b/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/catalog/TestServerCatalog.java
index 76e1bbd14..1e41c1ede 100644
---
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/catalog/TestServerCatalog.java
+++
b/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/catalog/TestServerCatalog.java
@@ -22,10 +22,10 @@ import org.apache.amoro.api.CatalogMeta;
import org.apache.amoro.formats.AmoroCatalogTestHelper;
import org.apache.amoro.formats.IcebergHadoopCatalogTestHelper;
import org.apache.amoro.formats.MixedIcebergHadoopCatalogTestHelper;
-import org.apache.amoro.formats.PaimonHadoopCatalogTestHelper;
+import org.apache.amoro.formats.paimon.PaimonHadoopCatalogTestHelper;
+import org.apache.amoro.formats.paimon.PaimonHiveCatalogTestHelper;
import org.apache.amoro.hive.formats.IcebergHiveCatalogTestHelper;
import org.apache.amoro.hive.formats.MixedIcebergHiveCatalogTestHelper;
-import org.apache.amoro.hive.formats.PaimonHiveCatalogTestHelper;
import org.apache.amoro.properties.CatalogMetaProperties;
import org.junit.Assert;
import org.junit.Assume;
diff --git
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestIcebergServerTableDescriptor.java
b/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestIcebergServerTableDescriptor.java
index fcf279668..7bd401154 100644
---
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestIcebergServerTableDescriptor.java
+++
b/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestIcebergServerTableDescriptor.java
@@ -21,6 +21,8 @@ package org.apache.amoro.server.dashboard;
import org.apache.amoro.formats.AmoroCatalogTestHelper;
import org.apache.amoro.formats.IcebergHadoopCatalogTestHelper;
import org.apache.amoro.hive.formats.IcebergHiveCatalogTestHelper;
+import org.apache.amoro.table.descriptor.FormatTableDescriptor;
+import org.apache.amoro.table.descriptor.TestServerTableDescriptor;
import org.apache.iceberg.Table;
import org.apache.iceberg.types.Types;
import org.junit.runner.RunWith;
@@ -77,6 +79,11 @@ public class TestIcebergServerTableDescriptor extends
TestServerTableDescriptor
getTable().updateSchema().deleteColumn("renamed_col").commit();
}
+ @Override
+ protected FormatTableDescriptor getTableDescriptor() {
+ return new MixedAndIcebergTableDescriptor();
+ }
+
private Table getTable() {
return (Table) getAmoroCatalog().loadTable(TEST_DB,
TEST_TABLE).originalTable();
}
diff --git a/amoro-core/pom.xml b/amoro-core/pom.xml
index 3166403d3..71686781c 100644
--- a/amoro-core/pom.xml
+++ b/amoro-core/pom.xml
@@ -59,23 +59,6 @@
<artifactId>caffeine</artifactId>
</dependency>
- <!-- apache hudi dependencies -->
- <dependency>
- <groupId>org.apache.hudi</groupId>
- <artifactId>hudi-java-client</artifactId>
- <scope>provided</scope>
- <exclusions>
- <exclusion>
- <groupId>org.apache.orc</groupId>
- <artifactId>orc-core</artifactId>
- </exclusion>
- <exclusion>
- <groupId>asm</groupId>
- <artifactId>asm</artifactId>
- </exclusion>
- </exclusions>
- </dependency>
-
<dependency>
<groupId>org.apache.amoro</groupId>
<artifactId>amoro-shade-zookeeper-3</artifactId>
@@ -118,11 +101,6 @@
<classifier>nohive</classifier>
</dependency>
- <dependency>
- <groupId>org.apache.paimon</groupId>
- <artifactId>paimon-bundle</artifactId>
- </dependency>
-
<dependency>
<groupId>org.roaringbitmap</groupId>
<artifactId>RoaringBitmap</artifactId>
@@ -134,11 +112,6 @@
<artifactId>hive-metastore</artifactId>
</dependency>
- <dependency>
- <groupId>org.apache.iceberg</groupId>
- <artifactId>iceberg-hive-metastore</artifactId>
- </dependency>
-
<dependency>
<groupId>org.apache.iceberg</groupId>
<artifactId>iceberg-bundled-guava</artifactId>
diff --git
a/amoro-core/src/main/java/org/apache/amoro/hive/AuthenticatedHiveClientPool.java
b/amoro-core/src/main/java/org/apache/amoro/hive/AuthenticatedHiveClientPool.java
index 33206eeb2..6c5c0107b 100644
---
a/amoro-core/src/main/java/org/apache/amoro/hive/AuthenticatedHiveClientPool.java
+++
b/amoro-core/src/main/java/org/apache/amoro/hive/AuthenticatedHiveClientPool.java
@@ -19,13 +19,13 @@
package org.apache.amoro.hive;
import org.apache.amoro.client.ClientPoolImpl;
+import
org.apache.amoro.shade.guava32.com.google.common.annotations.VisibleForTesting;
import org.apache.amoro.table.TableMetaStore;
import org.apache.amoro.utils.DynConstructors;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hive.conf.HiveConf;
import org.apache.hadoop.hive.metastore.HiveMetaStoreClient;
import org.apache.hadoop.hive.metastore.api.MetaException;
-import org.apache.iceberg.hive.RuntimeMetaException;
import org.apache.thrift.TException;
import org.apache.thrift.transport.TTransportException;
import org.slf4j.Logger;
@@ -47,6 +47,36 @@ public class AuthenticatedHiveClientPool extends
ClientPoolImpl<HMSClient, TExce
.impl(HiveMetaStoreClient.class, Configuration.class)
.build();
+ @VisibleForTesting
+ public static HMSClient createHiveMetaStoreClient(HiveConf hiveConf) {
+ try {
+ try {
+ HiveMetaStoreClient client = CLIENT_CTOR.newInstance(hiveConf);
+ return new HMSClientImpl(client);
+ } catch (RuntimeException e) {
+ // any MetaException would be wrapped into RuntimeException during
reflection, so
+ // let's double-check type
+ // here
+ if (e.getCause() instanceof MetaException) {
+ throw (MetaException) e.getCause();
+ }
+ throw e;
+ }
+ } catch (MetaException e) {
+ throw new RuntimeException("Failed to connect to Hive Metastore", e);
+ } catch (Throwable t) {
+ if (t.getMessage().contains("Another instance of Derby may have already
booted")) {
+ throw new RuntimeException(
+ "Failed to start an embedded metastore because embedded "
+ + "Derby supports only one client at a time. To fix this, use
a metastore that supports "
+ + "multiple clients.",
+ t);
+ }
+
+ throw new RuntimeException("Failed to connect to Hive Metastore", t);
+ }
+ }
+
public AuthenticatedHiveClientPool(TableMetaStore tableMetaStore, int
poolSize) {
super(poolSize, TTransportException.class, true);
this.hiveConf =
@@ -58,35 +88,7 @@ public class AuthenticatedHiveClientPool extends
ClientPoolImpl<HMSClient, TExce
@Override
protected HMSClient newClient() {
- return metaStore.doAs(
- () -> {
- try {
- try {
- HiveMetaStoreClient client = CLIENT_CTOR.newInstance(hiveConf);
- return new HMSClientImpl(client);
- } catch (RuntimeException e) {
- // any MetaException would be wrapped into RuntimeException
during reflection, so
- // let's double-check type
- // here
- if (e.getCause() instanceof MetaException) {
- throw (MetaException) e.getCause();
- }
- throw e;
- }
- } catch (MetaException e) {
- throw new RuntimeMetaException(e, "Failed to connect to Hive
Metastore");
- } catch (Throwable t) {
- if (t.getMessage().contains("Another instance of Derby may have
already booted")) {
- throw new RuntimeMetaException(
- t,
- "Failed to start an embedded metastore because embedded "
- + "Derby supports only one client at a time. To fix
this, use a metastore that supports "
- + "multiple clients.");
- }
-
- throw new RuntimeMetaException(t, "Failed to connect to Hive
Metastore");
- }
- });
+ return metaStore.doAs(() -> createHiveMetaStoreClient(hiveConf));
}
@Override
@@ -98,7 +100,7 @@ public class AuthenticatedHiveClientPool extends
ClientPoolImpl<HMSClient, TExce
client.close();
client.reconnect();
} catch (MetaException e) {
- throw new RuntimeMetaException(e, "Failed to reconnect to Hive
Metastore");
+ throw new RuntimeException("Failed to reconnect to Hive
Metastore", e);
}
return client;
});
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/OptimizingStage.java
b/amoro-core/src/main/java/org/apache/amoro/process/OptimizingStage.java
similarity index 98%
rename from
amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/OptimizingStage.java
rename to amoro-core/src/main/java/org/apache/amoro/process/OptimizingStage.java
index ea340c280..81d5b5274 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/OptimizingStage.java
+++ b/amoro-core/src/main/java/org/apache/amoro/process/OptimizingStage.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.api.process;
+package org.apache.amoro.process;
/** The stage of the optimizing process. */
public enum OptimizingStage {
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessStatus.java
b/amoro-core/src/main/java/org/apache/amoro/process/ProcessStatus.java
similarity index 96%
copy from
amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessStatus.java
copy to amoro-core/src/main/java/org/apache/amoro/process/ProcessStatus.java
index 0df65e5c0..60a22545d 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessStatus.java
+++ b/amoro-core/src/main/java/org/apache/amoro/process/ProcessStatus.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.api.process;
+package org.apache.amoro.process;
/**
* Status of any {@link AmoroProcess}. Only UNKNOWN, RUNNING,
FINISHED(SUCCESS, CLOSED, FAILED) are
diff --git
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessStatus.java
b/amoro-core/src/main/java/org/apache/amoro/process/ProcessTaskStatus.java
similarity index 68%
rename from
amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessStatus.java
rename to
amoro-core/src/main/java/org/apache/amoro/process/ProcessTaskStatus.java
index 0df65e5c0..f139a1c7e 100644
---
a/amoro-ams/amoro-ams-api/src/main/java/org/apache/amoro/api/process/ProcessStatus.java
+++ b/amoro-core/src/main/java/org/apache/amoro/process/ProcessTaskStatus.java
@@ -16,18 +16,13 @@
* limitations under the License.
*/
-package org.apache.amoro.api.process;
+package org.apache.amoro.process;
-/**
- * Status of any {@link AmoroProcess}. Only UNKNOWN, RUNNING,
FINISHED(SUCCESS, CLOSED, FAILED) are
- * necessary Stage classes are used to define multiple phases of one process
such as OptimizingStage
- */
-public enum ProcessStatus {
- UNKNOWN,
-
- /** This status containing scheduled and running phases */
- ACTIVE,
+public enum ProcessTaskStatus {
+ PLANNED,
+ SCHEDULED,
+ ACKED,
+ FAILED,
SUCCESS,
- CLOSED,
- FAILED
+ CANCELED // If Optimizing process failed, all tasks will be CANCELED except
for SUCCESS tasks
}
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AMSColumnInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/AMSColumnInfo.java
similarity index 75%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AMSColumnInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/AMSColumnInfo.java
index 63a0fc857..6b92f08fc 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AMSColumnInfo.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/AMSColumnInfo.java
@@ -16,11 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
-
-import org.apache.amoro.table.PrimaryKeySpec;
-import org.apache.iceberg.Schema;
-import org.apache.iceberg.types.Types;
+package org.apache.amoro.table.descriptor;
/** AMS server column info. */
public class AMSColumnInfo {
@@ -70,24 +66,6 @@ public class AMSColumnInfo {
this.comment = comment;
}
- public static AMSColumnInfo buildFromNestedField(Types.NestedField field) {
- if (field == null) {
- return null;
- }
- return new Builder()
- .field(field.name())
- .type(field.type().toString())
- .required(field.isRequired())
- .comment(field.doc())
- .build();
- }
-
- /** Construct ColumnInfo based on schema and primary key field. */
- public static AMSColumnInfo buildFromPartitionSpec(
- Schema schema, PrimaryKeySpec.PrimaryKeyField pkf) {
- return buildFromNestedField(schema.findField(pkf.fieldName()));
- }
-
public static class Builder {
String field;
String type;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AMSPartitionField.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/AMSPartitionField.java
similarity index 85%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AMSPartitionField.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/AMSPartitionField.java
index 385165ba8..473b0918b 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AMSPartitionField.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/AMSPartitionField.java
@@ -16,10 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
-
-import org.apache.iceberg.PartitionField;
-import org.apache.iceberg.Schema;
+package org.apache.amoro.table.descriptor;
public class AMSPartitionField {
String field;
@@ -79,16 +76,6 @@ public class AMSPartitionField {
this.sourceFieldId = sourceFieldId;
}
- public static AMSPartitionField buildFromPartitionSpec(Schema schema,
PartitionField pf) {
- return new Builder()
- .field(pf.name())
- .sourceField(schema.findColumnName(pf.sourceId()))
- .transform(pf.transform().toString())
- .fieldId(pf.fieldId())
- .sourceFieldId(pf.sourceId())
- .build();
- }
-
public static class Builder {
String field;
String sourceField;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AmoroSnapshotsOfTable.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/AmoroSnapshotsOfTable.java
similarity index 96%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AmoroSnapshotsOfTable.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/AmoroSnapshotsOfTable.java
index f3ac22ca3..fdc666633 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/AmoroSnapshotsOfTable.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/AmoroSnapshotsOfTable.java
@@ -16,10 +16,10 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
-import org.apache.amoro.server.dashboard.utils.AmsUtil;
import org.apache.amoro.shade.guava32.com.google.common.base.Objects;
+import org.apache.amoro.utils.CommonUtil;
import java.util.Map;
@@ -67,7 +67,7 @@ public class AmoroSnapshotsOfTable {
}
public String getFileSize() {
- return AmsUtil.byteToXB(fileSize);
+ return CommonUtil.byteToXB(fileSize);
}
public void setFileSize(long fileSize) {
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/ConsumerInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/ConsumerInfo.java
similarity index 96%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/ConsumerInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/ConsumerInfo.java
index 4e21a69d6..dd5eb6d47 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/ConsumerInfo.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/ConsumerInfo.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
public class ConsumerInfo {
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/DDLInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/DDLInfo.java
similarity index 97%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/DDLInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/DDLInfo.java
index 445cc497f..ca9542807 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/DDLInfo.java
+++ b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/DDLInfo.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
import org.apache.amoro.table.TableIdentifier;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/DDLReverser.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/DDLReverser.java
similarity index 98%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/DDLReverser.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/DDLReverser.java
index 576c0de32..d89e8e342 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/DDLReverser.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/DDLReverser.java
@@ -16,9 +16,8 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.component.reverser;
+package org.apache.amoro.table.descriptor;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
import org.apache.amoro.shade.guava32.com.google.common.collect.Maps;
import org.apache.amoro.shade.guava32.com.google.common.collect.Sets;
import org.apache.amoro.table.TableIdentifier;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/FilesStatistics.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/FilesStatistics.java
similarity index 98%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/FilesStatistics.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/FilesStatistics.java
index 72eb3bc74..84d634379 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/FilesStatistics.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/FilesStatistics.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
import org.apache.amoro.shade.guava32.com.google.common.base.MoreObjects;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/FilesStatisticsBuilder.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/FilesStatisticsBuilder.java
similarity index 92%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/FilesStatisticsBuilder.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/FilesStatisticsBuilder.java
index 0c7865fd2..4e1064e9b 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/utils/FilesStatisticsBuilder.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/FilesStatisticsBuilder.java
@@ -16,9 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.utils;
-
-import org.apache.amoro.server.dashboard.model.FilesStatistics;
+package org.apache.amoro.table.descriptor;
public class FilesStatisticsBuilder {
private int fileCnt = 0;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/FormatTableDescriptor.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/FormatTableDescriptor.java
similarity index 79%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/FormatTableDescriptor.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/FormatTableDescriptor.java
index 72ecf271b..baee06a36 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/FormatTableDescriptor.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/FormatTableDescriptor.java
@@ -16,27 +16,25 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard;
+package org.apache.amoro.table.descriptor;
import org.apache.amoro.AmoroTable;
import org.apache.amoro.TableFormat;
-import org.apache.amoro.server.dashboard.model.AmoroSnapshotsOfTable;
-import org.apache.amoro.server.dashboard.model.ConsumerInfo;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
-import org.apache.amoro.server.dashboard.model.OperationType;
-import org.apache.amoro.server.dashboard.model.OptimizingProcessInfo;
-import org.apache.amoro.server.dashboard.model.OptimizingTaskInfo;
-import org.apache.amoro.server.dashboard.model.PartitionBaseInfo;
-import org.apache.amoro.server.dashboard.model.PartitionFileBaseInfo;
-import org.apache.amoro.server.dashboard.model.ServerTableMeta;
-import org.apache.amoro.server.dashboard.model.TagOrBranchInfo;
-import org.apache.iceberg.util.Pair;
+import org.apache.commons.lang3.tuple.Pair;
import java.util.List;
+import java.util.concurrent.ExecutorService;
/** API for obtaining metadata information of various formats. */
public interface FormatTableDescriptor {
+ /**
+ * Global io-executor pool for table descriptor
+ *
+ * @param ioExecutor io executor pool.
+ */
+ void withIoExecutor(ExecutorService ioExecutor);
+
/** Get the format supported by this descriptor. */
List<TableFormat> supportFormat();
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/MetadataChangeHandler.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/MetadataChangeHandler.java
similarity index 97%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/MetadataChangeHandler.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/MetadataChangeHandler.java
index ab11cb28a..e2ebfd06e 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/MetadataChangeHandler.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/MetadataChangeHandler.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.component.reverser;
+package org.apache.amoro.table.descriptor;
import java.util.List;
import java.util.Map;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OperationType.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/OperationType.java
similarity index 97%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OperationType.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/OperationType.java
index 23d42718c..32e5c1983 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OperationType.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/OperationType.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
import java.util.Arrays;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OptimizingProcessInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/OptimizingProcessInfo.java
similarity index 60%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OptimizingProcessInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/OptimizingProcessInfo.java
index f7f258790..acd503e07 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OptimizingProcessInfo.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/OptimizingProcessInfo.java
@@ -16,15 +16,10 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
-import org.apache.amoro.server.optimizing.MetricsSummary;
-import org.apache.amoro.server.optimizing.OptimizingProcess;
-import org.apache.amoro.server.optimizing.OptimizingProcessMeta;
-import org.apache.amoro.server.optimizing.OptimizingTaskMeta;
-import org.apache.amoro.server.optimizing.OptimizingType;
+import org.apache.amoro.process.ProcessStatus;
-import java.util.List;
import java.util.Map;
public class OptimizingProcessInfo {
@@ -35,8 +30,8 @@ public class OptimizingProcessInfo {
private String processId;
private long startTime;
- private OptimizingType optimizingType;
- private OptimizingProcess.Status status;
+ private String optimizingType;
+ private ProcessStatus status;
private String failReason;
private long duration;
private int successTasks;
@@ -95,19 +90,19 @@ public class OptimizingProcessInfo {
this.startTime = startTime;
}
- public OptimizingType getOptimizingType() {
+ public String getOptimizingType() {
return optimizingType;
}
- public void setOptimizingType(OptimizingType optimizingType) {
+ public void setOptimizingType(String optimizingType) {
this.optimizingType = optimizingType;
}
- public OptimizingProcess.Status getStatus() {
+ public ProcessStatus getStatus() {
return status;
}
- public void setStatus(OptimizingProcess.Status status) {
+ public void setStatus(ProcessStatus status) {
this.status = status;
}
@@ -182,54 +177,4 @@ public class OptimizingProcessInfo {
public void setSummary(Map<String, String> summary) {
this.summary = summary;
}
-
- public static OptimizingProcessInfo build(
- OptimizingProcessMeta meta, List<OptimizingTaskMeta>
optimizingTaskStats) {
- if (meta == null) {
- return null;
- }
- OptimizingProcessInfo result = new OptimizingProcessInfo();
-
- if (optimizingTaskStats != null) {
- int successTasks = 0;
- int runningTasks = 0;
- for (OptimizingTaskMeta optimizingTaskStat : optimizingTaskStats) {
- switch (optimizingTaskStat.getStatus()) {
- case SUCCESS:
- successTasks++;
- break;
- case SCHEDULED:
- case ACKED:
- runningTasks++;
- break;
- }
- }
- result.setTotalTasks(optimizingTaskStats.size());
- result.setSuccessTasks(successTasks);
- result.setRunningTasks(runningTasks);
- }
- MetricsSummary summary = meta.getSummary();
- if (summary != null) {
- result.setInputFiles(summary.getInputFilesStatistics());
- result.setOutputFiles(summary.getOutputFilesStatistics());
- }
-
- result.setTableId(meta.getTableId());
- result.setCatalogName(meta.getCatalogName());
- result.setDbName(meta.getDbName());
- result.setTableName(meta.getTableName());
-
- result.setProcessId(String.valueOf(meta.getProcessId()));
- result.setStartTime(meta.getPlanTime());
- result.setOptimizingType(meta.getOptimizingType());
- result.setStatus(meta.getStatus());
- result.setFailReason(meta.getFailReason());
- result.setDuration(
- meta.getEndTime() > 0
- ? meta.getEndTime() - meta.getPlanTime()
- : System.currentTimeMillis() - meta.getPlanTime());
- result.setFinishTime(meta.getEndTime());
- result.setSummary(meta.getSummary().summaryAsMap(true));
- return result;
- }
}
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OptimizingTaskInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/OptimizingTaskInfo.java
similarity index 91%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OptimizingTaskInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/OptimizingTaskInfo.java
index c4acb0b0c..0eff65ca7 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/OptimizingTaskInfo.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/OptimizingTaskInfo.java
@@ -16,10 +16,9 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
-import org.apache.amoro.server.AmoroServiceConstants;
-import org.apache.amoro.server.optimizing.TaskRuntime;
+import org.apache.amoro.process.ProcessTaskStatus;
import java.util.Map;
@@ -31,7 +30,7 @@ public class OptimizingTaskInfo {
private String processId;
private int taskId;
private String partitionData;
- private TaskRuntime.Status status;
+ private ProcessTaskStatus status;
private int retryNum;
private String optimizerToken;
private int threadId;
@@ -49,7 +48,7 @@ public class OptimizingTaskInfo {
String processId,
int taskId,
String partitionData,
- TaskRuntime.Status status,
+ ProcessTaskStatus status,
int retryNum,
String optimizerToken,
int threadId,
@@ -71,9 +70,7 @@ public class OptimizingTaskInfo {
this.threadId = threadId;
this.startTime = startTime;
this.endTime = endTime;
- if (costTime == 0
- && startTime != AmoroServiceConstants.INVALID_TIME
- && endTime == AmoroServiceConstants.INVALID_TIME) {
+ if (costTime == 0 && startTime > 0 && endTime <= 0) {
this.costTime = System.currentTimeMillis() - startTime;
} else {
this.costTime = costTime;
@@ -122,11 +119,11 @@ public class OptimizingTaskInfo {
this.partitionData = partitionData;
}
- public TaskRuntime.Status getStatus() {
+ public ProcessTaskStatus getStatus() {
return status;
}
- public void setStatus(TaskRuntime.Status status) {
+ public void setStatus(ProcessTaskStatus status) {
this.status = status;
}
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/PartitionBaseInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/PartitionBaseInfo.java
similarity index 93%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/PartitionBaseInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/PartitionBaseInfo.java
index 8ba48a09b..933bcb874 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/PartitionBaseInfo.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/PartitionBaseInfo.java
@@ -16,9 +16,9 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
-import org.apache.amoro.server.dashboard.utils.AmsUtil;
+import org.apache.amoro.utils.CommonUtil;
public class PartitionBaseInfo {
String partition;
@@ -71,7 +71,7 @@ public class PartitionBaseInfo {
public void setFileSize(long fileSize) {
this.fileSize = fileSize;
- this.size = AmsUtil.byteToXB(fileSize);
+ this.size = CommonUtil.byteToXB(fileSize);
}
public long getLastCommitTime() {
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/PartitionFileBaseInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/PartitionFileBaseInfo.java
similarity index 87%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/PartitionFileBaseInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/PartitionFileBaseInfo.java
index 0cf18049b..70944bc98 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/PartitionFileBaseInfo.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/PartitionFileBaseInfo.java
@@ -16,14 +16,13 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
-import org.apache.amoro.data.DataFileType;
-import org.apache.amoro.server.dashboard.utils.AmsUtil;
+import org.apache.amoro.utils.CommonUtil;
public class PartitionFileBaseInfo {
private String commitId;
- private DataFileType fileType;
+ private String fileType;
private Long commitTime;
private String size;
private String partition;
@@ -35,7 +34,7 @@ public class PartitionFileBaseInfo {
public PartitionFileBaseInfo(
String commitId,
- DataFileType fileType,
+ String fileType,
Long commitTime,
String partition,
int specId,
@@ -52,7 +51,7 @@ public class PartitionFileBaseInfo {
public PartitionFileBaseInfo(
String commitId,
- DataFileType fileType,
+ String fileType,
Long commitTime,
String partition,
String path,
@@ -75,11 +74,11 @@ public class PartitionFileBaseInfo {
this.commitId = commitId;
}
- public DataFileType getFileType() {
+ public String getFileType() {
return fileType;
}
- public void setFileType(DataFileType fileType) {
+ public void setFileType(String fileType) {
this.fileType = fileType;
}
@@ -113,7 +112,7 @@ public class PartitionFileBaseInfo {
public void setPath(String path) {
this.path = path;
- this.file = AmsUtil.getFileName(path);
+ this.file = CommonUtil.getFileName(path);
}
public String getFile() {
@@ -126,7 +125,7 @@ public class PartitionFileBaseInfo {
public void setFileSize(long fileSize) {
this.fileSize = fileSize;
- this.size = AmsUtil.byteToXB(fileSize);
+ this.size = CommonUtil.byteToXB(fileSize);
}
public String getOperation() {
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/ServerTableMeta.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/ServerTableMeta.java
similarity index 99%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/ServerTableMeta.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/ServerTableMeta.java
index 9394c6535..910b74bac 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/ServerTableMeta.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/ServerTableMeta.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
import org.apache.amoro.shade.guava32.com.google.common.base.MoreObjects;
import org.apache.amoro.table.TableIdentifier;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/SparkMetadataChangeHandler.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/SparkMetadataChangeHandler.java
similarity index 98%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/SparkMetadataChangeHandler.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/SparkMetadataChangeHandler.java
index e7d79991e..7f9870f60 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/SparkMetadataChangeHandler.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/SparkMetadataChangeHandler.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.component.reverser;
+package org.apache.amoro.table.descriptor;
import java.util.List;
import java.util.Map;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/TableMetaExtract.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/TableMetaExtract.java
similarity index 97%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/TableMetaExtract.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/TableMetaExtract.java
index b385434db..34b60f4dd 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/TableMetaExtract.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/TableMetaExtract.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.component.reverser;
+package org.apache.amoro.table.descriptor;
import org.apache.amoro.shade.guava32.com.google.common.base.MoreObjects;
@@ -31,7 +31,7 @@ import java.util.Map;
*
* @param <T> Table type
*/
-interface TableMetaExtract<T> {
+public interface TableMetaExtract<T> {
/** Extract the historical metadata of a table. */
List<InternalTableMeta> extractTable(T table);
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableSummary.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/TableSummary.java
similarity index 97%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableSummary.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/TableSummary.java
index 1ffdf853e..95e611e7f 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TableSummary.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/TableSummary.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
+package org.apache.amoro.table.descriptor;
/** Table summary for page of details */
public class TableSummary {
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TagOrBranchInfo.java
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/TagOrBranchInfo.java
similarity index 78%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TagOrBranchInfo.java
rename to
amoro-core/src/main/java/org/apache/amoro/table/descriptor/TagOrBranchInfo.java
index 6773fa11e..828672994 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/model/TagOrBranchInfo.java
+++
b/amoro-core/src/main/java/org/apache/amoro/table/descriptor/TagOrBranchInfo.java
@@ -16,15 +16,13 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.model;
-
-import org.apache.iceberg.SnapshotRef;
+package org.apache.amoro.table.descriptor;
public class TagOrBranchInfo {
public static final String TAG = "tag";
public static final String BRANCH = "branch";
public static final TagOrBranchInfo MAIN_BRANCH =
- new TagOrBranchInfo(SnapshotRef.MAIN_BRANCH, -1, -1, 0L, 0L, BRANCH);
+ new TagOrBranchInfo("main", -1, -1, 0L, 0L, BRANCH);
private String name;
private long snapshotId;
@@ -50,21 +48,6 @@ public class TagOrBranchInfo {
this.type = type;
}
- public TagOrBranchInfo(String name, SnapshotRef snapshotRef) {
- this.name = name;
- this.snapshotId = snapshotRef.snapshotId();
- this.minSnapshotsToKeep = snapshotRef.minSnapshotsToKeep();
- this.maxSnapshotAgeMs = snapshotRef.maxSnapshotAgeMs();
- this.maxRefAgeMs = snapshotRef.maxRefAgeMs();
- if (snapshotRef.isTag()) {
- this.type = TAG;
- } else if (snapshotRef.isBranch()) {
- this.type = BRANCH;
- } else {
- throw new RuntimeException("Invalid snapshot ref: " + snapshotRef);
- }
- }
-
public String getName() {
return name;
}
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonSnapshot.java
b/amoro-core/src/main/java/org/apache/amoro/utils/CommonUtil.java
similarity index 53%
copy from
amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonSnapshot.java
copy to amoro-core/src/main/java/org/apache/amoro/utils/CommonUtil.java
index 52c0a369d..eadc6c71c 100644
---
a/amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonSnapshot.java
+++ b/amoro-core/src/main/java/org/apache/amoro/utils/CommonUtil.java
@@ -16,32 +16,28 @@
* limitations under the License.
*/
-package org.apache.amoro.formats.paimon;
-
-import org.apache.amoro.TableSnapshot;
-import org.apache.paimon.Snapshot;
-
-public class PaimonSnapshot implements TableSnapshot {
-
- private final Snapshot snapshot;
-
- public PaimonSnapshot(Snapshot snapshot) {
- this.snapshot = snapshot;
- }
-
- @Override
- public long watermark() {
- Long watermark = snapshot.watermark();
- return watermark == null ? -1 : watermark;
- }
-
- @Override
- public long commitTime() {
- return snapshot.timeMillis();
+package org.apache.amoro.utils;
+
+import java.io.File;
+
+public class CommonUtil {
+ /** Convert size to a different unit, ensuring that the converted value is >
1 */
+ public static String byteToXB(long size) {
+ String[] units = new String[] {"B", "KB", "MB", "GB", "TB", "PB", "EB"};
+ float result = size, tmpResult = size;
+ int unitIdx = 0;
+ int unitCnt = units.length;
+ while (true) {
+ result = result / 1024;
+ if (result < 1 || unitIdx >= unitCnt - 1) {
+ return String.format("%2.2f%s", tmpResult, units[unitIdx]);
+ }
+ tmpResult = result;
+ unitIdx += 1;
+ }
}
- @Override
- public String id() {
- return String.valueOf(snapshot.id());
+ public static String getFileName(String path) {
+ return path == null ? null : new File(path).getName();
}
}
diff --git
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
b/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
index 72d74f17d..9d35e3840 100644
---
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
+++
b/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
@@ -15,6 +15,3 @@
# See the License for the specific language governing permissions and
# limitations under the License.
#
-
-org.apache.amoro.formats.paimon.PaimonCatalogFactory
-org.apache.amoro.formats.hudi.HudiCatalogFactory
\ No newline at end of file
diff --git
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/HMSMockServer.java
b/amoro-core/src/test/java/org/apache/amoro/hive/HMSMockServer.java
similarity index 94%
rename from
amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/HMSMockServer.java
rename to amoro-core/src/test/java/org/apache/amoro/hive/HMSMockServer.java
index 14921a7fb..a1b743956 100644
---
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/HMSMockServer.java
+++ b/amoro-core/src/test/java/org/apache/amoro/hive/HMSMockServer.java
@@ -18,6 +18,8 @@
package org.apache.amoro.hive;
+import org.apache.amoro.utils.DynConstructors;
+import org.apache.amoro.utils.DynMethods;
import org.apache.commons.io.FileUtils;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FileStatus;
@@ -29,10 +31,6 @@ import org.apache.hadoop.hive.metastore.HiveMetaStoreClient;
import org.apache.hadoop.hive.metastore.IHMSHandler;
import org.apache.hadoop.hive.metastore.RetryingHMSHandler;
import org.apache.hadoop.hive.metastore.TSetIpAddressProcessor;
-import org.apache.iceberg.common.DynConstructors;
-import org.apache.iceberg.common.DynMethods;
-import org.apache.iceberg.hadoop.Util;
-import org.apache.iceberg.hive.HiveClientPool;
import org.apache.thrift.protocol.TBinaryProtocol;
import org.apache.thrift.server.TServer;
import org.apache.thrift.server.TThreadPoolServer;
@@ -110,7 +108,7 @@ public class HMSMockServer {
private ExecutorService executorService;
private TServer server;
private HiveMetaStore.HMSHandler baseHandler;
- private HiveClientPool clientPool;
+ private HMSClient clientPool;
private final int port;
private HiveMetaStoreClient client;
private boolean started = false;
@@ -159,7 +157,7 @@ public class HMSMockServer {
HiveConf.ConfVars.METASTOREURIS.varname,
hiveConf.getVar(HiveConf.ConfVars.METASTOREURIS));
- this.clientPool = new HiveClientPool(1, hiveConf);
+ this.clientPool =
AuthenticatedHiveClientPool.createHiveMetaStoreClient(hiveConf);
started = true;
} catch (Exception e) {
throw new RuntimeException("Cannot start TestHiveMetastore", e);
@@ -224,7 +222,7 @@ public class HMSMockServer {
return hiveConf;
}
- public HiveClientPool clientPool() {
+ public HMSClient clientPool() {
return clientPool;
}
@@ -244,27 +242,20 @@ public class HMSMockServer {
}
public void reset() throws Exception {
- for (String dbName : clientPool.run(client -> client.getAllDatabases())) {
- for (String tblName : clientPool.run(client ->
client.getAllTables(dbName))) {
- clientPool.run(
- client -> {
- client.dropTable(dbName, tblName, true, true, true);
- return null;
- });
+ for (String dbName : client.getAllDatabases()) {
+ for (String tblName : client.getAllTables(dbName)) {
+ client.dropTable(dbName, tblName, true, true, true);
}
if (!DEFAULT_DATABASE_NAME.equals(dbName)) {
// Drop cascade, functions dropped by cascade
- clientPool.run(
- client -> {
- client.dropDatabase(dbName, true, true, true);
- return null;
- });
+ client.dropDatabase(dbName, true, true, true);
}
}
Path warehouseRoot = new Path(hiveLocalDir.getAbsolutePath());
- FileSystem fs = Util.getFs(warehouseRoot, hiveConf);
+ FileSystem fs = warehouseRoot.getFileSystem(hiveConf);
+ // FileSystem fs = Util.getFs(warehouseRoot, hiveConf);
for (FileStatus fileStatus : fs.listStatus(warehouseRoot)) {
if (!fileStatus.getPath().getName().equals("derby.log")
&& !fileStatus.getPath().getName().equals("metastore_db")) {
diff --git
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/TestHMS.java
b/amoro-core/src/test/java/org/apache/amoro/hive/TestHMS.java
similarity index 100%
rename from
amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/TestHMS.java
rename to amoro-core/src/test/java/org/apache/amoro/hive/TestHMS.java
diff --git
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestServerTableDescriptor.java
b/amoro-core/src/test/java/org/apache/amoro/table/descriptor/TestServerTableDescriptor.java
similarity index 63%
rename from
amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestServerTableDescriptor.java
rename to
amoro-core/src/test/java/org/apache/amoro/table/descriptor/TestServerTableDescriptor.java
index be6942d8b..ef6f4d262 100644
---
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestServerTableDescriptor.java
+++
b/amoro-core/src/test/java/org/apache/amoro/table/descriptor/TestServerTableDescriptor.java
@@ -16,35 +16,63 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard;
+package org.apache.amoro.table.descriptor;
-import org.apache.amoro.api.config.Configurations;
+import org.apache.amoro.AmoroCatalog;
+import org.apache.amoro.AmoroTable;
import org.apache.amoro.formats.AmoroCatalogTestHelper;
-import org.apache.amoro.server.catalog.TableCatalogTestBase;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
-import org.apache.amoro.table.TableIdentifier;
+import org.apache.amoro.hive.TestHMS;
+import org.junit.After;
import org.junit.Assert;
import org.junit.Before;
+import org.junit.ClassRule;
+import org.junit.Rule;
import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+import java.io.IOException;
import java.util.List;
+import java.util.concurrent.Executors;
-public abstract class TestServerTableDescriptor extends TableCatalogTestBase {
+public abstract class TestServerTableDescriptor {
protected static final String TEST_DB = "test_db";
protected static final String TEST_TABLE = "test_table";
+ @Rule public TemporaryFolder temp = new TemporaryFolder();
+
+ @ClassRule public static TestHMS TEST_HMS = new TestHMS();
+
+ private final AmoroCatalogTestHelper<?> amoroCatalogTestHelper;
+ private AmoroCatalog amoroCatalog;
+ private Object originalCatalog;
+
public TestServerTableDescriptor(AmoroCatalogTestHelper<?>
amoroCatalogTestHelper) {
- super(amoroCatalogTestHelper);
+ this.amoroCatalogTestHelper = amoroCatalogTestHelper;
}
@Before
- public void before() {
+ public void before() throws IOException {
+ String path = temp.newFolder().getPath();
+ amoroCatalogTestHelper.initWarehouse(path);
+ amoroCatalogTestHelper.initHiveConf(TEST_HMS.getHiveConf());
+ this.amoroCatalog = amoroCatalogTestHelper.amoroCatalog();
+ this.originalCatalog = amoroCatalogTestHelper.originalCatalog();
+
getAmoroCatalog().createDatabase(TEST_DB);
+ try {
+ this.amoroCatalogTestHelper.createTable(TEST_DB, TEST_TABLE);
+ } catch (Exception e) {
+ throw new RuntimeException(e);
+ }
+ }
+ @After
+ public void after() throws IOException {
try {
- getAmoroCatalogTestHelper().createTable(TEST_DB, TEST_TABLE);
+ this.amoroCatalogTestHelper.amoroCatalog().dropTable(TEST_DB,
TEST_TABLE, true);
+ this.amoroCatalogTestHelper.amoroCatalog().dropDatabase(TEST_DB);
} catch (Exception e) {
throw new RuntimeException(e);
}
@@ -52,14 +80,14 @@ public abstract class TestServerTableDescriptor extends
TableCatalogTestBase {
@Test
public void tableOperations() throws Exception {
- ServerTableDescriptor serverTableDescriptor =
- new ServerTableDescriptor(tableService(), new Configurations());
+ FormatTableDescriptor tableDescriptor = getTableDescriptor();
+ tableDescriptor.withIoExecutor(Executors.newSingleThreadExecutor());
// add properties
- getAmoroCatalogTestHelper().setTableProperties(TEST_DB, TEST_TABLE, "k1",
"v1");
+ amoroCatalogTestHelper.setTableProperties(TEST_DB, TEST_TABLE, "k1", "v1");
// remove properties
- getAmoroCatalogTestHelper().removeTableProperties(TEST_DB, TEST_TABLE,
"k1");
+ amoroCatalogTestHelper.removeTableProperties(TEST_DB, TEST_TABLE, "k1");
// add columns
tableOperationsAddColumns();
@@ -79,10 +107,9 @@ public abstract class TestServerTableDescriptor extends
TableCatalogTestBase {
// change columns default value
tableOperationsDropColumn();
- List<DDLInfo> tableOperations =
- serverTableDescriptor.getTableOperations(
- TableIdentifier.of(getAmoroCatalogTestHelper().catalogName(),
TEST_DB, TEST_TABLE)
- .buildTableIdentifier());
+ AmoroTable<?> table = amoroCatalog.loadTable(TEST_DB, TEST_TABLE);
+
+ List<DDLInfo> tableOperations = tableDescriptor.getTableOperations(table);
Assert.assertEquals(
tableOperations.get(0).getDdl(), "ALTER TABLE test_table SET
TBLPROPERTIES ('k1' = 'v1')");
@@ -139,4 +166,14 @@ public abstract class TestServerTableDescriptor extends
TableCatalogTestBase {
protected abstract void tableOperationsChangeColumnRequired();
protected abstract void tableOperationsDropColumn();
+
+ protected abstract FormatTableDescriptor getTableDescriptor();
+
+ protected AmoroCatalog getAmoroCatalog() {
+ return amoroCatalog;
+ }
+
+ protected Object getOriginalCatalog() {
+ return originalCatalog;
+ }
}
diff --git
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/hadoop/hive/ql/optimizer/ppr/PartitionExpressionForMetastore.java
b/amoro-core/src/test/java/org/apache/hadoop/hive/ql/optimizer/ppr/PartitionExpressionForMetastore.java
similarity index 100%
rename from
amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/hadoop/hive/ql/optimizer/ppr/PartitionExpressionForMetastore.java
rename to
amoro-core/src/test/java/org/apache/hadoop/hive/ql/optimizer/ppr/PartitionExpressionForMetastore.java
diff --git
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/hadoop/hive/ql/security/authorization/AuthorizationPreEventListener.java
b/amoro-core/src/test/java/org/apache/hadoop/hive/ql/security/authorization/AuthorizationPreEventListener.java
similarity index 100%
rename from
amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/hadoop/hive/ql/security/authorization/AuthorizationPreEventListener.java
rename to
amoro-core/src/test/java/org/apache/hadoop/hive/ql/security/authorization/AuthorizationPreEventListener.java
diff --git
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/resources/hive-schema-3.1.0.derby.sql
b/amoro-core/src/test/resources/hive-schema-3.1.0.derby.sql
similarity index 100%
rename from
amoro-mixed-format/amoro-mixed-format-hive/src/test/resources/hive-schema-3.1.0.derby.sql
rename to amoro-core/src/test/resources/hive-schema-3.1.0.derby.sql
diff --git a/amoro-hudi-format/pom.xml b/amoro-hudi-format/pom.xml
new file mode 100644
index 000000000..e69d94b91
--- /dev/null
+++ b/amoro-hudi-format/pom.xml
@@ -0,0 +1,62 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+ ~ 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.
+ -->
+<project xmlns="http://maven.apache.org/POM/4.0.0"
+ xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+ xsi:schemaLocation="http://maven.apache.org/POM/4.0.0
http://maven.apache.org/xsd/maven-4.0.0.xsd">
+ <modelVersion>4.0.0</modelVersion>
+ <parent>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-parent</artifactId>
+ <version>0.8-SNAPSHOT</version>
+ <relativePath>../pom.xml</relativePath>
+ </parent>
+
+ <artifactId>amoro-hudi-format</artifactId>
+ <name>Amoro Hudi Format Integration</name>
+
+ <properties>
+ <maven.compiler.source>8</maven.compiler.source>
+ <maven.compiler.target>8</maven.compiler.target>
+ <project.build.sourceEncoding>UTF-8</project.build.sourceEncoding>
+ </properties>
+
+ <dependencies>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-core</artifactId>
+ </dependency>
+
+ <!-- apache hudi dependencies -->
+ <dependency>
+ <groupId>org.apache.hudi</groupId>
+ <artifactId>hudi-java-client</artifactId>
+ <exclusions>
+ <exclusion>
+ <groupId>org.apache.orc</groupId>
+ <artifactId>orc-core</artifactId>
+ </exclusion>
+ <exclusion>
+ <groupId>asm</groupId>
+ <artifactId>asm</artifactId>
+ </exclusion>
+ </exclusions>
+ </dependency>
+ </dependencies>
+
+</project>
\ No newline at end of file
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiCatalogFactory.java
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiCatalogFactory.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiCatalogFactory.java
rename to
amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiCatalogFactory.java
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiHadoopCatalog.java
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiHadoopCatalog.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiHadoopCatalog.java
rename to
amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiHadoopCatalog.java
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiHiveCatalog.java
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiHiveCatalog.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiHiveCatalog.java
rename to
amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiHiveCatalog.java
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiSnapshot.java
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiSnapshot.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiSnapshot.java
rename to
amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiSnapshot.java
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiTable.java
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTable.java
similarity index 90%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiTable.java
rename to
amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTable.java
index e87704d00..fc4bffb94 100644
--- a/amoro-core/src/main/java/org/apache/amoro/formats/hudi/HudiTable.java
+++
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTable.java
@@ -18,16 +18,6 @@
package org.apache.amoro.formats.hudi;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.CLEAN_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.COMMIT_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.COMPACTION_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.DELTA_COMMIT_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.INDEXING_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.LOG_COMPACTION_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.REPLACE_COMMIT_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.ROLLBACK_ACTION;
-import static
org.apache.hudi.common.table.timeline.HoodieTimeline.SAVEPOINT_ACTION;
-
import org.apache.amoro.AmoroTable;
import org.apache.amoro.TableFormat;
import org.apache.amoro.TableSnapshot;
@@ -131,18 +121,18 @@ public class HudiTable implements
AmoroTable<HoodieJavaTable> {
private static final Set<String> OPTIMIZING_INSTANT_TYPES =
Sets.newHashSet(
- CLEAN_ACTION,
- COMPACTION_ACTION,
- REPLACE_COMMIT_ACTION,
- INDEXING_ACTION,
- LOG_COMPACTION_ACTION);
+ HoodieTimeline.CLEAN_ACTION,
+ HoodieTimeline.COMPACTION_ACTION,
+ HoodieTimeline.REPLACE_COMMIT_ACTION,
+ HoodieTimeline.INDEXING_ACTION,
+ HoodieTimeline.LOG_COMPACTION_ACTION);
private static final Set<String> WRITE_INSTANT_TYPES =
Sets.newHashSet(
- COMMIT_ACTION,
- DELTA_COMMIT_ACTION,
- REPLACE_COMMIT_ACTION,
- SAVEPOINT_ACTION,
- ROLLBACK_ACTION);
+ HoodieTimeline.COMMIT_ACTION,
+ HoodieTimeline.DELTA_COMMIT_ACTION,
+ HoodieTimeline.REPLACE_COMMIT_ACTION,
+ HoodieTimeline.SAVEPOINT_ACTION,
+ HoodieTimeline.ROLLBACK_ACTION);
private static final Set<String> OPTIMIZING_HOODIE_OPERATION =
Sets.newHashSet(
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/HudiTableDescriptor.java
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTableDescriptor.java
similarity index 90%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/HudiTableDescriptor.java
rename to
amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTableDescriptor.java
index 50bc36a47..642a0714b 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/HudiTableDescriptor.java
+++
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTableDescriptor.java
@@ -16,36 +16,33 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard;
+package org.apache.amoro.formats.hudi;
import org.apache.amoro.AmoroTable;
import org.apache.amoro.TableFormat;
-import org.apache.amoro.data.DataFileType;
-import org.apache.amoro.formats.hudi.HudiTable;
-import org.apache.amoro.server.AmoroServiceConstants;
-import org.apache.amoro.server.dashboard.model.AMSColumnInfo;
-import org.apache.amoro.server.dashboard.model.AMSPartitionField;
-import org.apache.amoro.server.dashboard.model.AmoroSnapshotsOfTable;
-import org.apache.amoro.server.dashboard.model.ConsumerInfo;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
-import org.apache.amoro.server.dashboard.model.FilesStatistics;
-import org.apache.amoro.server.dashboard.model.OperationType;
-import org.apache.amoro.server.dashboard.model.OptimizingProcessInfo;
-import org.apache.amoro.server.dashboard.model.OptimizingTaskInfo;
-import org.apache.amoro.server.dashboard.model.PartitionBaseInfo;
-import org.apache.amoro.server.dashboard.model.PartitionFileBaseInfo;
-import org.apache.amoro.server.dashboard.model.ServerTableMeta;
-import org.apache.amoro.server.dashboard.model.TableSummary;
-import org.apache.amoro.server.dashboard.model.TagOrBranchInfo;
-import org.apache.amoro.server.dashboard.utils.AmsUtil;
-import org.apache.amoro.server.optimizing.OptimizingProcess;
-import org.apache.amoro.server.optimizing.OptimizingType;
-import org.apache.amoro.server.optimizing.TaskRuntime;
-import org.apache.amoro.server.utils.HudiTableUtil;
+import org.apache.amoro.process.ProcessStatus;
+import org.apache.amoro.process.ProcessTaskStatus;
import org.apache.amoro.shade.guava32.com.google.common.collect.Lists;
import org.apache.amoro.shade.guava32.com.google.common.collect.Maps;
import org.apache.amoro.shade.guava32.com.google.common.collect.Sets;
+import org.apache.amoro.table.descriptor.AMSColumnInfo;
+import org.apache.amoro.table.descriptor.AMSPartitionField;
+import org.apache.amoro.table.descriptor.AmoroSnapshotsOfTable;
+import org.apache.amoro.table.descriptor.ConsumerInfo;
+import org.apache.amoro.table.descriptor.DDLInfo;
+import org.apache.amoro.table.descriptor.FilesStatistics;
+import org.apache.amoro.table.descriptor.FormatTableDescriptor;
+import org.apache.amoro.table.descriptor.OperationType;
+import org.apache.amoro.table.descriptor.OptimizingProcessInfo;
+import org.apache.amoro.table.descriptor.OptimizingTaskInfo;
+import org.apache.amoro.table.descriptor.PartitionBaseInfo;
+import org.apache.amoro.table.descriptor.PartitionFileBaseInfo;
+import org.apache.amoro.table.descriptor.ServerTableMeta;
+import org.apache.amoro.table.descriptor.TableSummary;
+import org.apache.amoro.table.descriptor.TagOrBranchInfo;
+import org.apache.amoro.utils.CommonUtil;
import org.apache.avro.Schema;
+import org.apache.commons.lang3.tuple.Pair;
import org.apache.hudi.avro.model.HoodieClusteringGroup;
import org.apache.hudi.avro.model.HoodieClusteringPlan;
import org.apache.hudi.avro.model.HoodieClusteringStrategy;
@@ -74,7 +71,6 @@ import org.apache.hudi.common.util.Option;
import org.apache.hudi.common.util.StringUtils;
import org.apache.hudi.metadata.HoodieTableMetadata;
import org.apache.hudi.table.HoodieJavaTable;
-import org.apache.iceberg.util.Pair;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@@ -98,9 +94,10 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
private static final Logger LOG =
LoggerFactory.getLogger(HudiTableDescriptor.class);
- private final ExecutorService ioExecutors;
+ private ExecutorService ioExecutors;
- public HudiTableDescriptor(ExecutorService ioExecutor) {
+ @Override
+ public void withIoExecutor(ExecutorService ioExecutor) {
this.ioExecutors = ioExecutor;
}
@@ -188,29 +185,29 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
long totalFileCount = baseFileCount + logFileCount;
long totalFileSize = totalBaseSizeInByte + totalLogSizeInByte;
String averageFileSize =
- AmsUtil.byteToXB(totalFileCount == 0 ? 0 : totalFileSize /
totalFileCount);
+ CommonUtil.byteToXB(totalFileCount == 0 ? 0 : totalFileSize /
totalFileCount);
String tableType = metaClient.getTableType() ==
HoodieTableType.COPY_ON_WRITE ? "cow" : "mor";
String tableFormat = "Hudi(" + tableType + ")";
TableSummary tableSummary =
new TableSummary(
- totalFileCount, AmsUtil.byteToXB(totalFileSize), averageFileSize,
0, tableFormat);
+ totalFileCount, CommonUtil.byteToXB(totalFileSize),
averageFileSize, 0, tableFormat);
meta.setTableSummary(tableSummary);
Map<String, Object> baseSummary = new HashMap<>();
- baseSummary.put("totalSize", AmsUtil.byteToXB(totalBaseSizeInByte));
+ baseSummary.put("totalSize", CommonUtil.byteToXB(totalBaseSizeInByte));
baseSummary.put("fileCount", baseFileCount);
baseSummary.put(
"averageFileSize",
- AmsUtil.byteToXB(baseFileCount == 0 ? 0 : totalBaseSizeInByte /
baseFileCount));
+ CommonUtil.byteToXB(baseFileCount == 0 ? 0 : totalBaseSizeInByte /
baseFileCount));
meta.setBaseMetrics(baseSummary);
if (HoodieTableType.MERGE_ON_READ == metaClient.getTableType()) {
Map<String, Object> logSummary = new HashMap<>();
- logSummary.put("totalSize", AmsUtil.byteToXB(totalLogSizeInByte));
+ logSummary.put("totalSize", CommonUtil.byteToXB(totalLogSizeInByte));
logSummary.put("fileCount", logFileCount);
logSummary.put(
"averageFileSize",
- AmsUtil.byteToXB(logFileCount == 0 ? 0 : totalLogSizeInByte /
logFileCount));
+ CommonUtil.byteToXB(logFileCount == 0 ? 0 : totalLogSizeInByte /
logFileCount));
meta.setChangeMetrics(logSummary);
}
return meta;
@@ -309,7 +306,7 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
PartitionFileBaseInfo file =
new PartitionFileBaseInfo(
baseFile.getCommitTime(),
- DataFileType.BASE_FILE,
+ "BASE_FILE",
commitTime,
partition,
0,
@@ -323,13 +320,7 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
// TODO: can't get commit time from log file
PartitionFileBaseInfo file =
new PartitionFileBaseInfo(
- "",
- DataFileType.LOG_FILE,
- 0L,
- partition,
- 0,
- l.getPath().toString(),
- l.getFileSize());
+ "", "LOG_FILE", 0L, partition, 0,
l.getPath().toString(), l.getFileSize());
files.add(file);
});
return files.stream();
@@ -418,7 +409,7 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
long commitTimestamp =
parseHoodieCommitTime(commit.getStateTransitionTime());
processInfo.setDuration(commitTimestamp - startTime);
processInfo.setFinishTime(commitTimestamp);
- processInfo.setStatus(OptimizingProcess.Status.SUCCESS);
+ processInfo.setStatus(ProcessStatus.SUCCESS);
Option<byte[]> commitDetail = timeline.getInstantDetails(commit);
HoodieCommitMetadata commitMetadata =
HoodieCommitMetadata.fromBytes(commitDetail.get(),
HoodieCommitMetadata.class);
@@ -437,7 +428,7 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
HoodieInstant inf =
instantMap.get(instantTimestamp + "_" +
HoodieInstant.State.INFLIGHT.name());
if (inf != null) {
- processInfo.setStatus(OptimizingProcess.Status.RUNNING);
+ processInfo.setStatus(ProcessStatus.ACTIVE);
}
}
return processInfo;
@@ -464,7 +455,7 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
processInfo.getSummary().put("strategy",
strategy.getCompactorClassName());
processInfo.getSummary().putAll(strategy.getStrategyParams());
}
- processInfo.setOptimizingType(OptimizingType.MINOR);
+ processInfo.setOptimizingType("Compact");
}
private OptimizingProcessInfo fillClusterProcessInfo(
@@ -490,7 +481,7 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
processInfo.setInputFiles(FilesStatistics.build(inputFileCount,
inputFileSize));
int tasks = plan.getInputGroups().size();
processInfo.setTotalTasks(tasks);
- processInfo.setOptimizingType(OptimizingType.MAJOR);
+ processInfo.setOptimizingType("Cluster");
HoodieClusteringStrategy strategy = plan.getStrategy();
if (strategy != null) {
@@ -521,16 +512,16 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
return Lists.newArrayList();
}
long startTime = parseHoodieCommitTime(request.getStateTransitionTime());
- long endTime = AmoroServiceConstants.INVALID_TIME;
- long costTime = AmoroServiceConstants.INVALID_TIME;
- TaskRuntime.Status status = TaskRuntime.Status.ACKED;
+ long endTime = -1;
+ long costTime = -1;
+ ProcessTaskStatus status = ProcessTaskStatus.ACKED;
Option<byte[]> requestDetails = timeline.getInstantDetails(request);
if (!requestDetails.isPresent()) {
return Lists.newArrayList();
}
byte[] commitDetails = null;
if (complete != null) {
- status = TaskRuntime.Status.SUCCESS;
+ status = ProcessTaskStatus.ACKED;
endTime = parseHoodieCommitTime(complete.getStateTransitionTime());
costTime = endTime - startTime;
Option<byte[]> commitDetail = timeline.getInstantDetails(complete);
@@ -579,8 +570,8 @@ public class HudiTableDescriptor implements
FormatTableDescriptor {
List<OptimizingTaskInfo> results = Lists.newArrayList();
int taskId = 0;
for (String fileGroupId : inputFileStatistic.keySet()) {
- FilesStatistics input = inputFileStatistic.get(fileGroupId).second();
- String partition = inputFileStatistic.get(fileGroupId).first();
+ FilesStatistics input = inputFileStatistic.get(fileGroupId).getRight();
+ String partition = inputFileStatistic.get(fileGroupId).getLeft();
OptimizingTaskInfo task =
new OptimizingTaskInfo(
-1L,
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/utils/HudiTableUtil.java
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTableUtil.java
similarity index 99%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/utils/HudiTableUtil.java
rename to
amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTableUtil.java
index 6a3c93f7d..6541183d2 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/utils/HudiTableUtil.java
+++
b/amoro-hudi-format/src/main/java/org/apache/amoro/formats/hudi/HudiTableUtil.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.utils;
+package org.apache.amoro.formats.hudi;
import org.apache.avro.LogicalTypes;
import org.apache.avro.Schema;
diff --git
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
b/amoro-hudi-format/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
similarity index 88%
copy from
amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
copy to
amoro-hudi-format/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
index 72d74f17d..8c60e2607 100644
---
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
+++
b/amoro-hudi-format/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
@@ -16,5 +16,4 @@
# limitations under the License.
#
-org.apache.amoro.formats.paimon.PaimonCatalogFactory
org.apache.amoro.formats.hudi.HudiCatalogFactory
\ No newline at end of file
diff --git
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
b/amoro-hudi-format/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
similarity index 88%
copy from
amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
copy to
amoro-hudi-format/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
index 72d74f17d..fa2cb74a2 100644
---
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
+++
b/amoro-hudi-format/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
@@ -15,6 +15,4 @@
# See the License for the specific language governing permissions and
# limitations under the License.
#
-
-org.apache.amoro.formats.paimon.PaimonCatalogFactory
-org.apache.amoro.formats.hudi.HudiCatalogFactory
\ No newline at end of file
+org.apache.amoro.formats.paimon.PaimonTableDescriptor
\ No newline at end of file
diff --git
a/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/pom.xml
b/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/pom.xml
index 81a995458..1d6a2bc9f 100644
---
a/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/pom.xml
+++
b/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/pom.xml
@@ -334,6 +334,21 @@
<scope>test</scope>
</dependency>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ <version>${project.version}</version>
+ <classifier>tests</classifier>
+ <scope>test</scope>
+ </dependency>
+
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ <version>${project.version}</version>
+ <scope>test</scope>
+ </dependency>
+
<dependency>
<groupId>org.apache.flink</groupId>
<artifactId>flink-metrics-jmx</artifactId>
diff --git
a/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/src/test/java/org/apache/amoro/flink/catalog/FlinkAmoroCatalogITCase.java
b/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/src/test/java/org/apache/amoro/flink/catalog/FlinkAmoroCatalogITCase.java
index 0038cc282..92b2a286a 100644
---
a/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/src/test/java/org/apache/amoro/flink/catalog/FlinkAmoroCatalogITCase.java
+++
b/amoro-mixed-format/amoro-mixed-format-flink/amoro-mixed-format-flink-common/src/test/java/org/apache/amoro/flink/catalog/FlinkAmoroCatalogITCase.java
@@ -24,10 +24,10 @@ import static org.junit.jupiter.api.Assertions.assertTrue;
import org.apache.amoro.flink.table.AmoroCatalogITCaseBase;
import org.apache.amoro.formats.AmoroCatalogTestHelper;
-import org.apache.amoro.formats.PaimonHadoopCatalogTestHelper;
+import org.apache.amoro.formats.paimon.PaimonHadoopCatalogTestHelper;
+import org.apache.amoro.formats.paimon.PaimonHiveCatalogTestHelper;
import org.apache.amoro.formats.paimon.PaimonTable;
import org.apache.amoro.hive.TestHMS;
-import org.apache.amoro.hive.formats.PaimonHiveCatalogTestHelper;
import org.apache.flink.table.api.TableResult;
import org.apache.flink.table.catalog.AbstractCatalog;
import org.apache.flink.table.catalog.Catalog;
diff --git
a/amoro-mixed-format/amoro-mixed-format-spark/v3.2/amoro-mixed-format-spark-3.2/pom.xml
b/amoro-mixed-format/amoro-mixed-format-spark/v3.2/amoro-mixed-format-spark-3.2/pom.xml
index bcaab550b..16fcbcb3e 100644
---
a/amoro-mixed-format/amoro-mixed-format-spark/v3.2/amoro-mixed-format-spark-3.2/pom.xml
+++
b/amoro-mixed-format/amoro-mixed-format-spark/v3.2/amoro-mixed-format-spark-3.2/pom.xml
@@ -321,6 +321,13 @@
<type>test-jar</type>
</dependency>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ <version>${project.version}</version>
+ <scope>test</scope>
+ </dependency>
+
<dependency>
<groupId>org.apache.hive</groupId>
<artifactId>hive-metastore</artifactId>
diff --git
a/amoro-mixed-format/amoro-mixed-format-spark/v3.3/amoro-mixed-format-spark-3.3/pom.xml
b/amoro-mixed-format/amoro-mixed-format-spark/v3.3/amoro-mixed-format-spark-3.3/pom.xml
index e096ffb07..e36baa0e4 100644
---
a/amoro-mixed-format/amoro-mixed-format-spark/v3.3/amoro-mixed-format-spark-3.3/pom.xml
+++
b/amoro-mixed-format/amoro-mixed-format-spark/v3.3/amoro-mixed-format-spark-3.3/pom.xml
@@ -274,6 +274,12 @@
<version>${paimon.version}</version>
<scope>test</scope>
</dependency>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ <version>${parent.version}</version>
+ <scope>test</scope>
+ </dependency>
<dependency>
<groupId>org.junit.platform</groupId>
diff --git a/amoro-paimon-format/pom.xml b/amoro-paimon-format/pom.xml
new file mode 100644
index 000000000..0b7daf87d
--- /dev/null
+++ b/amoro-paimon-format/pom.xml
@@ -0,0 +1,60 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+ ~ 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.
+ -->
+<project xmlns="http://maven.apache.org/POM/4.0.0"
+ xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+ xsi:schemaLocation="http://maven.apache.org/POM/4.0.0
http://maven.apache.org/xsd/maven-4.0.0.xsd">
+ <modelVersion>4.0.0</modelVersion>
+ <parent>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-parent</artifactId>
+ <version>0.8-SNAPSHOT</version>
+ <relativePath>../pom.xml</relativePath>
+ </parent>
+
+ <artifactId>amoro-paimon-format</artifactId>
+ <name>Amoro Paimon Format Integration</name>
+
+ <dependencies>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-core</artifactId>
+ </dependency>
+
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-core</artifactId>
+ <version>${project.version}</version>
+ <classifier>tests</classifier>
+ <scope>test</scope>
+ </dependency>
+
+ <dependency>
+ <groupId>org.apache.paimon</groupId>
+ <artifactId>paimon-bundle</artifactId>
+ </dependency>
+
+ <dependency>
+ <groupId>org.apache.spark</groupId>
+ <artifactId>spark-sql_2.12</artifactId>
+ <version>3.3.2</version>
+ <scope>provided</scope>
+ </dependency>
+ </dependencies>
+
+</project>
\ No newline at end of file
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalog.java
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalog.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalog.java
rename to
amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalog.java
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalogFactory.java
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalogFactory.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalogFactory.java
rename to
amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonCatalogFactory.java
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonSnapshot.java
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonSnapshot.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonSnapshot.java
rename to
amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonSnapshot.java
diff --git
a/amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonTable.java
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTable.java
similarity index 100%
rename from
amoro-core/src/main/java/org/apache/amoro/formats/paimon/PaimonTable.java
rename to
amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTable.java
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/PaimonTableDescriptor.java
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTableDescriptor.java
similarity index 92%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/PaimonTableDescriptor.java
rename to
amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTableDescriptor.java
index 5b0d225b0..89f94b571 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/PaimonTableDescriptor.java
+++
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTableDescriptor.java
@@ -16,39 +16,36 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard;
+package org.apache.amoro.formats.paimon;
-import static org.apache.amoro.data.DataFileType.INSERT_FILE;
import static org.apache.paimon.operation.FileStoreScan.Plan.groupByPartFiles;
import org.apache.amoro.AmoroTable;
import org.apache.amoro.TableFormat;
import org.apache.amoro.api.CommitMetaProducer;
-import org.apache.amoro.data.DataFileType;
-import org.apache.amoro.server.dashboard.component.reverser.DDLReverser;
-import
org.apache.amoro.server.dashboard.component.reverser.PaimonTableMetaExtract;
-import org.apache.amoro.server.dashboard.model.AMSColumnInfo;
-import org.apache.amoro.server.dashboard.model.AMSPartitionField;
-import org.apache.amoro.server.dashboard.model.AmoroSnapshotsOfTable;
-import org.apache.amoro.server.dashboard.model.ConsumerInfo;
-import org.apache.amoro.server.dashboard.model.DDLInfo;
-import org.apache.amoro.server.dashboard.model.OperationType;
-import org.apache.amoro.server.dashboard.model.OptimizingProcessInfo;
-import org.apache.amoro.server.dashboard.model.OptimizingTaskInfo;
-import org.apache.amoro.server.dashboard.model.PartitionBaseInfo;
-import org.apache.amoro.server.dashboard.model.PartitionFileBaseInfo;
-import org.apache.amoro.server.dashboard.model.ServerTableMeta;
-import org.apache.amoro.server.dashboard.model.TableSummary;
-import org.apache.amoro.server.dashboard.model.TagOrBranchInfo;
-import org.apache.amoro.server.dashboard.utils.AmsUtil;
-import org.apache.amoro.server.dashboard.utils.FilesStatisticsBuilder;
-import org.apache.amoro.server.optimizing.OptimizingProcess;
-import org.apache.amoro.server.optimizing.OptimizingType;
+import org.apache.amoro.process.ProcessStatus;
import org.apache.amoro.shade.guava32.com.google.common.collect.ImmutableList;
import org.apache.amoro.shade.guava32.com.google.common.collect.Lists;
import org.apache.amoro.shade.guava32.com.google.common.collect.Streams;
import org.apache.amoro.table.TableIdentifier;
-import org.apache.iceberg.util.Pair;
+import org.apache.amoro.table.descriptor.AMSColumnInfo;
+import org.apache.amoro.table.descriptor.AMSPartitionField;
+import org.apache.amoro.table.descriptor.AmoroSnapshotsOfTable;
+import org.apache.amoro.table.descriptor.ConsumerInfo;
+import org.apache.amoro.table.descriptor.DDLInfo;
+import org.apache.amoro.table.descriptor.DDLReverser;
+import org.apache.amoro.table.descriptor.FilesStatisticsBuilder;
+import org.apache.amoro.table.descriptor.FormatTableDescriptor;
+import org.apache.amoro.table.descriptor.OperationType;
+import org.apache.amoro.table.descriptor.OptimizingProcessInfo;
+import org.apache.amoro.table.descriptor.OptimizingTaskInfo;
+import org.apache.amoro.table.descriptor.PartitionBaseInfo;
+import org.apache.amoro.table.descriptor.PartitionFileBaseInfo;
+import org.apache.amoro.table.descriptor.ServerTableMeta;
+import org.apache.amoro.table.descriptor.TableSummary;
+import org.apache.amoro.table.descriptor.TagOrBranchInfo;
+import org.apache.amoro.utils.CommonUtil;
+import org.apache.commons.lang3.tuple.Pair;
import org.apache.paimon.CoreOptions;
import org.apache.paimon.FileStore;
import org.apache.paimon.Snapshot;
@@ -90,10 +87,11 @@ public class PaimonTableDescriptor implements
FormatTableDescriptor {
public static final String PAIMON_MAIN_BRANCH_NAME = "main";
- private final ExecutorService executor;
+ private ExecutorService executor;
- public PaimonTableDescriptor(ExecutorService executor) {
- this.executor = executor;
+ @Override
+ public void withIoExecutor(ExecutorService ioExecutor) {
+ this.executor = ioExecutor;
}
@Override
@@ -145,10 +143,10 @@ public class PaimonTableDescriptor implements
FormatTableDescriptor {
AmoroSnapshotsOfTable snapshotsOfTable =
manifestListInfo(store, snapshot, (m, s) -> s.dataManifests(m));
long fileSize = snapshotsOfTable.getOriginalFileSize();
- String totalSize = AmsUtil.byteToXB(fileSize);
+ String totalSize = CommonUtil.byteToXB(fileSize);
int fileCount = snapshotsOfTable.getFileCount();
- String averageFileSize = AmsUtil.byteToXB(fileCount == 0 ? 0 : fileSize
/ fileCount);
+ String averageFileSize = CommonUtil.byteToXB(fileCount == 0 ? 0 :
fileSize / fileCount);
tableSummary =
new TableSummary(
@@ -241,7 +239,7 @@ public class PaimonTableDescriptor implements
FormatTableDescriptor {
amsDataFileInfos.add(
new PartitionFileBaseInfo(
null,
- DataFileType.BASE_FILE,
+ "BASE_FILE",
entry.file().creationTimeEpochMillis(),
partitionString(entry.partition(), entry.bucket(),
fileStorePathFactory),
fullFilePath(store, entry),
@@ -348,7 +346,7 @@ public class PaimonTableDescriptor implements
FormatTableDescriptor {
partitionFileBases.add(
new PartitionFileBaseInfo(
snapshotId == null ? null : snapshotId.toString(),
- INSERT_FILE,
+ "INSERT_FILE",
manifestEntry.file().creationTimeEpochMillis(),
partitionSt,
0,
@@ -388,7 +386,7 @@ public class PaimonTableDescriptor implements
FormatTableDescriptor {
optimizingProcessInfo.setCatalogName(tableIdentifier.getCatalog());
optimizingProcessInfo.setDbName(tableIdentifier.getDatabase());
optimizingProcessInfo.setTableName(tableIdentifier.getTableName());
-
optimizingProcessInfo.setStatus(OptimizingProcess.Status.SUCCESS);
+ optimizingProcessInfo.setStatus(ProcessStatus.SUCCESS);
optimizingProcessInfo.setFinishTime(s.timeMillis());
FilesStatisticsBuilder inputBuilder = new
FilesStatisticsBuilder();
FilesStatisticsBuilder outputBuilder = new
FilesStatisticsBuilder();
@@ -423,9 +421,9 @@ public class PaimonTableDescriptor implements
FormatTableDescriptor {
}
}
if (isPrimaryTable && hasMaxLevels) {
-
optimizingProcessInfo.setOptimizingType(OptimizingType.FULL);
+ optimizingProcessInfo.setOptimizingType("FULL");
} else {
-
optimizingProcessInfo.setOptimizingType(OptimizingType.MINOR);
+ optimizingProcessInfo.setOptimizingType("MINOR");
}
optimizingProcessInfo.setSuccessTasks(buckets.size());
optimizingProcessInfo.setTotalTasks(buckets.size());
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/PaimonTableMetaExtract.java
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTableMetaExtract.java
similarity index 97%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/PaimonTableMetaExtract.java
rename to
amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTableMetaExtract.java
index 13b1e6767..02c44f5a8 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/PaimonTableMetaExtract.java
+++
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTableMetaExtract.java
@@ -16,9 +16,10 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.component.reverser;
+package org.apache.amoro.formats.paimon;
import org.apache.amoro.shade.guava32.com.google.common.collect.Lists;
+import org.apache.amoro.table.descriptor.TableMetaExtract;
import org.apache.paimon.schema.SchemaManager;
import org.apache.paimon.schema.TableSchema;
import org.apache.paimon.table.DataTable;
diff --git
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/PaimonTypeToSparkType.java
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTypeToSparkType.java
similarity index 99%
rename from
amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/PaimonTypeToSparkType.java
rename to
amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTypeToSparkType.java
index 72ad832b5..106a4fcf2 100644
---
a/amoro-ams/amoro-ams-server/src/main/java/org/apache/amoro/server/dashboard/component/reverser/PaimonTypeToSparkType.java
+++
b/amoro-paimon-format/src/main/java/org/apache/amoro/formats/paimon/PaimonTypeToSparkType.java
@@ -16,7 +16,7 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard.component.reverser;
+package org.apache.amoro.formats.paimon;
import org.apache.paimon.types.ArrayType;
import org.apache.paimon.types.BigIntType;
diff --git
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
b/amoro-paimon-format/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
similarity index 88%
copy from
amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
copy to
amoro-paimon-format/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
index 72d74f17d..a22924a95 100644
---
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
+++
b/amoro-paimon-format/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
@@ -16,5 +16,4 @@
# limitations under the License.
#
-org.apache.amoro.formats.paimon.PaimonCatalogFactory
-org.apache.amoro.formats.hudi.HudiCatalogFactory
\ No newline at end of file
+org.apache.amoro.formats.paimon.PaimonCatalogFactory
\ No newline at end of file
diff --git
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
b/amoro-paimon-format/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
similarity index 88%
copy from
amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
copy to
amoro-paimon-format/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
index 72d74f17d..fa2cb74a2 100644
---
a/amoro-core/src/main/resources/META-INF/services/org.apache.amoro.FormatCatalogFactory
+++
b/amoro-paimon-format/src/main/resources/META-INF/services/org.apache.amoro.table.descriptor.FormatTableDescriptor
@@ -15,6 +15,4 @@
# See the License for the specific language governing permissions and
# limitations under the License.
#
-
-org.apache.amoro.formats.paimon.PaimonCatalogFactory
-org.apache.amoro.formats.hudi.HudiCatalogFactory
\ No newline at end of file
+org.apache.amoro.formats.paimon.PaimonTableDescriptor
\ No newline at end of file
diff --git
a/amoro-core/src/test/java/org/apache/amoro/formats/PaimonHadoopCatalogTestHelper.java
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/PaimonHadoopCatalogTestHelper.java
similarity index 97%
rename from
amoro-core/src/test/java/org/apache/amoro/formats/PaimonHadoopCatalogTestHelper.java
rename to
amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/PaimonHadoopCatalogTestHelper.java
index 11cf0c9d3..02a826a29 100644
---
a/amoro-core/src/test/java/org/apache/amoro/formats/PaimonHadoopCatalogTestHelper.java
+++
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/PaimonHadoopCatalogTestHelper.java
@@ -16,12 +16,12 @@
* limitations under the License.
*/
-package org.apache.amoro.formats;
+package org.apache.amoro.formats.paimon;
import org.apache.amoro.AlreadyExistsException;
import org.apache.amoro.AmoroCatalog;
import org.apache.amoro.TableFormat;
-import org.apache.amoro.formats.paimon.PaimonCatalogFactory;
+import org.apache.amoro.formats.AbstractFormatCatalogTestHelper;
import org.apache.amoro.table.TableMetaStore;
import org.apache.amoro.utils.CatalogUtil;
import org.apache.hadoop.conf.Configuration;
diff --git
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/formats/PaimonHiveCatalogTestHelper.java
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/PaimonHiveCatalogTestHelper.java
similarity index 94%
rename from
amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/formats/PaimonHiveCatalogTestHelper.java
rename to
amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/PaimonHiveCatalogTestHelper.java
index b77cec571..190bbd81e 100644
---
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/formats/PaimonHiveCatalogTestHelper.java
+++
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/PaimonHiveCatalogTestHelper.java
@@ -16,9 +16,8 @@
* limitations under the License.
*/
-package org.apache.amoro.hive.formats;
+package org.apache.amoro.formats.paimon;
-import org.apache.amoro.formats.PaimonHadoopCatalogTestHelper;
import org.apache.amoro.properties.CatalogMetaProperties;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hive.conf.HiveConf;
diff --git
a/amoro-core/src/test/java/org/apache/amoro/formats/TestPaimonAmoroCatalog.java
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonAmoroCatalog.java
similarity index 94%
rename from
amoro-core/src/test/java/org/apache/amoro/formats/TestPaimonAmoroCatalog.java
rename to
amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonAmoroCatalog.java
index 23936ce1d..7cabedf35 100644
---
a/amoro-core/src/test/java/org/apache/amoro/formats/TestPaimonAmoroCatalog.java
+++
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonAmoroCatalog.java
@@ -16,8 +16,10 @@
* limitations under the License.
*/
-package org.apache.amoro.formats;
+package org.apache.amoro.formats.paimon;
+import org.apache.amoro.formats.AmoroCatalogTestHelper;
+import org.apache.amoro.formats.TestAmoroCatalogBase;
import org.apache.paimon.catalog.Catalog;
import org.apache.paimon.catalog.Identifier;
import org.apache.paimon.schema.Schema;
diff --git
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/formats/TestPaimonHiveAmoroCatalog.java
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonHiveAmoroCatalog.java
similarity index 94%
rename from
amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/formats/TestPaimonHiveAmoroCatalog.java
rename to
amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonHiveAmoroCatalog.java
index 1882c834a..c64bea1c0 100644
---
a/amoro-mixed-format/amoro-mixed-format-hive/src/test/java/org/apache/amoro/hive/formats/TestPaimonHiveAmoroCatalog.java
+++
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonHiveAmoroCatalog.java
@@ -16,10 +16,9 @@
* limitations under the License.
*/
-package org.apache.amoro.hive.formats;
+package org.apache.amoro.formats.paimon;
import org.apache.amoro.formats.AmoroCatalogTestHelper;
-import org.apache.amoro.formats.TestPaimonAmoroCatalog;
import org.apache.amoro.hive.TestHMS;
import org.junit.ClassRule;
import org.junit.runner.RunWith;
diff --git
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestPaimonServerTableDescriptor.java
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonServerTableDescriptor.java
similarity index 76%
rename from
amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestPaimonServerTableDescriptor.java
rename to
amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonServerTableDescriptor.java
index 7da1aeb5b..10eec7c10 100644
---
a/amoro-ams/amoro-ams-server/src/test/java/org/apache/amoro/server/dashboard/TestPaimonServerTableDescriptor.java
+++
b/amoro-paimon-format/src/test/java/org/apache/amoro/formats/paimon/TestPaimonServerTableDescriptor.java
@@ -16,11 +16,11 @@
* limitations under the License.
*/
-package org.apache.amoro.server.dashboard;
+package org.apache.amoro.formats.paimon;
import org.apache.amoro.formats.AmoroCatalogTestHelper;
-import org.apache.amoro.formats.PaimonHadoopCatalogTestHelper;
-import org.apache.amoro.hive.formats.PaimonHiveCatalogTestHelper;
+import org.apache.amoro.table.descriptor.FormatTableDescriptor;
+import org.apache.amoro.table.descriptor.TestServerTableDescriptor;
import org.apache.paimon.catalog.Catalog;
import org.apache.paimon.catalog.Identifier;
import org.apache.paimon.schema.SchemaChange;
@@ -47,7 +47,8 @@ public class TestPaimonServerTableDescriptor extends
TestServerTableDescriptor {
try {
getCatalog()
.alterTable(
- Identifier.create(TEST_DB, TEST_TABLE),
+ Identifier.create(
+ TestServerTableDescriptor.TEST_DB,
TestServerTableDescriptor.TEST_TABLE),
SchemaChange.addColumn("new_col", DataTypes.INT()),
false);
} catch (Exception e) {
@@ -60,7 +61,8 @@ public class TestPaimonServerTableDescriptor extends
TestServerTableDescriptor {
try {
getCatalog()
.alterTable(
- Identifier.create(TEST_DB, TEST_TABLE),
+ Identifier.create(
+ TestServerTableDescriptor.TEST_DB,
TestServerTableDescriptor.TEST_TABLE),
SchemaChange.renameColumn("new_col", "renamed_col"),
false);
} catch (Exception e) {
@@ -73,7 +75,8 @@ public class TestPaimonServerTableDescriptor extends
TestServerTableDescriptor {
try {
getCatalog()
.alterTable(
- Identifier.create(TEST_DB, TEST_TABLE),
+ Identifier.create(
+ TestServerTableDescriptor.TEST_DB,
TestServerTableDescriptor.TEST_TABLE),
SchemaChange.updateColumnType("renamed_col", DataTypes.BIGINT()),
false);
} catch (Exception e) {
@@ -86,7 +89,8 @@ public class TestPaimonServerTableDescriptor extends
TestServerTableDescriptor {
try {
getCatalog()
.alterTable(
- Identifier.create(TEST_DB, TEST_TABLE),
+ Identifier.create(
+ TestServerTableDescriptor.TEST_DB,
TestServerTableDescriptor.TEST_TABLE),
SchemaChange.updateColumnComment("renamed_col", "new comment"),
false);
} catch (Exception e) {
@@ -99,7 +103,8 @@ public class TestPaimonServerTableDescriptor extends
TestServerTableDescriptor {
try {
getCatalog()
.alterTable(
- Identifier.create(TEST_DB, TEST_TABLE),
+ Identifier.create(
+ TestServerTableDescriptor.TEST_DB,
TestServerTableDescriptor.TEST_TABLE),
SchemaChange.updateColumnNullability("renamed_col", false),
false);
} catch (Exception e) {
@@ -112,7 +117,8 @@ public class TestPaimonServerTableDescriptor extends
TestServerTableDescriptor {
try {
getCatalog()
.alterTable(
- Identifier.create(TEST_DB, TEST_TABLE),
+ Identifier.create(
+ TestServerTableDescriptor.TEST_DB,
TestServerTableDescriptor.TEST_TABLE),
SchemaChange.dropColumn("renamed_col"),
false);
} catch (Exception e) {
@@ -120,6 +126,11 @@ public class TestPaimonServerTableDescriptor extends
TestServerTableDescriptor {
}
}
+ @Override
+ protected FormatTableDescriptor getTableDescriptor() {
+ return new PaimonTableDescriptor();
+ }
+
private Catalog getCatalog() {
return (Catalog) getOriginalCatalog();
}
diff --git a/pom.xml b/pom.xml
index b8cce93a1..873917eff 100644
--- a/pom.xml
+++ b/pom.xml
@@ -50,6 +50,8 @@
<module>amoro-ams</module>
<module>amoro-iceberg-format</module>
<module>amoro-mixed-format</module>
+ <module>amoro-hudi-format</module>
+ <module>amoro-paimon-format</module>
</modules>
<scm>
@@ -247,6 +249,18 @@
<version>${project.version}</version>
</dependency>
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-paimon-format</artifactId>
+ <version>${project.version}</version>
+ </dependency>
+
+ <dependency>
+ <groupId>org.apache.amoro</groupId>
+ <artifactId>amoro-hudi-format</artifactId>
+ <version>${project.version}</version>
+ </dependency>
+
<dependency>
<groupId>org.apache.amoro</groupId>
<artifactId>amoro-mixed-format-hive</artifactId>