Skip to content
New issue

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

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

Already on GitHub? Sign in to your account

[Enhancement] desc iceberg table with transform type (backport #49094) #49594

Merged
merged 3 commits into from
Aug 8, 2024
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
14 changes: 6 additions & 8 deletions fe/fe-core/src/main/java/com/starrocks/catalog/IcebergTable.java
Original file line number Diff line number Diff line change
Expand Up @@ -45,6 +45,7 @@
import com.starrocks.thrift.TTableType;
import org.apache.iceberg.BaseTable;
import org.apache.iceberg.PartitionField;
import org.apache.iceberg.PartitionSpec;
import org.apache.iceberg.Schema;
import org.apache.iceberg.Snapshot;
import org.apache.iceberg.SortField;
Expand Down Expand Up @@ -283,6 +284,11 @@ public List<String> getPartitionColumnNames() {
.collect(Collectors.toList());
}

public List<String> getPartitionColumnNamesWithTransform() {
PartitionSpec partitionSpec = getNativeTable().spec();
return IcebergApiConverter.toPartitionFields(partitionSpec);
}

@Override
public String getTableIdentifier() {
String uuid = ((BaseTable) getNativeTable()).operations().current().uuid();
Expand Down Expand Up @@ -319,14 +325,6 @@ public org.apache.iceberg.Table getNativeTable() {
return nativeTable;
}

public long getRefreshSnapshotTime() {
return refreshSnapshotTime;
}

public void setRefreshSnapshotTime(long refreshSnapshotTime) {
this.refreshSnapshotTime = refreshSnapshotTime;
}

public void setIdentifierFieldIds(Set<Integer> identifierFieldIds) {
this.identifierFieldIds = identifierFieldIds;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -33,6 +33,7 @@
import org.apache.commons.lang3.StringUtils;
import org.apache.iceberg.FileFormat;
import org.apache.iceberg.Metrics;
import org.apache.iceberg.PartitionField;
import org.apache.iceberg.PartitionSpec;
import org.apache.iceberg.Schema;
import org.apache.iceberg.Table;
Expand All @@ -50,16 +51,22 @@
import java.util.Locale;
import java.util.Map;
import java.util.concurrent.atomic.AtomicInteger;
import java.util.regex.Matcher;
import java.util.regex.Pattern;

import static com.google.common.collect.ImmutableList.toImmutableList;
import static com.starrocks.analysis.OutFileClause.PARQUET_COMPRESSION_TYPE_MAP;
import static com.starrocks.connector.ColumnTypeConverter.fromIcebergType;
import static com.starrocks.connector.ConnectorTableId.CONNECTOR_ID_GENERATOR;
import static com.starrocks.connector.iceberg.IcebergConnector.ICEBERG_CATALOG_TYPE;
import static com.starrocks.server.CatalogMgr.ResourceMappingCatalog.toResourceName;
import static java.lang.String.format;

public class IcebergApiConverter {
private static final Logger LOG = LogManager.getLogger(IcebergApiConverter.class);
public static final String PARTITION_NULL_VALUE = "null";
private static final Pattern ICEBERG_BUCKET_PATTERN = Pattern.compile("bucket\\[(\\d+)]");
private static final Pattern ICEBERG_TRUNCATE_PATTERN = Pattern.compile("truncate\\[(\\d+)]");
private static final int FAKE_FIELD_ID = -1;

public static IcebergTable toIcebergTable(Table nativeTbl, String catalogName, String remoteDbName,
Expand Down Expand Up @@ -293,4 +300,38 @@ public static Map<String, String> rebuildCreateTableProperties(Map<String, Strin

return tableProperties.build();
}

public static List<String> toPartitionFields(PartitionSpec spec) {
return spec.fields().stream()
.map(field -> toPartitionField(spec, field))
.collect(toImmutableList());
}

private static String toPartitionField(PartitionSpec spec, PartitionField field) {
String name = spec.schema().findColumnName(field.sourceId());
String transform = field.transform().toString();

switch (transform) {
case "identity":
return name;
case "year":
case "month":
case "day":
case "hour":
case "void":
return format("%s(%s)", transform, name);
}

Matcher matcher = ICEBERG_BUCKET_PATTERN.matcher(transform);
if (matcher.matches()) {
return format("bucket(%s, %s)", name, matcher.group(1));
}

matcher = ICEBERG_TRUNCATE_PATTERN.matcher(transform);
if (matcher.matches()) {
return format("truncate(%s, %s)", name, matcher.group(1));
}

throw new StarRocksConnectorException("Unsupported partition transform: " + field);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -57,6 +57,7 @@
import com.starrocks.catalog.HiveMetaStoreTable;
import com.starrocks.catalog.HiveTable;
import com.starrocks.catalog.HiveView;
import com.starrocks.catalog.IcebergTable;
import com.starrocks.catalog.Table;
import com.starrocks.common.Pair;
import com.starrocks.common.util.ParseUtil;
Expand Down Expand Up @@ -1351,10 +1352,17 @@ public static String getExternalCatalogTableDdlStmt(Table table) {
.append("\n)");

// Partition column names
List<String> partitionNames;
if (table.getType() != JDBC && !table.isUnPartitioned()) {
createTableSql.append("\nPARTITION BY ( ")
.append(String.join(", ", table.getPartitionColumnNames()))
.append(" )");
createTableSql.append("\nPARTITION BY (");

if (!table.isIcebergTable()) {
partitionNames = table.getPartitionColumnNames();
} else {
partitionNames = ((IcebergTable) table).getPartitionColumnNamesWithTransform();
}

createTableSql.append(String.join(", ", partitionNames)).append(")");
}

// Location
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -263,7 +263,7 @@ public void testShowCreateHiveTbl() {
" `col2` int(11) DEFAULT NULL,\n" +
" `col1` int(11) DEFAULT NULL\n" +
")\n" +
"PARTITION BY ( col1 )\n" +
"PARTITION BY (col1)\n" +
"PROPERTIES (\"location\" = \"hdfs://127.0.0.1:10000/hive\");",
AstToStringBuilder.getExternalCatalogTableDdlStmt(hiveTable));
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1093,7 +1093,7 @@ public Table getTable(String catalogName, String dbName, String tblName) {
" `year` int(11) DEFAULT NULL,\n" +
" `dt` int(11) DEFAULT NULL\n" +
")\n" +
"PARTITION BY ( year, dt )\n" +
"PARTITION BY (year, dt)\n" +
"PROPERTIES (\"location\" = \"hdfs://hadoop/hive/warehouse/test.db/test\");", resultSet.getResultRows().get(0).get(1));
}

Expand Down
21 changes: 21 additions & 0 deletions test/sql/test_iceberg/R/test_iceberg_show_stmt
Original file line number Diff line number Diff line change
@@ -0,0 +1,21 @@
-- name: test_iceberg_show_stmt
create external catalog iceberg_sql_test_${uuid0} PROPERTIES ("type"="iceberg", "iceberg.catalog.type"="hive", "iceberg.catalog.hive.metastore.uris"="${iceberg_catalog_hive_metastore_uris}","enable_iceberg_metadata_cache"="true","aws.s3.access_key" = "${oss_ak}","aws.s3.secret_key" = "${oss_sk}","aws.s3.endpoint" = "${oss_endpoint}");
-- result:
-- !result
show create table iceberg_sql_test_${uuid0}.iceberg_ci_db.partition_transform_table;
-- result:
partition_transform_table CREATE TABLE `partition_transform_table` (
`k1` int(11) DEFAULT NULL,
`t1` datetime DEFAULT NULL,
`t2` datetime DEFAULT NULL,
`t3` datetime DEFAULT NULL,
`t4` datetime DEFAULT NULL,
`p1` varchar(1073741824) DEFAULT NULL,
`p2` varchar(1073741824) DEFAULT NULL
)
PARTITION BY (year(t1), month(t2), day(t3), hour(t4), truncate(p1, 5), bucket(p2, 3))
PROPERTIES ("location" = "oss://starrocks-ci-test/iceberg_ci_db/partition_transform_table");
-- !result
drop catalog iceberg_sql_test_${uuid0};
-- result:
-- !result
7 changes: 7 additions & 0 deletions test/sql/test_iceberg/T/test_iceberg_show_stmt
Original file line number Diff line number Diff line change
@@ -0,0 +1,7 @@
-- name: test_iceberg_show_stmt

create external catalog iceberg_sql_test_${uuid0} PROPERTIES ("type"="iceberg", "iceberg.catalog.type"="hive", "iceberg.catalog.hive.metastore.uris"="${iceberg_catalog_hive_metastore_uris}","enable_iceberg_metadata_cache"="true","aws.s3.access_key" = "${oss_ak}","aws.s3.secret_key" = "${oss_sk}","aws.s3.endpoint" = "${oss_endpoint}");

show create table iceberg_sql_test_${uuid0}.iceberg_ci_db.partition_transform_table;

drop catalog iceberg_sql_test_${uuid0};
Loading