diff --git a/docs/content/flink/procedures.md b/docs/content/flink/procedures.md index 4f7a4eb1511e..6c66f54114fe 100644 --- a/docs/content/flink/procedures.md +++ b/docs/content/flink/procedures.md @@ -185,21 +185,6 @@ All available procedures are listed below. merge_into - -- when matched then upsert
- CALL [catalog.]sys.merge_into('identifier','targetAlias',
- 'sourceSqls','sourceTable','mergeCondition',
- 'matchedUpsertCondition','matchedUpsertSetting')

- -- when matched then upsert; when not matched then insert
- CALL [catalog.]sys.merge_into('identifier','targetAlias',
- 'sourceSqls','sourceTable','mergeCondition',
- 'matchedUpsertCondition','matchedUpsertSetting',
- 'notMatchedInsertCondition','notMatchedInsertValues')

- -- when matched then delete
- CALL [catalog].sys.merge_into('identifier','targetAlias',
- 'sourceSqls','sourceTable','mergeCondition',
- 'matchedDeleteCondition')

- -- when matched then upsert + delete;
- -- when not matched then insert
CALL [catalog].sys.merge_into('identifier','targetAlias',
'sourceSqls','sourceTable','mergeCondition',
'matchedUpsertCondition','matchedUpsertSetting',
@@ -216,7 +201,12 @@ All available procedures are listed below. -- and if there is no match,
-- insert the order from
-- the source table
- CALL sys.merge_into('default.T', '', '', 'default.S', 'T.id=S.order_id', '', 'price=T.price+20', '', '*') + CALL sys.merge_into(
+ target_table => 'default.T',
+ source_table => 'default.S',
+ merge_condition => 'T.id=S.order_id',
+ matched_upsert_setting => 'price=T.price+20',
+ not_matched_insert_values => '*')

@@ -259,9 +249,9 @@ All available procedures are listed below. rollback_to -- rollback to a snapshot
- CALL sys.rollback_to('identifier', snapshotId)

+ CALL sys.rollback_to(`table` => 'identifier', snapshot_id => snapshotId)

-- rollback to a tag
- CALL sys.rollback_to('identifier', 'tagName') + CALL sys.rollback_to(`table` => 'identifier', tag => 'tagName') To rollback to a specific version of target table. Argument: @@ -269,7 +259,7 @@ All available procedures are listed below.
  • snapshotId (Long): id of the snapshot that will roll back to.
  • tagName: name of the tag that will roll back to.
  • - CALL sys.rollback_to('default.T', 10) + CALL sys.rollback_to(`table` => 'default.T', snapshot_id => 10) expire_snapshots diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CompactDatabaseProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CompactDatabaseProcedure.java new file mode 100644 index 000000000000..75511894149d --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CompactDatabaseProcedure.java @@ -0,0 +1,143 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.flink.action.CompactDatabaseAction; +import org.apache.paimon.utils.StringUtils; +import org.apache.paimon.utils.TimeUtils; + +import org.apache.flink.table.procedure.ProcedureContext; + +import java.util.Map; + +import static org.apache.paimon.utils.ParameterUtils.parseCommaSeparatedKeyValues; + +/** + * Compact database procedure. Usage: + * + *
    
    + *  -- NOTE: use '' as placeholder for optional arguments
    + *
    + *  -- compact all databases
    + *  CALL sys.compact_database()
    + *
    + *  -- compact some databases (accept regular expression)
    + *  CALL sys.compact_database('includingDatabases')
    + *
    + *  -- set compact mode
    + *  CALL sys.compact_database('includingDatabases', 'mode')
    + *
    + *  -- compact some tables (accept regular expression)
    + *  CALL sys.compact_database('includingDatabases', 'mode', 'includingTables')
    + *
    + *  -- exclude some tables (accept regular expression)
    + *  CALL sys.compact_database('includingDatabases', 'mode', 'includingTables', 'excludingTables')
    + *
    + *  -- set table options ('k=v,...')
    + *  CALL sys.compact_database('includingDatabases', 'mode', 'includingTables', 'excludingTables', 'tableOptions')
    + * 
    + */ +public class CompactDatabaseProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "compact_database"; + + public String[] call(ProcedureContext procedureContext) throws Exception { + return call(procedureContext, ""); + } + + public String[] call(ProcedureContext procedureContext, String includingDatabases) + throws Exception { + return call(procedureContext, includingDatabases, ""); + } + + public String[] call(ProcedureContext procedureContext, String includingDatabases, String mode) + throws Exception { + return call(procedureContext, includingDatabases, mode, ""); + } + + public String[] call( + ProcedureContext procedureContext, + String includingDatabases, + String mode, + String includingTables) + throws Exception { + return call(procedureContext, includingDatabases, mode, includingTables, ""); + } + + public String[] call( + ProcedureContext procedureContext, + String includingDatabases, + String mode, + String includingTables, + String excludingTables) + throws Exception { + return call( + procedureContext, includingDatabases, mode, includingTables, excludingTables, ""); + } + + public String[] call( + ProcedureContext procedureContext, + String includingDatabases, + String mode, + String includingTables, + String excludingTables, + String tableOptions) + throws Exception { + return call( + procedureContext, + includingDatabases, + mode, + includingTables, + excludingTables, + tableOptions, + ""); + } + + public String[] call( + ProcedureContext procedureContext, + String includingDatabases, + String mode, + String includingTables, + String excludingTables, + String tableOptions, + String partitionIdleTime) + throws Exception { + String warehouse = catalog.warehouse(); + Map catalogOptions = catalog.options(); + CompactDatabaseAction action = + new CompactDatabaseAction(warehouse, catalogOptions) + .includingDatabases(nullable(includingDatabases)) + .includingTables(nullable(includingTables)) + .excludingTables(nullable(excludingTables)) + .withDatabaseCompactMode(nullable(mode)); + if (!StringUtils.isBlank(tableOptions)) { + action.withTableOptions(parseCommaSeparatedKeyValues(tableOptions)); + } + if (!StringUtils.isBlank(partitionIdleTime)) { + action.withPartitionIdleTime(TimeUtils.parseDuration(partitionIdleTime)); + } + + return execute(procedureContext, action, "Compact database job"); + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CreateBranchProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CreateBranchProcedure.java new file mode 100644 index 000000000000..093505923fd6 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CreateBranchProcedure.java @@ -0,0 +1,65 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Catalog; +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.table.Table; + +import org.apache.commons.lang3.StringUtils; +import org.apache.flink.table.procedure.ProcedureContext; + +/** + * Create branch procedure for given tag. Usage: + * + *
    
    + *  CALL sys.create_branch('tableId', 'branchName', 'tagName')
    + * 
    + */ +public class CreateBranchProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "create_branch"; + + @Override + public String identifier() { + return IDENTIFIER; + } + + public String[] call( + ProcedureContext procedureContext, String tableId, String branchName, String tagName) + throws Catalog.TableNotExistException { + return innerCall(tableId, branchName, tagName); + } + + public String[] call(ProcedureContext procedureContext, String tableId, String branchName) + throws Catalog.TableNotExistException { + return innerCall(tableId, branchName, null); + } + + private String[] innerCall(String tableId, String branchName, String tagName) + throws Catalog.TableNotExistException { + Table table = catalog.getTable(Identifier.fromString(tableId)); + if (!StringUtils.isBlank(tagName)) { + table.createBranch(branchName, tagName); + } else { + table.createBranch(branchName); + } + return new String[] {"Success"}; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CreateTagProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CreateTagProcedure.java new file mode 100644 index 000000000000..1a7b03ef6512 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/CreateTagProcedure.java @@ -0,0 +1,98 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Catalog; +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.table.Table; +import org.apache.paimon.utils.TimeUtils; + +import org.apache.flink.table.procedure.ProcedureContext; + +import javax.annotation.Nullable; + +import java.time.Duration; + +/** + * Create tag procedure. Usage: + * + *
    
    + *  CALL sys.create_tag('tableId', 'tagName', snapshotId, 'timeRetained')
    + * 
    + */ +public class CreateTagProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "create_tag"; + + public String[] call( + ProcedureContext procedureContext, String tableId, String tagName, long snapshotId) + throws Catalog.TableNotExistException { + return innerCall(tableId, tagName, snapshotId, null); + } + + public String[] call(ProcedureContext procedureContext, String tableId, String tagName) + throws Catalog.TableNotExistException { + return innerCall(tableId, tagName, null, null); + } + + public String[] call( + ProcedureContext procedureContext, + String tableId, + String tagName, + long snapshotId, + String timeRetained) + throws Catalog.TableNotExistException { + return innerCall(tableId, tagName, snapshotId, timeRetained); + } + + public String[] call( + ProcedureContext procedureContext, String tableId, String tagName, String timeRetained) + throws Catalog.TableNotExistException { + return innerCall(tableId, tagName, null, timeRetained); + } + + private String[] innerCall( + String tableId, + String tagName, + @Nullable Long snapshotId, + @Nullable String timeRetained) + throws Catalog.TableNotExistException { + Table table = catalog.getTable(Identifier.fromString(tableId)); + if (snapshotId == null) { + table.createTag(tagName, toDuration(timeRetained)); + } else { + table.createTag(tagName, snapshotId, toDuration(timeRetained)); + } + return new String[] {"Success"}; + } + + @Nullable + private static Duration toDuration(@Nullable String s) { + if (s == null) { + return null; + } + + return TimeUtils.parseDuration(s); + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MarkPartitionDoneProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MarkPartitionDoneProcedure.java new file mode 100644 index 000000000000..d70cccf6ba25 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MarkPartitionDoneProcedure.java @@ -0,0 +1,84 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.CoreOptions; +import org.apache.paimon.catalog.Catalog; +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.partition.actions.PartitionMarkDoneAction; +import org.apache.paimon.table.FileStoreTable; +import org.apache.paimon.table.Table; +import org.apache.paimon.utils.IOUtils; +import org.apache.paimon.utils.PartitionPathUtils; + +import org.apache.flink.table.procedure.ProcedureContext; + +import java.io.IOException; +import java.util.List; + +import static org.apache.paimon.flink.sink.partition.PartitionMarkDone.markDone; +import static org.apache.paimon.utils.ParameterUtils.getPartitions; +import static org.apache.paimon.utils.Preconditions.checkArgument; + +/** + * Partition mark done procedure. Usage: + * + *
    
    + *  CALL sys.mark_partition_done('tableId', 'partition1', 'partition2', ...)
    + * 
    + */ +public class MarkPartitionDoneProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "mark_partition_done"; + + public String[] call( + ProcedureContext procedureContext, String tableId, String... partitionStrings) + throws Catalog.TableNotExistException, IOException { + checkArgument( + partitionStrings.length > 0, + "mark_partition_done procedure must specify partitions."); + + Identifier identifier = Identifier.fromString(tableId); + Table table = catalog.getTable(identifier); + checkArgument( + table instanceof FileStoreTable, + "Only FileStoreTable supports mark_partition_done procedure. The table type is '%s'.", + table.getClass().getName()); + + FileStoreTable fileStoreTable = (FileStoreTable) table; + CoreOptions coreOptions = fileStoreTable.coreOptions(); + List actions = + PartitionMarkDoneAction.createActions(fileStoreTable, coreOptions); + + List partitionPaths = + PartitionPathUtils.generatePartitionPaths( + getPartitions(partitionStrings), fileStoreTable.store().partitionType()); + + markDone(partitionPaths, actions); + + IOUtils.closeAllQuietly(actions); + + return new String[] {"Success"}; + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MergeIntoProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MergeIntoProcedure.java new file mode 100644 index 000000000000..acda2afd2e69 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MergeIntoProcedure.java @@ -0,0 +1,236 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.flink.action.MergeIntoAction; + +import org.apache.flink.core.execution.JobClient; +import org.apache.flink.streaming.api.datastream.DataStream; +import org.apache.flink.table.api.TableResult; +import org.apache.flink.table.data.RowData; +import org.apache.flink.table.procedure.ProcedureContext; + +import java.util.Map; + +import static org.apache.paimon.utils.Preconditions.checkArgument; +import static org.apache.paimon.utils.Preconditions.checkNotNull; + +/** + * Merge Into procedure. Usage: + * + *
    
    + *  -- NOTE: use '' as placeholder for optional arguments
    + *
    + *  -- when matched then upsert
    + *  CALL sys.merge_into(
    + *      'targetTableId',
    + *      'targetAlias',
    + *      'sourceSqls', -- separate with ';'
    + *      'sourceTable',
    + *      'mergeCondition',
    + *      'matchedUpsertCondition',
    + *      'matchedUpsertSetting'
    + *  )
    + *
    + *  -- when matched then upsert + when not matched then insert
    + *  CALL sys.merge_into(
    + *      'targetTableId'
    + *      'targetAlias',
    + *      'sourceSqls',
    + *      'sourceTable',
    + *      'mergeCondition',
    + *      'matchedUpsertCondition',
    + *      'matchedUpsertSetting',
    + *      'notMatchedInsertCondition',
    + *      'notMatchedInsertValues'
    + *  )
    + *
    + *  -- above + when matched then delete
    + *  -- IMPORTANT: Use 'TRUE' if you want to delete data without filter condition.
    + *  -- If matchedDeleteCondition='', it will ignore matched-delete action!
    + *  CALL sys.merge_into(
    + *      'targetTableId',
    + *      'targetAlias',
    + *      'sourceSqls',
    + *      'sourceTable',
    + *      'mergeCondition',
    + *      'matchedUpsertCondition',
    + *      'matchedUpsertSetting',
    + *      'notMatchedInsertCondition',
    + *      'notMatchedInsertValues',
    + *      'matchedDeleteCondition'
    + *  )
    + *
    + *  -- when matched then delete (short form)
    + *  CALL sys.merge_into(
    + *      'targetTableId'
    + *      'targetAlias',
    + *      'sourceSqls',
    + *      'sourceTable',
    + *      'mergeCondition',
    + *      'matchedDeleteCondition'
    + *  )
    + * 
    + * + *

    This procedure will be forced to use batch environments. Compared to {@link MergeIntoAction}, + * this procedure doesn't provide arguments to control not-matched-by-source behavior because they + * are not commonly used and will make the methods too complex to use. + */ +public class MergeIntoProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "merge_into"; + + public String[] call( + ProcedureContext procedureContext, + String targetTableId, + String targetAlias, + String sourceSqls, + String sourceTable, + String mergeCondition, + String matchedUpsertCondition, + String matchedUpsertSetting) { + return call( + procedureContext, + targetTableId, + targetAlias, + sourceSqls, + sourceTable, + mergeCondition, + matchedUpsertCondition, + matchedUpsertSetting, + "", + "", + ""); + } + + public String[] call( + ProcedureContext procedureContext, + String targetTableId, + String targetAlias, + String sourceSqls, + String sourceTable, + String mergeCondition, + String matchedUpsertCondition, + String matchedUpsertSetting, + String notMatchedInsertCondition, + String notMatchedInsertValues) { + return call( + procedureContext, + targetTableId, + targetAlias, + sourceSqls, + sourceTable, + mergeCondition, + matchedUpsertCondition, + matchedUpsertSetting, + notMatchedInsertCondition, + notMatchedInsertValues, + ""); + } + + public String[] call( + ProcedureContext procedureContext, + String targetTableId, + String targetAlias, + String sourceSqls, + String sourceTable, + String mergeCondition, + String matchedDeleteCondition) { + return call( + procedureContext, + targetTableId, + targetAlias, + sourceSqls, + sourceTable, + mergeCondition, + "", + "", + "", + "", + matchedDeleteCondition); + } + + public String[] call( + ProcedureContext procedureContext, + String targetTableId, + String targetAlias, + String sourceSqls, + String sourceTable, + String mergeCondition, + String matchedUpsertCondition, + String matchedUpsertSetting, + String notMatchedInsertCondition, + String notMatchedInsertValues, + String matchedDeleteCondition) { + String warehouse = catalog.warehouse(); + Map catalogOptions = catalog.options(); + Identifier identifier = Identifier.fromString(targetTableId); + MergeIntoAction action = + new MergeIntoAction( + warehouse, + identifier.getDatabaseName(), + identifier.getObjectName(), + catalogOptions); + action.withTargetAlias(nullable(targetAlias)); + + if (!sourceSqls.isEmpty()) { + action.withSourceSqls(sourceSqls.split(";")); + } + + checkArgument(!sourceTable.isEmpty(), "Must specify source table."); + action.withSourceTable(sourceTable); + + checkArgument(!mergeCondition.isEmpty(), "Must specify merge condition."); + action.withMergeCondition(mergeCondition); + + if (!matchedUpsertCondition.isEmpty() || !matchedUpsertSetting.isEmpty()) { + String condition = nullable(matchedUpsertCondition); + String setting = nullable(matchedUpsertSetting); + checkNotNull(setting, "matched-upsert must set the 'matchedUpsertSetting' argument"); + action.withMatchedUpsert(condition, setting); + } + + if (!notMatchedInsertCondition.isEmpty() || !notMatchedInsertValues.isEmpty()) { + String condition = nullable(notMatchedInsertCondition); + String values = nullable(notMatchedInsertValues); + checkNotNull( + values, "not-matched-insert must set the 'notMatchedInsertValues' argument"); + action.withNotMatchedInsert(condition, values); + } + + if (!matchedDeleteCondition.isEmpty()) { + action.withMatchedDelete(matchedDeleteCondition); + } + + action.withStreamExecutionEnvironment(procedureContext.getExecutionEnvironment()); + action.validate(); + + DataStream dataStream = action.buildDataStream(); + TableResult tableResult = action.batchSink(dataStream); + JobClient jobClient = tableResult.getJobClient().get(); + + return execute(procedureContext, jobClient); + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateDatabaseProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateDatabaseProcedure.java new file mode 100644 index 000000000000..128875a8b862 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateDatabaseProcedure.java @@ -0,0 +1,63 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.flink.utils.TableMigrationUtils; +import org.apache.paimon.migrate.Migrator; +import org.apache.paimon.utils.ParameterUtils; + +import org.apache.flink.table.procedure.ProcedureContext; + +import java.util.List; + +/** Migrate procedure to migrate all hive tables in database to paimon table. */ +public class MigrateDatabaseProcedure extends ProcedureBase { + + @Override + public String identifier() { + return "migrate_database"; + } + + public String[] call( + ProcedureContext procedureContext, String connector, String sourceDatabasePath) + throws Exception { + return call(procedureContext, connector, sourceDatabasePath, ""); + } + + public String[] call( + ProcedureContext procedureContext, + String connector, + String sourceDatabasePath, + String properties) + throws Exception { + List migrators = + TableMigrationUtils.getImporters( + connector, + catalog, + sourceDatabasePath, + ParameterUtils.parseCommaSeparatedKeyValues(properties)); + + for (Migrator migrator : migrators) { + migrator.executeMigrate(); + migrator.renameTable(false); + } + + return new String[] {"Success"}; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateFileProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateFileProcedure.java new file mode 100644 index 000000000000..110b4e25fc00 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateFileProcedure.java @@ -0,0 +1,84 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.flink.utils.TableMigrationUtils; +import org.apache.paimon.migrate.Migrator; + +import org.apache.flink.table.procedure.ProcedureContext; + +import java.util.Collections; + +/** Add file procedure to add file from hive to paimon. */ +public class MigrateFileProcedure extends ProcedureBase { + + @Override + public String identifier() { + return "migrate_file"; + } + + public String[] call( + ProcedureContext procedureContext, + String connector, + String sourceTablePath, + String targetPaimonTablePath) + throws Exception { + call(procedureContext, connector, sourceTablePath, targetPaimonTablePath, true); + return new String[] {"Success"}; + } + + public String[] call( + ProcedureContext procedureContext, + String connector, + String sourceTablePath, + String targetPaimonTablePath, + boolean deleteOrigin) + throws Exception { + migrateHandle(connector, sourceTablePath, targetPaimonTablePath, deleteOrigin); + return new String[] {"Success"}; + } + + public void migrateHandle( + String connector, + String sourceTablePath, + String targetPaimonTablePath, + boolean deleteOrigin) + throws Exception { + Identifier sourceTableId = Identifier.fromString(sourceTablePath); + Identifier targetTableId = Identifier.fromString(targetPaimonTablePath); + + if (!(catalog.tableExists(targetTableId))) { + throw new IllegalArgumentException( + "Target paimon table does not exist: " + targetPaimonTablePath); + } + + Migrator importer = + TableMigrationUtils.getImporter( + connector, + catalog, + sourceTableId.getDatabaseName(), + sourceTableId.getObjectName(), + targetTableId.getDatabaseName(), + targetTableId.getObjectName(), + Collections.emptyMap()); + importer.deleteOriginTable(deleteOrigin); + importer.executeMigrate(); + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateTableProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateTableProcedure.java new file mode 100644 index 000000000000..39e6092d8496 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/MigrateTableProcedure.java @@ -0,0 +1,72 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.flink.utils.TableMigrationUtils; +import org.apache.paimon.utils.ParameterUtils; + +import org.apache.flink.table.procedure.ProcedureContext; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +/** Migrate procedure to migrate hive table to paimon table. */ +public class MigrateTableProcedure extends ProcedureBase { + + private static final Logger LOG = LoggerFactory.getLogger(MigrateTableProcedure.class); + + private static final String PAIMON_SUFFIX = "_paimon_"; + + @Override + public String identifier() { + return "migrate_table"; + } + + public String[] call( + ProcedureContext procedureContext, String connector, String sourceTablePath) + throws Exception { + return call(procedureContext, connector, sourceTablePath, ""); + } + + public String[] call( + ProcedureContext procedureContext, + String connector, + String sourceTablePath, + String properties) + throws Exception { + String targetPaimonTablePath = sourceTablePath + PAIMON_SUFFIX; + + Identifier sourceTableId = Identifier.fromString(sourceTablePath); + Identifier targetTableId = Identifier.fromString(targetPaimonTablePath); + + TableMigrationUtils.getImporter( + connector, + catalog, + sourceTableId.getDatabaseName(), + sourceTableId.getObjectName(), + targetTableId.getDatabaseName(), + targetTableId.getObjectName(), + ParameterUtils.parseCommaSeparatedKeyValues(properties)) + .executeMigrate(); + + LOG.info("Last step: rename " + targetTableId + " to " + sourceTableId); + catalog.renameTable(targetTableId, sourceTableId, false); + return new String[] {"Success"}; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RemoveOrphanFilesProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RemoveOrphanFilesProcedure.java new file mode 100644 index 000000000000..d43056f9779e --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RemoveOrphanFilesProcedure.java @@ -0,0 +1,83 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.operation.OrphanFilesClean; +import org.apache.paimon.utils.StringUtils; + +import org.apache.flink.table.procedure.ProcedureContext; + +import java.util.List; + +import static org.apache.paimon.operation.OrphanFilesClean.executeOrphanFilesClean; + +/** + * Remove orphan files procedure. Usage: + * + *

    
    + *  -- use the default file delete interval
    + *  CALL sys.remove_orphan_files('tableId')
    + *
    + *  -- use custom file delete interval
    + *  CALL sys.remove_orphan_files('tableId', '2023-12-31 23:59:59')
    + *
    + *  -- remove all tables' orphan files in db
    + *  CALL sys.remove_orphan_files('databaseName.*', '2023-12-31 23:59:59')
    + * 
    + */ +public class RemoveOrphanFilesProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "remove_orphan_files"; + + public String[] call(ProcedureContext procedureContext, String tableId) throws Exception { + return call(procedureContext, tableId, ""); + } + + public String[] call(ProcedureContext procedureContext, String tableId, String olderThan) + throws Exception { + return call(procedureContext, tableId, olderThan, false); + } + + public String[] call( + ProcedureContext procedureContext, String tableId, String olderThan, boolean dryRun) + throws Exception { + Identifier identifier = Identifier.fromString(tableId); + String databaseName = identifier.getDatabaseName(); + String tableName = identifier.getObjectName(); + + List tableCleans = + OrphanFilesClean.createOrphanFilesCleans(catalog, databaseName, tableName); + + if (!StringUtils.isBlank(olderThan)) { + tableCleans.forEach(clean -> clean.olderThan(olderThan)); + } + + if (dryRun) { + tableCleans.forEach(clean -> clean.fileCleaner(path -> {})); + } + + return executeOrphanFilesClean(tableCleans); + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/ResetConsumerProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/ResetConsumerProcedure.java new file mode 100644 index 000000000000..0355d6dc1cab --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/ResetConsumerProcedure.java @@ -0,0 +1,80 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Catalog; +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.consumer.Consumer; +import org.apache.paimon.consumer.ConsumerManager; +import org.apache.paimon.table.FileStoreTable; + +import org.apache.flink.table.procedure.ProcedureContext; + +/** + * Reset consumer procedure. Usage: + * + *
    
    + *  -- reset the new next snapshot id in the consumer
    + *  CALL sys.reset_consumer('tableId', 'consumerId', nextSnapshotId)
    + *
    + *  -- delete consumer
    + *  CALL sys.reset_consumer('tableId', 'consumerId')
    + * 
    + */ +public class ResetConsumerProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "reset_consumer"; + + public String[] call( + ProcedureContext procedureContext, + String tableId, + String consumerId, + long nextSnapshotId) + throws Catalog.TableNotExistException { + FileStoreTable fileStoreTable = + (FileStoreTable) catalog.getTable(Identifier.fromString(tableId)); + ConsumerManager consumerManager = + new ConsumerManager( + fileStoreTable.fileIO(), + fileStoreTable.location(), + fileStoreTable.snapshotManager().branch()); + consumerManager.resetConsumer(consumerId, new Consumer(nextSnapshotId)); + + return new String[] {"Success"}; + } + + public String[] call(ProcedureContext procedureContext, String tableId, String consumerId) + throws Catalog.TableNotExistException { + FileStoreTable fileStoreTable = + (FileStoreTable) catalog.getTable(Identifier.fromString(tableId)); + ConsumerManager consumerManager = + new ConsumerManager( + fileStoreTable.fileIO(), + fileStoreTable.location(), + fileStoreTable.snapshotManager().branch()); + consumerManager.deleteConsumer(consumerId); + + return new String[] {"Success"}; + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedure.java new file mode 100644 index 000000000000..29ae1b25b57a --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedure.java @@ -0,0 +1,138 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.CoreOptions; +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.flink.sink.NoneCopyVersionedSerializerTypeSerializerProxy; +import org.apache.paimon.flink.sink.RewriteFileIndexSink; +import org.apache.paimon.flink.source.RewriteFileIndexSource; +import org.apache.paimon.manifest.ManifestEntry; +import org.apache.paimon.manifest.ManifestEntrySerializer; +import org.apache.paimon.predicate.Predicate; +import org.apache.paimon.predicate.PredicateBuilder; +import org.apache.paimon.table.FileStoreTable; +import org.apache.paimon.table.Table; +import org.apache.paimon.utils.StringUtils; + +import org.apache.flink.api.common.ExecutionConfig; +import org.apache.flink.api.common.eventtime.WatermarkStrategy; +import org.apache.flink.api.common.typeutils.TypeSerializer; +import org.apache.flink.api.java.typeutils.GenericTypeInfo; +import org.apache.flink.core.io.SimpleVersionedSerializer; +import org.apache.flink.streaming.api.datastream.DataStreamSource; +import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; +import org.apache.flink.table.procedure.ProcedureContext; + +import java.io.IOException; +import java.util.List; +import java.util.Map; + +import static org.apache.paimon.utils.ParameterUtils.getPartitions; + +/** Rewrite file index procedure to re-generated all file index. */ +public class RewriteFileIndexProcedure extends ProcedureBase { + + @Override + public String identifier() { + return "rewrite_file_index"; + } + + public String[] call(ProcedureContext procedureContext, String sourceTablePath) + throws Exception { + return call(procedureContext, sourceTablePath, ""); + } + + public String[] call( + ProcedureContext procedureContext, String sourceTablePath, String partitions) + throws Exception { + + StreamExecutionEnvironment env = procedureContext.getExecutionEnvironment(); + Table table = catalog.getTable(Identifier.fromString(sourceTablePath)); + + List> partitionList = + StringUtils.isBlank(partitions) ? null : getPartitions(partitions.split(";")); + + Predicate partitionPredicate; + if (partitionList != null) { + // This predicate is based on the row type of the original table, not bucket table. + // Because TableScan in BucketsTable is the same with FileStoreTable, + // and partition filter is done by scan. + partitionPredicate = + PredicateBuilder.or( + partitionList.stream() + .map( + p -> + PredicateBuilder.partition( + p, + ((FileStoreTable) table) + .schema() + .logicalPartitionType(), + CoreOptions.PARTITION_DEFAULT_NAME + .defaultValue())) + .toArray(Predicate[]::new)); + } else { + partitionPredicate = null; + } + + FileStoreTable storeTable = (FileStoreTable) table; + DataStreamSource source = + env.fromSource( + new RewriteFileIndexSource(storeTable, partitionPredicate), + WatermarkStrategy.noWatermarks(), + "index source", + new ManifestEntryTypeInfo()); + new RewriteFileIndexSink(storeTable).sinkFrom(source); + return execute(env, "Add file index for table: " + sourceTablePath); + } + + private static class ManifestEntryTypeInfo extends GenericTypeInfo { + + public ManifestEntryTypeInfo() { + super(ManifestEntry.class); + } + + @Override + public TypeSerializer createSerializer(ExecutionConfig config) { + return new NoneCopyVersionedSerializerTypeSerializerProxy<>( + () -> + new SimpleVersionedSerializer() { + private final ManifestEntrySerializer manifestEntrySerializer = + new ManifestEntrySerializer(); + + @Override + public int getVersion() { + return 0; + } + + @Override + public byte[] serialize(ManifestEntry manifestEntry) + throws IOException { + return manifestEntrySerializer.serializeToBytes(manifestEntry); + } + + @Override + public ManifestEntry deserialize(int i, byte[] bytes) + throws IOException { + return manifestEntrySerializer.deserializeFromBytes(bytes); + } + }); + } + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RollbackToProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RollbackToProcedure.java new file mode 100644 index 000000000000..1bf545004d93 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/RollbackToProcedure.java @@ -0,0 +1,62 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.catalog.Catalog; +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.table.Table; + +import org.apache.flink.table.procedure.ProcedureContext; + +/** + * Rollback procedure. Usage: + * + *
    
    + *  -- rollback to a snapshot
    + *  CALL sys.rollback_to('tableId', snapshotId)
    + *
    + *  -- rollback to a tag
    + *  CALL sys.rollback_to('tableId', 'tagName')
    + * 
    + */ +public class RollbackToProcedure extends ProcedureBase { + + public static final String IDENTIFIER = "rollback_to"; + + public String[] call(ProcedureContext procedureContext, String tableId, long snapshotId) + throws Catalog.TableNotExistException { + Table table = catalog.getTable(Identifier.fromString(tableId)); + table.rollbackTo(snapshotId); + + return new String[] {"Success"}; + } + + public String[] call(ProcedureContext procedureContext, String tableId, String tagName) + throws Catalog.TableNotExistException { + Table table = catalog.getTable(Identifier.fromString(tableId)); + table.rollbackTo(tagName); + + return new String[] {"Success"}; + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/privilege/GrantPrivilegeToUserProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/privilege/GrantPrivilegeToUserProcedure.java new file mode 100644 index 000000000000..e57b364a2f5d --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/privilege/GrantPrivilegeToUserProcedure.java @@ -0,0 +1,79 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure.privilege; + +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.privilege.PrivilegeType; + +import org.apache.flink.table.procedure.ProcedureContext; + +/** + * Procedure to grant privilege to a user. Privilege can be granted on the whole catalog, a database + * or a table. Only users with {@link PrivilegeType#ADMIN} privilege can perform this operation. + * Usage: + * + *
    
    + *  CALL sys.grant_privilege_to_user('username', 'privilege')
    + *  CALL sys.grant_privilege_to_user('username', 'privilege', 'database')
    + *  CALL sys.grant_privilege_to_user('username', 'privilege', 'database', 'table')
    + * 
    + */ +public class GrantPrivilegeToUserProcedure extends PrivilegeProcedureBase { + + public static final String IDENTIFIER = "grant_privilege_to_user"; + + public String[] call(ProcedureContext procedureContext, String user, String privilege) { + getPrivilegedCatalog().grantPrivilegeOnCatalog(user, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format("User %s is granted with privilege %s on the catalog.", user, privilege) + }; + } + + public String[] call( + ProcedureContext procedureContext, String user, String privilege, String database) { + getPrivilegedCatalog() + .grantPrivilegeOnDatabase(user, database, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is granted with privilege %s on database %s.", + user, privilege, database) + }; + } + + public String[] call( + ProcedureContext procedureContext, + String user, + String privilege, + String database, + String table) { + Identifier identifier = Identifier.create(database, table); + getPrivilegedCatalog() + .grantPrivilegeOnTable(user, identifier, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is granted with privilege %s on table %s.", + user, privilege, identifier) + }; + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/privilege/RevokePrivilegeFromUserProcedure.java b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/privilege/RevokePrivilegeFromUserProcedure.java new file mode 100644 index 000000000000..5f511eaa61b9 --- /dev/null +++ b/paimon-flink/paimon-flink-1.18/src/main/java/org/apache/paimon/flink/procedure/privilege/RevokePrivilegeFromUserProcedure.java @@ -0,0 +1,87 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure.privilege; + +import org.apache.paimon.catalog.Identifier; +import org.apache.paimon.privilege.PrivilegeType; + +import org.apache.flink.table.procedure.ProcedureContext; + +/** + * Procedure to revoke privilege from a user. Privilege can be revoked from the whole catalog, a + * database or a table. Only users with {@link PrivilegeType#ADMIN} privilege can perform this + * operation. Usage: + * + *
    
    + *  CALL sys.revoke_privilege_from_user('username', 'privilege')
    + *  CALL sys.revoke_privilege_from_user('username', 'privilege', 'database')
    + *  CALL sys.revoke_privilege_from_user('username', 'privilege', 'database', 'table')
    + * 
    + */ +public class RevokePrivilegeFromUserProcedure extends PrivilegeProcedureBase { + + public static final String IDENTIFIER = "revoke_privilege_from_user"; + + public String[] call(ProcedureContext procedureContext, String user, String privilege) { + int count = + getPrivilegedCatalog() + .revokePrivilegeOnCatalog(user, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format("User %s is revoked with privilege %s on the catalog.", user, privilege), + "Number of privileges revoked: " + count + }; + } + + public String[] call( + ProcedureContext procedureContext, String user, String privilege, String database) { + int count = + getPrivilegedCatalog() + .revokePrivilegeOnDatabase( + user, database, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is revoked with privilege %s on database %s.", + user, privilege, database), + "Number of privileges revoked: " + count + }; + } + + public String[] call( + ProcedureContext procedureContext, + String user, + String privilege, + String database, + String table) { + Identifier identifier = Identifier.create(database, table); + int count = + getPrivilegedCatalog() + .revokePrivilegeOnTable(user, identifier, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is revoked with privilege %s on table %s.", + user, privilege, identifier), + "Number of privileges revoked: " + count + }; + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-1.18/src/test/java/org/apache/paimon/flink/procedure/ProcedurePositionalArgumentsITCase.java b/paimon-flink/paimon-flink-1.18/src/test/java/org/apache/paimon/flink/procedure/ProcedurePositionalArgumentsITCase.java index cd9a8b184f3e..967eb9d65639 100644 --- a/paimon-flink/paimon-flink-1.18/src/test/java/org/apache/paimon/flink/procedure/ProcedurePositionalArgumentsITCase.java +++ b/paimon-flink/paimon-flink-1.18/src/test/java/org/apache/paimon/flink/procedure/ProcedurePositionalArgumentsITCase.java @@ -19,22 +19,28 @@ package org.apache.paimon.flink.procedure; import org.apache.paimon.flink.CatalogITCaseBase; +import org.apache.paimon.privilege.NoPrivilegeException; import org.apache.paimon.table.FileStoreTable; +import org.apache.flink.types.Row; +import org.apache.flink.util.CloseableIterator; import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.function.Executable; import java.io.IOException; import java.util.ArrayList; +import java.util.Arrays; import java.util.List; import static org.assertj.core.api.Assertions.assertThat; import static org.assertj.core.api.Assertions.assertThatCode; +import static org.assertj.core.api.Assertions.assertThatThrownBy; +import static org.junit.jupiter.api.Assertions.assertThrows; /** Ensure that the legacy multiply overloaded CALL with positional arguments can be invoked. */ public class ProcedurePositionalArgumentsITCase extends CatalogITCaseBase { - @Test - public void testCallCompact() { + public void testCompactDatabaseAndTable() { sql( "CREATE TABLE T (" + " k INT," @@ -61,6 +67,166 @@ public void testCallCompact() { assertThatCode(() -> sql("CALL sys.compact('default.T', '', 'zorder', 'k', '','','5s')")) .message() .contains("sort compact do not support 'partition_idle_time'."); + + assertThatCode(() -> sql("CALL sys.compact_database('default')")) + .doesNotThrowAnyException(); + } + + @Test + public void testUserPrivileges() throws Exception { + sql( + String.format( + "CREATE CATALOG mycat WITH (\n" + + " 'type' = 'paimon',\n" + + " 'warehouse' = '%s'\n" + + ")", + path)); + sql("USE CATALOG mycat"); + sql("CREATE DATABASE mydb"); + sql("CREATE DATABASE mydb2"); + sql( + "CREATE TABLE mydb.T1 (\n" + + " k INT,\n" + + " v INT,\n" + + " PRIMARY KEY (k) NOT ENFORCED\n" + + ")"); + sql("INSERT INTO mydb.T1 VALUES (1, 10), (2, 20), (3, 30)"); + sql("CALL sys.init_file_based_privilege('root-passwd')"); + + sql( + String.format( + "CREATE CATALOG anonymouscat WITH (\n" + + " 'type' = 'paimon',\n" + + " 'warehouse' = '%s'\n" + + ")", + path)); + + sql("USE CATALOG anonymouscat"); + assertNoPrivilege(() -> sql("INSERT INTO mydb.T1 VALUES (1, 11), (2, 21)")); + assertNoPrivilege(() -> collect("SELECT * FROM mydb.T1 ORDER BY k")); + + sql( + String.format( + "CREATE CATALOG rootcat WITH (\n" + + " 'type' = 'paimon',\n" + + " 'warehouse' = '%s',\n" + + " 'user' = 'root',\n" + + " 'password' = 'root-passwd'\n" + + ")", + path)); + sql("USE CATALOG rootcat"); + sql( + "CREATE TABLE mydb2.T2 (\n" + + " k INT,\n" + + " v INT,\n" + + " PRIMARY KEY (k) NOT ENFORCED\n" + + ")"); + sql("INSERT INTO mydb2.T2 VALUES (100, 1000), (200, 2000), (300, 3000)"); + sql("CALL sys.create_privileged_user('test', 'test-passwd')"); + sql("CALL sys.grant_privilege_to_user('test', 'CREATE_TABLE', 'mydb')"); + sql("CALL sys.grant_privilege_to_user('test', 'SELECT', 'mydb')"); + sql("CALL sys.grant_privilege_to_user('test', 'INSERT', 'mydb')"); + + sql( + String.format( + "CREATE CATALOG testcat WITH (\n" + + " 'type' = 'paimon',\n" + + " 'warehouse' = '%s',\n" + + " 'user' = 'test',\n" + + " 'password' = 'test-passwd'\n" + + ")", + path)); + sql("USE CATALOG testcat"); + sql("INSERT INTO mydb.T1 VALUES (1, 12), (2, 22)"); + assertThat(collect("SELECT * FROM mydb.T1 ORDER BY k")) + .isEqualTo(Arrays.asList(Row.of(1, 12), Row.of(2, 22), Row.of(3, 30))); + sql("CREATE TABLE mydb.S1 ( a INT, b INT )"); + sql("INSERT INTO mydb.S1 VALUES (1, 100), (2, 200), (3, 300)"); + assertThat(collect("SELECT * FROM mydb.S1 ORDER BY a")) + .isEqualTo(Arrays.asList(Row.of(1, 100), Row.of(2, 200), Row.of(3, 300))); + assertNoPrivilege(() -> sql("DROP TABLE mydb.T1")); + assertNoPrivilege(() -> sql("ALTER TABLE mydb.T1 RENAME TO mydb.T2")); + assertNoPrivilege(() -> sql("DROP TABLE mydb.S1")); + assertNoPrivilege(() -> sql("ALTER TABLE mydb.S1 RENAME TO mydb.S2")); + assertNoPrivilege(() -> sql("CREATE DATABASE anotherdb")); + assertNoPrivilege(() -> sql("DROP DATABASE mydb CASCADE")); + assertNoPrivilege(() -> sql("CALL sys.create_privileged_user('test2', 'test2-passwd')")); + + sql("USE CATALOG rootcat"); + sql("CALL sys.create_privileged_user('test2', 'test2-passwd')"); + sql("CALL sys.grant_privilege_to_user('test2', 'SELECT', 'mydb2')"); + sql("CALL sys.grant_privilege_to_user('test2', 'INSERT', 'mydb', 'T1')"); + sql("CALL sys.grant_privilege_to_user('test2', 'SELECT', 'mydb', 'S1')"); + sql("CALL sys.grant_privilege_to_user('test2', 'CREATE_DATABASE')"); + + sql( + String.format( + "CREATE CATALOG test2cat WITH (\n" + + " 'type' = 'paimon',\n" + + " 'warehouse' = '%s',\n" + + " 'user' = 'test2',\n" + + " 'password' = 'test2-passwd'\n" + + ")", + path)); + sql("USE CATALOG test2cat"); + sql("INSERT INTO mydb.T1 VALUES (1, 13), (2, 23)"); + assertNoPrivilege(() -> collect("SELECT * FROM mydb.T1 ORDER BY k")); + assertNoPrivilege(() -> sql("CREATE TABLE mydb.S2 ( a INT, b INT )")); + assertNoPrivilege(() -> sql("INSERT INTO mydb.S1 VALUES (1, 100), (2, 200), (3, 300)")); + assertThat(collect("SELECT * FROM mydb.S1 ORDER BY a")) + .isEqualTo(Arrays.asList(Row.of(1, 100), Row.of(2, 200), Row.of(3, 300))); + assertNoPrivilege( + () -> sql("INSERT INTO mydb2.T2 VALUES (100, 1001), (200, 2001), (300, 3001)")); + assertThat(collect("SELECT * FROM mydb2.T2 ORDER BY k")) + .isEqualTo(Arrays.asList(Row.of(100, 1000), Row.of(200, 2000), Row.of(300, 3000))); + sql("CREATE DATABASE anotherdb"); + assertNoPrivilege(() -> sql("DROP TABLE mydb.T1")); + assertNoPrivilege(() -> sql("ALTER TABLE mydb.T1 RENAME TO mydb.T2")); + assertNoPrivilege(() -> sql("DROP TABLE mydb.S1")); + assertNoPrivilege(() -> sql("ALTER TABLE mydb.S1 RENAME TO mydb.S2")); + assertNoPrivilege(() -> sql("DROP DATABASE mydb CASCADE")); + assertNoPrivilege(() -> sql("CALL sys.create_privileged_user('test3', 'test3-passwd')")); + + sql("USE CATALOG rootcat"); + assertThat(collect("SELECT * FROM mydb.T1 ORDER BY k")) + .isEqualTo(Arrays.asList(Row.of(1, 13), Row.of(2, 23), Row.of(3, 30))); + sql("CALL sys.revoke_privilege_from_user('test2', 'SELECT')"); + sql("CALL sys.drop_privileged_user('test')"); + + sql("USE CATALOG testcat"); + Exception e = + assertThrows(Exception.class, () -> collect("SELECT * FROM mydb.T1 ORDER BY k")); + assertThat(e).hasRootCauseMessage("User test not found, or password incorrect."); + + sql("USE CATALOG test2cat"); + assertNoPrivilege(() -> collect("SELECT * FROM mydb.S1 ORDER BY a")); + assertNoPrivilege(() -> collect("SELECT * FROM mydb2.T2 ORDER BY k")); + sql("INSERT INTO mydb.T1 VALUES (1, 14), (2, 24)"); + + sql("USE CATALOG rootcat"); + assertThat(collect("SELECT * FROM mydb.T1 ORDER BY k")) + .isEqualTo(Arrays.asList(Row.of(1, 14), Row.of(2, 24), Row.of(3, 30))); + sql("DROP DATABASE mydb CASCADE"); + sql("DROP DATABASE mydb2 CASCADE"); + } + + private List collect(String sql) throws Exception { + List result = new ArrayList<>(); + try (CloseableIterator it = tEnv.executeSql(sql).collect()) { + while (it.hasNext()) { + result.add(it.next()); + } + } + return result; + } + + private void assertNoPrivilege(Executable executable) { + Exception e = assertThrows(Exception.class, executable); + if (e.getCause() != null) { + assertThat(e).hasRootCauseInstanceOf(NoPrivilegeException.class); + } else { + assertThat(e).isInstanceOf(NoPrivilegeException.class); + } } @Test @@ -88,4 +254,235 @@ private List read(FileStoreTable table) throws IOException { .forEachRemaining(row -> ret.add(row.getString(0) + ":" + row.getString(1))); return ret; } + + @Test + public void testCreateDeleteTag() { + sql( + "CREATE TABLE T (" + + " k STRING," + + " dt STRING," + + " PRIMARY KEY (k, dt) NOT ENFORCED" + + ") PARTITIONED BY (dt) WITH (" + + " 'bucket' = '1'" + + ")"); + sql("insert into T values('k', '2024-01-01')"); + sql("insert into T values('k2', '2024-01-02')"); + + sql("CALL sys.create_tag('default.T', 'tag1')"); + + assertThat( + sql("select * from T /*+ OPTIONS('scan.tag-name'='tag1') */").stream() + .map(Row::toString)) + .containsExactlyInAnyOrder("+I[k2, 2024-01-02]", "+I[k, 2024-01-01]"); + + sql("CALL sys.rollback_to('default.T', 'tag1')"); + + assertThat(sql("select * from T").stream().map(Row::toString)) + .containsExactlyInAnyOrder("+I[k2, 2024-01-02]", "+I[k, 2024-01-01]"); + + sql("CALL sys.delete_tag('default.T', 'tag1')"); + + assertThatThrownBy( + () -> + sql("select * from T /*+ OPTIONS('scan.tag-name'='tag1') */") + .stream() + .map(Row::toString)) + .hasRootCauseInstanceOf(IllegalArgumentException.class) + .hasMessageContaining("Tag 'tag1' doesn't exist."); + } + + @Test + public void testCreateDeleteAndForwardBranch() throws Exception { + sql( + "CREATE TABLE T (" + + " pt INT" + + ", k INT" + + ", v STRING" + + ", PRIMARY KEY (pt, k) NOT ENFORCED" + + " ) PARTITIONED BY (pt) WITH (" + + " 'bucket' = '2'" + + " )"); + + // snapshot 1. + sql("INSERT INTO T VALUES(1, 10, 'apple')"); + + // snapshot 2. + sql("INSERT INTO T VALUES(1, 20, 'dog')"); + + sql("CALL sys.create_tag('default.T', 'tag1', 1)"); + + sql("CALL sys.create_tag('default.T', 'tag2', 2)"); + + sql("CALL sys.create_branch('default.T', 'test', 'tag1')"); + sql("CALL sys.create_branch('default.T', 'test2', 'tag2')"); + + assertThat(collectToString("SELECT branch_name, created_from_snapshot FROM `T$branches`")) + .containsExactlyInAnyOrder("+I[test, 1]", "+I[test2, 2]"); + + sql("CALL sys.delete_branch('default.T', 'test')"); + + assertThat(collectToString("SELECT branch_name, created_from_snapshot FROM `T$branches`")) + .containsExactlyInAnyOrder("+I[test2, 2]"); + + sql("CALL sys.fast_forward('default.T', 'test2')"); + + // Branch `test` replaces the main branch. + assertThat(collectToString("SELECT * FROM `T`")) + .containsExactlyInAnyOrder("+I[1, 10, apple]", "+I[1, 20, dog]"); + } + + private List collectToString(String sql) throws Exception { + List result = new ArrayList<>(); + try (CloseableIterator it = tEnv.executeSql(sql).collect()) { + while (it.hasNext()) { + result.add(it.next().toString()); + } + } + return result; + } + + @Test + public void testPartitionMarkDone() { + sql( + "CREATE TABLE T (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + + assertThatCode(() -> sql("CALL sys.mark_partition_done('default.T', 'pt = 0')")) + .doesNotThrowAnyException(); + } + + @Test + public void testMergeInto() { + sql( + "CREATE TABLE T (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + sql( + "CREATE TABLE S (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + + assertThatCode( + () -> + sql( + "CALL sys.merge_into('default.T', 'TT', '', 'S', 'TT.k = S.k', '', '', '', '', 'S.v IS NULL')")) + .doesNotThrowAnyException(); + } + + @Test + public void testMigrateProcedures() { + sql( + "CREATE TABLE S (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + + assertThatThrownBy(() -> sql("CALL sys.migrate_database('hive', 'default', '')")) + .hasMessageContaining("Only support Hive Catalog."); + assertThatThrownBy(() -> sql("CALL sys.migrate_table('hive', 'default.T', '')")) + .hasMessageContaining("Only support Hive Catalog."); + assertThatThrownBy(() -> sql("CALL sys.migrate_file('hive', 'default.T', 'default.S')")) + .hasMessageContaining("Only support Hive Catalog."); + } + + @Test + public void testQueryService() { + sql( + "CREATE TABLE DIM (k INT PRIMARY KEY NOT ENFORCED, v INT) WITH ('bucket' = '2', 'continuous.discovery-interval' = '1ms')"); + assertThatCode( + () -> { + CloseableIterator service = + streamSqlIter("CALL sys.query_service('default.DIM', 2)"); + service.close(); + }) + .doesNotThrowAnyException(); + } + + @Test + public void testRemoveOrphanFiles() { + sql( + "CREATE TABLE T (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + assertThatCode(() -> sql("CALL sys.remove_orphan_files('default.T')")) + .doesNotThrowAnyException(); + } + + @Test + public void testRepair() { + sql( + "CREATE TABLE T (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + assertThatThrownBy(() -> sql("CALL sys.repair('default.T')")) + .hasStackTraceContaining("Catalog.repairTable"); + } + + @Test + public void testResetConsumer() { + sql( + "CREATE TABLE T (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + assertThatCode(() -> sql("CALL sys.reset_consumer('default.T', 'myid')")) + .doesNotThrowAnyException(); + } + + @Test + public void testRewriteFileIndex() { + sql( + "CREATE TABLE T (" + + " k INT," + + " v INT," + + " pt INT," + + " PRIMARY KEY (k, pt) NOT ENFORCED" + + ") PARTITIONED BY (pt) WITH (" + + " 'write-only' = 'true'," + + " 'bucket' = '1'" + + ")"); + assertThatCode(() -> sql("CALL sys.rewrite_file_index('default.T', 'pt = 0')")) + .doesNotThrowAnyException(); + } } diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/action/RewriteFileIndexAction.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/action/RewriteFileIndexAction.java index 82dfe38bb7f1..f8ed73fd2025 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/action/RewriteFileIndexAction.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/action/RewriteFileIndexAction.java @@ -38,6 +38,6 @@ public RewriteFileIndexAction( public void run() throws Exception { RewriteFileIndexProcedure rewriteFileIndexProcedure = new RewriteFileIndexProcedure(); rewriteFileIndexProcedure.withCatalog(catalog); - rewriteFileIndexProcedure.call(new DefaultProcedureContext(env), identifier); + rewriteFileIndexProcedure.call(new DefaultProcedureContext(env), identifier, ""); } } diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/clone/CloneSourceBuilder.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/clone/CloneSourceBuilder.java index db0452873d36..d59734a5a82a 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/clone/CloneSourceBuilder.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/clone/CloneSourceBuilder.java @@ -76,7 +76,7 @@ public DataStream> build() throws Exception { private DataStream> build(Catalog sourceCatalog) throws Exception { List> result = new ArrayList<>(); - if (database == null) { + if (StringUtils.isBlank(database)) { checkArgument( StringUtils.isBlank(tableName), "tableName must be blank when database is null."); @@ -92,7 +92,7 @@ private DataStream> build(Catalog sourceCatalog) throws E result.add(new Tuple2<>(s, s)); } } - } else if (tableName == null) { + } else if (StringUtils.isBlank(tableName)) { checkArgument( !StringUtils.isBlank(targetDatabase), "targetDatabase must not be blank when clone all tables in a database."); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CloneProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CloneProcedure.java new file mode 100644 index 000000000000..8b3bc99567c8 --- /dev/null +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CloneProcedure.java @@ -0,0 +1,86 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.flink.action.CloneAction; + +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; +import org.apache.flink.table.procedure.ProcedureContext; + +/** Clone Procedure. */ +public class CloneProcedure extends ProcedureBase { + public static final String IDENTIFIER = "clone"; + + @ProcedureHint( + argument = { + @ArgumentHint(name = "warehouse", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "database", type = @DataTypeHint("STRING"), isOptional = true), + @ArgumentHint(name = "table", type = @DataTypeHint("STRING"), isOptional = true), + @ArgumentHint( + name = "catalog_conf", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint(name = "target_warehouse", type = @DataTypeHint("STRING")), + @ArgumentHint( + name = "target_database", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "target_table", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "target_catalog_conf", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint(name = "parallelism", type = @DataTypeHint("INT"), isOptional = true) + }) + public String[] call( + ProcedureContext procedureContext, + String warehouse, + String database, + String tableName, + String sourceCatalogConfigStr, + String targetWarehouse, + String targetDatabase, + String targetTableName, + String targetCatalogConfigStr, + Integer parallelismStr) + throws Exception { + CloneAction cloneAction = + new CloneAction( + warehouse, + database, + tableName, + optionalConfigMap(sourceCatalogConfigStr), + targetWarehouse, + targetDatabase, + targetTableName, + optionalConfigMap(targetCatalogConfigStr), + parallelismStr == null ? null : Integer.toString(parallelismStr)); + return execute(procedureContext, cloneAction, "Clone Job"); + } + + @Override + public String identifier() { + return IDENTIFIER; + } +} diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CompactDatabaseProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CompactDatabaseProcedure.java index 75511894149d..71c068d7eeb9 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CompactDatabaseProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CompactDatabaseProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.utils.StringUtils; import org.apache.paimon.utils.TimeUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import java.util.Map; @@ -57,58 +60,30 @@ public class CompactDatabaseProcedure extends ProcedureBase { public static final String IDENTIFIER = "compact_database"; - public String[] call(ProcedureContext procedureContext) throws Exception { - return call(procedureContext, ""); - } - - public String[] call(ProcedureContext procedureContext, String includingDatabases) - throws Exception { - return call(procedureContext, includingDatabases, ""); - } - - public String[] call(ProcedureContext procedureContext, String includingDatabases, String mode) - throws Exception { - return call(procedureContext, includingDatabases, mode, ""); - } - - public String[] call( - ProcedureContext procedureContext, - String includingDatabases, - String mode, - String includingTables) - throws Exception { - return call(procedureContext, includingDatabases, mode, includingTables, ""); - } - - public String[] call( - ProcedureContext procedureContext, - String includingDatabases, - String mode, - String includingTables, - String excludingTables) - throws Exception { - return call( - procedureContext, includingDatabases, mode, includingTables, excludingTables, ""); - } - - public String[] call( - ProcedureContext procedureContext, - String includingDatabases, - String mode, - String includingTables, - String excludingTables, - String tableOptions) - throws Exception { - return call( - procedureContext, - includingDatabases, - mode, - includingTables, - excludingTables, - tableOptions, - ""); - } - + @ProcedureHint( + argument = { + @ArgumentHint( + name = "including_databases", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint(name = "mode", type = @DataTypeHint("STRING"), isOptional = true), + @ArgumentHint( + name = "including_tables", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "excluding_tables", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "table_options", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "partition_idle_time", + type = @DataTypeHint("STRING"), + isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String includingDatabases, @@ -118,6 +93,7 @@ public String[] call( String tableOptions, String partitionIdleTime) throws Exception { + partitionIdleTime = notnull(partitionIdleTime); String warehouse = catalog.warehouse(); Map catalogOptions = catalog.options(); CompactDatabaseAction action = diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateBranchProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateBranchProcedure.java index 093505923fd6..74cf85a4c5c2 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateBranchProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateBranchProcedure.java @@ -23,6 +23,9 @@ import org.apache.paimon.table.Table; import org.apache.commons.lang3.StringUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -41,19 +44,15 @@ public String identifier() { return IDENTIFIER; } + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "branch", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "tag", type = @DataTypeHint("STRING"), isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String tableId, String branchName, String tagName) throws Catalog.TableNotExistException { - return innerCall(tableId, branchName, tagName); - } - - public String[] call(ProcedureContext procedureContext, String tableId, String branchName) - throws Catalog.TableNotExistException { - return innerCall(tableId, branchName, null); - } - - private String[] innerCall(String tableId, String branchName, String tagName) - throws Catalog.TableNotExistException { Table table = catalog.getTable(Identifier.fromString(tableId)); if (!StringUtils.isBlank(tagName)) { table.createBranch(branchName, tagName); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateTagProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateTagProcedure.java index 1a7b03ef6512..3fb51c8d935c 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateTagProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/CreateTagProcedure.java @@ -23,6 +23,9 @@ import org.apache.paimon.table.Table; import org.apache.paimon.utils.TimeUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import javax.annotation.Nullable; @@ -40,34 +43,21 @@ public class CreateTagProcedure extends ProcedureBase { public static final String IDENTIFIER = "create_tag"; - public String[] call( - ProcedureContext procedureContext, String tableId, String tagName, long snapshotId) - throws Catalog.TableNotExistException { - return innerCall(tableId, tagName, snapshotId, null); - } - - public String[] call(ProcedureContext procedureContext, String tableId, String tagName) - throws Catalog.TableNotExistException { - return innerCall(tableId, tagName, null, null); - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "tag", type = @DataTypeHint("STRING")), + @ArgumentHint( + name = "snapshot_id", + type = @DataTypeHint("BIGINT"), + isOptional = true), + @ArgumentHint( + name = "time_retained", + type = @DataTypeHint("STRING"), + isOptional = true) + }) public String[] call( ProcedureContext procedureContext, - String tableId, - String tagName, - long snapshotId, - String timeRetained) - throws Catalog.TableNotExistException { - return innerCall(tableId, tagName, snapshotId, timeRetained); - } - - public String[] call( - ProcedureContext procedureContext, String tableId, String tagName, String timeRetained) - throws Catalog.TableNotExistException { - return innerCall(tableId, tagName, null, timeRetained); - } - - private String[] innerCall( String tableId, String tagName, @Nullable Long snapshotId, diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteBranchProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteBranchProcedure.java index e7ff20f28b13..c95fd62bee40 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteBranchProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteBranchProcedure.java @@ -21,6 +21,9 @@ import org.apache.paimon.catalog.Catalog; import org.apache.paimon.catalog.Identifier; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -39,6 +42,11 @@ public String identifier() { return IDENTIFIER; } + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "branch", type = @DataTypeHint("STRING")) + }) public String[] call(ProcedureContext procedureContext, String tableId, String branchStr) throws Catalog.TableNotExistException { catalog.getTable(Identifier.fromString(tableId)).deleteBranches(branchStr); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteTagProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteTagProcedure.java index 58e6d637ff33..2ae104c9d191 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteTagProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/DeleteTagProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.catalog.Identifier; import org.apache.paimon.table.Table; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -35,6 +38,11 @@ public class DeleteTagProcedure extends ProcedureBase { public static final String IDENTIFIER = "delete_tag"; + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "tag", type = @DataTypeHint("STRING")) + }) public String[] call(ProcedureContext procedureContext, String tableId, String tagNameStr) throws Catalog.TableNotExistException { Table table = catalog.getTable(Identifier.fromString(tableId)); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/FastForwardProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/FastForwardProcedure.java index e77749f1147d..f707d74c3f12 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/FastForwardProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/FastForwardProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.catalog.Identifier; import org.apache.paimon.table.Table; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -40,6 +43,11 @@ public String identifier() { return IDENTIFIER; } + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "branch", type = @DataTypeHint("STRING")) + }) public String[] call(ProcedureContext procedureContext, String tableId, String branchName) throws Catalog.TableNotExistException { return innerCall(tableId, branchName); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MarkPartitionDoneProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MarkPartitionDoneProcedure.java index d70cccf6ba25..f0a89a0bb32b 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MarkPartitionDoneProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MarkPartitionDoneProcedure.java @@ -27,6 +27,9 @@ import org.apache.paimon.utils.IOUtils; import org.apache.paimon.utils.PartitionPathUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import java.io.IOException; @@ -40,16 +43,21 @@ * Partition mark done procedure. Usage: * *
    
    - *  CALL sys.mark_partition_done('tableId', 'partition1', 'partition2', ...)
    + *  CALL sys.mark_partition_done('tableId', 'partition1;partition2')
      * 
    */ public class MarkPartitionDoneProcedure extends ProcedureBase { public static final String IDENTIFIER = "mark_partition_done"; - public String[] call( - ProcedureContext procedureContext, String tableId, String... partitionStrings) + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "partitions", type = @DataTypeHint("STRING")) + }) + public String[] call(ProcedureContext procedureContext, String tableId, String partitions) throws Catalog.TableNotExistException, IOException { + String[] partitionStrings = partitions.split(";"); checkArgument( partitionStrings.length > 0, "mark_partition_done procedure must specify partitions."); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MergeIntoProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MergeIntoProcedure.java index acda2afd2e69..cf8d7191953e 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MergeIntoProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MergeIntoProcedure.java @@ -23,6 +23,9 @@ import org.apache.flink.core.execution.JobClient; import org.apache.flink.streaming.api.datastream.DataStream; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.api.TableResult; import org.apache.flink.table.data.RowData; import org.apache.flink.table.procedure.ProcedureContext; @@ -37,32 +40,6 @@ * *
    
      *  -- NOTE: use '' as placeholder for optional arguments
    - *
    - *  -- when matched then upsert
    - *  CALL sys.merge_into(
    - *      'targetTableId',
    - *      'targetAlias',
    - *      'sourceSqls', -- separate with ';'
    - *      'sourceTable',
    - *      'mergeCondition',
    - *      'matchedUpsertCondition',
    - *      'matchedUpsertSetting'
    - *  )
    - *
    - *  -- when matched then upsert + when not matched then insert
    - *  CALL sys.merge_into(
    - *      'targetTableId'
    - *      'targetAlias',
    - *      'sourceSqls',
    - *      'sourceTable',
    - *      'mergeCondition',
    - *      'matchedUpsertCondition',
    - *      'matchedUpsertSetting',
    - *      'notMatchedInsertCondition',
    - *      'notMatchedInsertValues'
    - *  )
    - *
    - *  -- above + when matched then delete
      *  -- IMPORTANT: Use 'TRUE' if you want to delete data without filter condition.
      *  -- If matchedDeleteCondition='', it will ignore matched-delete action!
      *  CALL sys.merge_into(
    @@ -77,16 +54,6 @@
      *      'notMatchedInsertValues',
      *      'matchedDeleteCondition'
      *  )
    - *
    - *  -- when matched then delete (short form)
    - *  CALL sys.merge_into(
    - *      'targetTableId'
    - *      'targetAlias',
    - *      'sourceSqls',
    - *      'sourceTable',
    - *      'mergeCondition',
    - *      'matchedDeleteCondition'
    - *  )
      * 
    * *

    This procedure will be forced to use batch environments. Compared to {@link MergeIntoAction}, @@ -97,76 +64,46 @@ public class MergeIntoProcedure extends ProcedureBase { public static final String IDENTIFIER = "merge_into"; - public String[] call( - ProcedureContext procedureContext, - String targetTableId, - String targetAlias, - String sourceSqls, - String sourceTable, - String mergeCondition, - String matchedUpsertCondition, - String matchedUpsertSetting) { - return call( - procedureContext, - targetTableId, - targetAlias, - sourceSqls, - sourceTable, - mergeCondition, - matchedUpsertCondition, - matchedUpsertSetting, - "", - "", - ""); - } - - public String[] call( - ProcedureContext procedureContext, - String targetTableId, - String targetAlias, - String sourceSqls, - String sourceTable, - String mergeCondition, - String matchedUpsertCondition, - String matchedUpsertSetting, - String notMatchedInsertCondition, - String notMatchedInsertValues) { - return call( - procedureContext, - targetTableId, - targetAlias, - sourceSqls, - sourceTable, - mergeCondition, - matchedUpsertCondition, - matchedUpsertSetting, - notMatchedInsertCondition, - notMatchedInsertValues, - ""); - } - - public String[] call( - ProcedureContext procedureContext, - String targetTableId, - String targetAlias, - String sourceSqls, - String sourceTable, - String mergeCondition, - String matchedDeleteCondition) { - return call( - procedureContext, - targetTableId, - targetAlias, - sourceSqls, - sourceTable, - mergeCondition, - "", - "", - "", - "", - matchedDeleteCondition); - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "target_table", type = @DataTypeHint("STRING")), + @ArgumentHint( + name = "target_alias", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "source_sqls", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "source_table", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "merge_condition", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "matched_upsert_condition", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "matched_upsert_setting", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "not_matched_insert_condition", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "not_matched_insert_values", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint( + name = "matched_delete_condition", + type = @DataTypeHint("STRING"), + isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String targetTableId, @@ -179,6 +116,16 @@ public String[] call( String notMatchedInsertCondition, String notMatchedInsertValues, String matchedDeleteCondition) { + targetAlias = notnull(targetAlias); + sourceSqls = notnull(sourceSqls); + sourceTable = notnull(sourceTable); + mergeCondition = notnull(mergeCondition); + matchedUpsertCondition = notnull(matchedUpsertCondition); + matchedUpsertSetting = notnull(matchedUpsertSetting); + notMatchedInsertCondition = notnull(notMatchedInsertCondition); + notMatchedInsertValues = notnull(notMatchedInsertValues); + matchedDeleteCondition = notnull(matchedDeleteCondition); + String warehouse = catalog.warehouse(); Map catalogOptions = catalog.options(); Identifier identifier = Identifier.fromString(targetTableId); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateDatabaseProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateDatabaseProcedure.java index 128875a8b862..ad4f704705a7 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateDatabaseProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateDatabaseProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.migrate.Migrator; import org.apache.paimon.utils.ParameterUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import java.util.List; @@ -34,18 +37,19 @@ public String identifier() { return "migrate_database"; } - public String[] call( - ProcedureContext procedureContext, String connector, String sourceDatabasePath) - throws Exception { - return call(procedureContext, connector, sourceDatabasePath, ""); - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "connector", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "source_database", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "options", type = @DataTypeHint("STRING"), isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String connector, String sourceDatabasePath, String properties) throws Exception { + properties = notnull(properties); List migrators = TableMigrationUtils.getImporters( connector, diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateFileProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateFileProcedure.java index 110b4e25fc00..2538be288109 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateFileProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateFileProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.flink.utils.TableMigrationUtils; import org.apache.paimon.migrate.Migrator; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import java.util.Collections; @@ -34,23 +37,26 @@ public String identifier() { return "migrate_file"; } - public String[] call( - ProcedureContext procedureContext, - String connector, - String sourceTablePath, - String targetPaimonTablePath) - throws Exception { - call(procedureContext, connector, sourceTablePath, targetPaimonTablePath, true); - return new String[] {"Success"}; - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "connector", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "source_table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "target_table", type = @DataTypeHint("STRING")), + @ArgumentHint( + name = "delete_origin", + type = @DataTypeHint("BOOLEAN"), + isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String connector, String sourceTablePath, String targetPaimonTablePath, - boolean deleteOrigin) + Boolean deleteOrigin) throws Exception { + if (deleteOrigin == null) { + deleteOrigin = true; + } migrateHandle(connector, sourceTablePath, targetPaimonTablePath, deleteOrigin); return new String[] {"Success"}; } diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateTableProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateTableProcedure.java index 39e6092d8496..3dfc948c6bec 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateTableProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/MigrateTableProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.flink.utils.TableMigrationUtils; import org.apache.paimon.utils.ParameterUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -38,18 +41,20 @@ public String identifier() { return "migrate_table"; } - public String[] call( - ProcedureContext procedureContext, String connector, String sourceTablePath) - throws Exception { - return call(procedureContext, connector, sourceTablePath, ""); - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "connector", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "source_table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "options", type = @DataTypeHint("STRING"), isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String connector, String sourceTablePath, String properties) throws Exception { + properties = notnull(properties); + String targetPaimonTablePath = sourceTablePath + PAIMON_SUFFIX; Identifier sourceTableId = Identifier.fromString(sourceTablePath); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ProcedureBase.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ProcedureBase.java index 7d5542109d28..814257234ad9 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ProcedureBase.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ProcedureBase.java @@ -35,7 +35,12 @@ import javax.annotation.Nullable; +import java.util.Collections; +import java.util.HashMap; +import java.util.Map; + import static org.apache.flink.table.api.config.TableConfigOptions.TABLE_DML_SYNC; +import static org.apache.paimon.utils.ParameterUtils.parseKeyValueString; /** Base implementation for flink {@link Procedure}. */ public abstract class ProcedureBase implements Procedure, Factory { @@ -51,6 +56,10 @@ protected Table table(String tableId) throws Catalog.TableNotExistException { return catalog.getTable(Identifier.fromString(tableId)); } + protected String notnull(@Nullable String arg) { + return arg == null ? "" : arg; + } + @Nullable protected String nullable(String arg) { return StringUtils.isBlank(arg) ? null : arg; @@ -92,4 +101,16 @@ private String[] execute(JobClient jobClient, boolean dmlSync) { return new String[] {"JobID=" + jobId}; } } + + protected Map optionalConfigMap(String configStr) { + if (StringUtils.isBlank(configStr)) { + return Collections.emptyMap(); + } + + Map config = new HashMap<>(); + for (String kvString : configStr.split(";")) { + parseKeyValueString(config, kvString); + } + return config; + } } diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/QueryServiceProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/QueryServiceProcedure.java index 2f236bb4fde8..1a1b34fe2b05 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/QueryServiceProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/QueryServiceProcedure.java @@ -23,6 +23,9 @@ import org.apache.paimon.table.Table; import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -41,6 +44,11 @@ public String identifier() { return IDENTIFIER; } + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "parallelism", type = @DataTypeHint("INT")) + }) public String[] call(ProcedureContext procedureContext, String tableId, int parallelism) throws Exception { Table table = catalog.getTable(Identifier.fromString(tableId)); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RemoveOrphanFilesProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RemoveOrphanFilesProcedure.java index d43056f9779e..b3eec470666a 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RemoveOrphanFilesProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RemoveOrphanFilesProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.operation.OrphanFilesClean; import org.apache.paimon.utils.StringUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import java.util.List; @@ -46,18 +49,26 @@ public class RemoveOrphanFilesProcedure extends ProcedureBase { public static final String IDENTIFIER = "remove_orphan_files"; - public String[] call(ProcedureContext procedureContext, String tableId) throws Exception { - return call(procedureContext, tableId, ""); - } - - public String[] call(ProcedureContext procedureContext, String tableId, String olderThan) - throws Exception { - return call(procedureContext, tableId, olderThan, false); - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint( + name = "older_than", + type = @DataTypeHint("STRING"), + isOptional = true), + @ArgumentHint(name = "dry_run", type = @DataTypeHint("BOOLEAN"), isOptional = true) + }) public String[] call( - ProcedureContext procedureContext, String tableId, String olderThan, boolean dryRun) + ProcedureContext procedureContext, + String tableId, + String nullableOlderThan, + Boolean dryRun) throws Exception { + final String olderThan = notnull(nullableOlderThan); + if (dryRun == null) { + dryRun = false; + } + Identifier identifier = Identifier.fromString(tableId); String databaseName = identifier.getDatabaseName(); String tableName = identifier.getObjectName(); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RepairProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RepairProcedure.java index 196adf4753cc..7bf4a8ac521d 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RepairProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RepairProcedure.java @@ -22,6 +22,9 @@ import org.apache.paimon.catalog.Identifier; import org.apache.paimon.utils.StringUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -47,11 +50,10 @@ public String identifier() { return IDENTIFIER; } - public String[] call(ProcedureContext procedureContext) - throws Catalog.TableNotExistException, Catalog.DatabaseNotExistException { - return call(procedureContext, null); - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING"), isOptional = true) + }) public String[] call(ProcedureContext procedureContext, String identifier) throws Catalog.DatabaseNotExistException, Catalog.TableNotExistException { if (StringUtils.isBlank(identifier)) { diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ResetConsumerProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ResetConsumerProcedure.java index 0355d6dc1cab..5bd4cbaafac6 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ResetConsumerProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/ResetConsumerProcedure.java @@ -24,6 +24,9 @@ import org.apache.paimon.consumer.ConsumerManager; import org.apache.paimon.table.FileStoreTable; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -41,11 +44,20 @@ public class ResetConsumerProcedure extends ProcedureBase { public static final String IDENTIFIER = "reset_consumer"; + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "consumer_id", type = @DataTypeHint("STRING")), + @ArgumentHint( + name = "next_snapshot_id", + type = @DataTypeHint("BIGINT"), + isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String tableId, String consumerId, - long nextSnapshotId) + Long nextSnapshotId) throws Catalog.TableNotExistException { FileStoreTable fileStoreTable = (FileStoreTable) catalog.getTable(Identifier.fromString(tableId)); @@ -54,21 +66,11 @@ public String[] call( fileStoreTable.fileIO(), fileStoreTable.location(), fileStoreTable.snapshotManager().branch()); - consumerManager.resetConsumer(consumerId, new Consumer(nextSnapshotId)); - - return new String[] {"Success"}; - } - - public String[] call(ProcedureContext procedureContext, String tableId, String consumerId) - throws Catalog.TableNotExistException { - FileStoreTable fileStoreTable = - (FileStoreTable) catalog.getTable(Identifier.fromString(tableId)); - ConsumerManager consumerManager = - new ConsumerManager( - fileStoreTable.fileIO(), - fileStoreTable.location(), - fileStoreTable.snapshotManager().branch()); - consumerManager.deleteConsumer(consumerId); + if (nextSnapshotId != null) { + consumerManager.resetConsumer(consumerId, new Consumer(nextSnapshotId)); + } else { + consumerManager.deleteConsumer(consumerId); + } return new String[] {"Success"}; } diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedure.java index 64aa23964547..7bb354a7740d 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedure.java @@ -38,6 +38,9 @@ import org.apache.flink.core.io.SimpleVersionedSerializer; import org.apache.flink.streaming.api.datastream.DataStreamSource; import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; import java.io.IOException; @@ -54,14 +57,18 @@ public String identifier() { return "rewrite_file_index"; } - public String[] call(ProcedureContext procedureContext, String sourceTablePath) - throws Exception { - return call(procedureContext, sourceTablePath, ""); - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint( + name = "partitions", + type = @DataTypeHint("STRING"), + isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String sourceTablePath, String partitions) throws Exception { + partitions = notnull(partitions); StreamExecutionEnvironment env = procedureContext.getExecutionEnvironment(); Table table = catalog.getTable(Identifier.fromString(sourceTablePath)); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RollbackToProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RollbackToProcedure.java index 1bf545004d93..5663c221975f 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RollbackToProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/RollbackToProcedure.java @@ -21,7 +21,11 @@ import org.apache.paimon.catalog.Catalog; import org.apache.paimon.catalog.Identifier; import org.apache.paimon.table.Table; +import org.apache.paimon.utils.StringUtils; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -29,29 +33,34 @@ * *

    
      *  -- rollback to a snapshot
    - *  CALL sys.rollback_to('tableId', snapshotId)
    + *  CALL sys.rollback_to(`table` => 'tableId', snapshot_id => snapshotId)
      *
      *  -- rollback to a tag
    - *  CALL sys.rollback_to('tableId', 'tagName')
    + *  CALL sys.rollback_to(`table` => 'tableId', tag => 'tagName')
      * 
    */ public class RollbackToProcedure extends ProcedureBase { public static final String IDENTIFIER = "rollback_to"; - public String[] call(ProcedureContext procedureContext, String tableId, long snapshotId) + @ProcedureHint( + argument = { + @ArgumentHint(name = "table", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "tag", type = @DataTypeHint("STRING"), isOptional = true), + @ArgumentHint( + name = "snapshot_id", + type = @DataTypeHint("BIGINT"), + isOptional = true) + }) + public String[] call( + ProcedureContext procedureContext, String tableId, String tagName, Long snapshotId) throws Catalog.TableNotExistException { Table table = catalog.getTable(Identifier.fromString(tableId)); - table.rollbackTo(snapshotId); - - return new String[] {"Success"}; - } - - public String[] call(ProcedureContext procedureContext, String tableId, String tagName) - throws Catalog.TableNotExistException { - Table table = catalog.getTable(Identifier.fromString(tableId)); - table.rollbackTo(tagName); - + if (!StringUtils.isBlank(tagName)) { + table.rollbackTo(tagName); + } else { + table.rollbackTo(snapshotId); + } return new String[] {"Success"}; } diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/CreatePrivilegedUserProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/CreatePrivilegedUserProcedure.java index 76b1f6af8edb..788020264e57 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/CreatePrivilegedUserProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/CreatePrivilegedUserProcedure.java @@ -18,6 +18,9 @@ package org.apache.paimon.flink.procedure.privilege; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -32,6 +35,11 @@ public class CreatePrivilegedUserProcedure extends PrivilegeProcedureBase { public static final String IDENTIFIER = "create_privileged_user"; + @ProcedureHint( + argument = { + @ArgumentHint(name = "username", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "password", type = @DataTypeHint("STRING")) + }) public String[] call(ProcedureContext procedureContext, String name, String password) { getPrivilegedCatalog().createPrivilegedUser(name, password); return new String[] {String.format("User %s is created without any privileges.", name)}; diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/DropPrivilegedUserProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/DropPrivilegedUserProcedure.java index 3902a7f1949a..debec2a8b005 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/DropPrivilegedUserProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/DropPrivilegedUserProcedure.java @@ -18,6 +18,9 @@ package org.apache.paimon.flink.procedure.privilege; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -32,6 +35,7 @@ public class DropPrivilegedUserProcedure extends PrivilegeProcedureBase { public static final String IDENTIFIER = "drop_privileged_user"; + @ProcedureHint(argument = {@ArgumentHint(name = "username", type = @DataTypeHint("STRING"))}) public String[] call(ProcedureContext procedureContext, String name) { getPrivilegedCatalog().dropPrivilegedUser(name); return new String[] {String.format("User %s is dropped.", name)}; diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/GrantPrivilegeToUserProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/GrantPrivilegeToUserProcedure.java index a647f36bcfce..8ddc8b6229f9 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/GrantPrivilegeToUserProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/GrantPrivilegeToUserProcedure.java @@ -20,7 +20,11 @@ import org.apache.paimon.catalog.Identifier; import org.apache.paimon.privilege.PrivilegeType; +import org.apache.paimon.utils.Preconditions; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -38,38 +42,46 @@ public class GrantPrivilegeToUserProcedure extends PrivilegeProcedureBase { public static final String IDENTIFIER = "grant_privilege_to_user"; - public String[] call(ProcedureContext procedureContext, String user, String privilege) { - getPrivilegedCatalog().grantPrivilegeOnCatalog(user, PrivilegeType.valueOf(privilege)); - return new String[] { - String.format("User %s is granted with privilege %s on the catalog.", user, privilege) - }; - } - - public String[] call( - ProcedureContext procedureContext, String user, String privilege, String database) { - getPrivilegedCatalog() - .grantPrivilegeOnDatabase(user, database, PrivilegeType.valueOf(privilege)); - return new String[] { - String.format( - "User %s is granted with privilege %s on database %s.", - user, privilege, database) - }; - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "username", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "privilege", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "database", type = @DataTypeHint("STRING"), isOptional = true), + @ArgumentHint(name = "table", type = @DataTypeHint("STRING"), isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String user, String privilege, String database, String table) { - Identifier identifier = Identifier.create(database, table); - getPrivilegedCatalog() - .grantPrivilegeOnTable(user, identifier, PrivilegeType.valueOf(privilege)); - return new String[] { - String.format( - "User %s is granted with privilege %s on table %s.", - user, privilege, identifier) - }; + if (database == null) { + Preconditions.checkState( + table == null, + "database must be set if privilege is granted at table's granularity."); + getPrivilegedCatalog().grantPrivilegeOnCatalog(user, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is granted with privilege %s on the catalog.", user, privilege) + }; + } else if (table == null) { + getPrivilegedCatalog() + .grantPrivilegeOnDatabase(user, database, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is granted with privilege %s on database %s.", + user, privilege, database) + }; + } else { + Identifier identifier = Identifier.create(database, table); + getPrivilegedCatalog() + .grantPrivilegeOnTable(user, identifier, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is granted with privilege %s on table %s.", + user, privilege, identifier) + }; + } } @Override diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/InitFileBasedPrivilegeProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/InitFileBasedPrivilegeProcedure.java index 35d42ce32c98..40c58a1c8a51 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/InitFileBasedPrivilegeProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/InitFileBasedPrivilegeProcedure.java @@ -24,6 +24,9 @@ import org.apache.paimon.privilege.PrivilegeManager; import org.apache.paimon.privilege.PrivilegedCatalog; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -38,6 +41,8 @@ public class InitFileBasedPrivilegeProcedure extends ProcedureBase { public static final String IDENTIFIER = "init_file_based_privilege"; + @ProcedureHint( + argument = {@ArgumentHint(name = "root_password", type = @DataTypeHint("STRING"))}) public String[] call(ProcedureContext procedureContext, String rootPassword) { if (catalog instanceof PrivilegedCatalog) { throw new IllegalArgumentException("Catalog is already a PrivilegedCatalog"); diff --git a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/RevokePrivilegeFromUserProcedure.java b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/RevokePrivilegeFromUserProcedure.java index d4834ac60885..e8d057ca3007 100644 --- a/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/RevokePrivilegeFromUserProcedure.java +++ b/paimon-flink/paimon-flink-common/src/main/java/org/apache/paimon/flink/procedure/privilege/RevokePrivilegeFromUserProcedure.java @@ -20,7 +20,11 @@ import org.apache.paimon.catalog.Identifier; import org.apache.paimon.privilege.PrivilegeType; +import org.apache.paimon.utils.Preconditions; +import org.apache.flink.table.annotation.ArgumentHint; +import org.apache.flink.table.annotation.DataTypeHint; +import org.apache.flink.table.annotation.ProcedureHint; import org.apache.flink.table.procedure.ProcedureContext; /** @@ -38,46 +42,55 @@ public class RevokePrivilegeFromUserProcedure extends PrivilegeProcedureBase { public static final String IDENTIFIER = "revoke_privilege_from_user"; - public String[] call(ProcedureContext procedureContext, String user, String privilege) { - int count = - getPrivilegedCatalog() - .revokePrivilegeOnCatalog(user, PrivilegeType.valueOf(privilege)); - return new String[] { - String.format("User %s is revoked with privilege %s on the catalog.", user, privilege), - "Number of privileges revoked: " + count - }; - } - - public String[] call( - ProcedureContext procedureContext, String user, String privilege, String database) { - int count = - getPrivilegedCatalog() - .revokePrivilegeOnDatabase( - user, database, PrivilegeType.valueOf(privilege)); - return new String[] { - String.format( - "User %s is revoked with privilege %s on database %s.", - user, privilege, database), - "Number of privileges revoked: " + count - }; - } - + @ProcedureHint( + argument = { + @ArgumentHint(name = "username", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "privilege", type = @DataTypeHint("STRING")), + @ArgumentHint(name = "database", type = @DataTypeHint("STRING"), isOptional = true), + @ArgumentHint(name = "table", type = @DataTypeHint("STRING"), isOptional = true) + }) public String[] call( ProcedureContext procedureContext, String user, String privilege, String database, String table) { - Identifier identifier = Identifier.create(database, table); - int count = - getPrivilegedCatalog() - .revokePrivilegeOnTable(user, identifier, PrivilegeType.valueOf(privilege)); - return new String[] { - String.format( - "User %s is revoked with privilege %s on table %s.", - user, privilege, identifier), - "Number of privileges revoked: " + count - }; + if (database == null) { + Preconditions.checkState( + table == null, + "database must be set if privilege is granted at table's granularity."); + int count = + getPrivilegedCatalog() + .revokePrivilegeOnCatalog(user, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is revoked with privilege %s on the catalog.", user, privilege), + "Number of privileges revoked: " + count + }; + } else if (table == null) { + int count = + getPrivilegedCatalog() + .revokePrivilegeOnDatabase( + user, database, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is revoked with privilege %s on database %s.", + user, privilege, database), + "Number of privileges revoked: " + count + }; + } else { + Identifier identifier = Identifier.create(database, table); + int count = + getPrivilegedCatalog() + .revokePrivilegeOnTable( + user, identifier, PrivilegeType.valueOf(privilege)); + return new String[] { + String.format( + "User %s is revoked with privilege %s on table %s.", + user, privilege, identifier), + "Number of privileges revoked: " + count + }; + } } @Override diff --git a/paimon-flink/paimon-flink-common/src/main/resources/META-INF/services/org.apache.paimon.factories.Factory b/paimon-flink/paimon-flink-common/src/main/resources/META-INF/services/org.apache.paimon.factories.Factory index d268add737de..e68bf251f493 100644 --- a/paimon-flink/paimon-flink-common/src/main/resources/META-INF/services/org.apache.paimon.factories.Factory +++ b/paimon-flink/paimon-flink-common/src/main/resources/META-INF/services/org.apache.paimon.factories.Factory @@ -68,3 +68,4 @@ org.apache.paimon.flink.procedure.privilege.RevokePrivilegeFromUserProcedure org.apache.paimon.flink.procedure.RepairProcedure org.apache.paimon.flink.procedure.FastForwardProcedure org.apache.paimon.flink.procedure.MarkPartitionDoneProcedure +org.apache.paimon.flink.procedure.CloneProcedure diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/RemoteLookupJoinITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/RemoteLookupJoinITCase.java index 37233552003f..df9a7ed59be1 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/RemoteLookupJoinITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/RemoteLookupJoinITCase.java @@ -41,6 +41,8 @@ import org.apache.flink.util.CloseableIterator; import org.junit.jupiter.api.Disabled; import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.io.Closeable; import java.io.IOException; @@ -66,11 +68,16 @@ protected int defaultParallelism() { return 1; } - @Test - public void testQueryServiceLookup() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testQueryServiceLookup(boolean isNamedArgument) throws Exception { sql( "CREATE TABLE DIM (k INT PRIMARY KEY NOT ENFORCED, v INT) WITH ('bucket' = '2', 'continuous.discovery-interval' = '1ms')"); - CloseableIterator service = streamSqlIter("CALL sys.query_service('default.DIM', 2)"); + CloseableIterator service = + isNamedArgument + ? streamSqlIter( + "CALL sys.query_service(`table` => 'default.DIM', parallelism => 2)") + : streamSqlIter("CALL sys.query_service('default.DIM', 2)"); RemoteTableQuery query = new RemoteTableQuery(paimonTable("DIM")); sql("INSERT INTO DIM VALUES (1, 11), (2, 22), (3, 33), (4, 44)"); diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/BranchActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/BranchActionITCase.java index 7b92f0e3c551..261515e8719f 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/BranchActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/BranchActionITCase.java @@ -83,11 +83,23 @@ void testCreateAndDeleteBranch() throws Exception { database, tableName)); assertThat(branchManager.branchExists("branch_name")).isTrue(); + callProcedure( + String.format( + "CALL sys.create_branch(`table` => '%s.%s', branch => 'branch_name_named_argument', tag => 'tag2')", + database, tableName)); + assertThat(branchManager.branchExists("branch_name_named_argument")).isTrue(); + callProcedure( String.format( "CALL sys.delete_branch('%s.%s', 'branch_name')", database, tableName)); assertThat(branchManager.branchExists("branch_name")).isFalse(); + callProcedure( + String.format( + "CALL sys.delete_branch(`table` => '%s.%s', branch => 'branch_name_named_argument')", + database, tableName)); + assertThat(branchManager.branchExists("branch_name_named_argument")).isFalse(); + createAction( CreateBranchAction.class, "create_branch", @@ -152,12 +164,24 @@ void testCreateAndDeleteEmptyBranch() throws Exception { database, tableName)); assertThat(branchManager.branchExists("empty_branch_name")).isTrue(); + callProcedure( + String.format( + "CALL sys.create_branch(`table` => '%s.%s', branch => 'empty_branch_named_argument')", + database, tableName)); + assertThat(branchManager.branchExists("empty_branch_named_argument")).isTrue(); + callProcedure( String.format( "CALL sys.delete_branch('%s.%s', 'empty_branch_name')", database, tableName)); assertThat(branchManager.branchExists("empty_branch_name")).isFalse(); + callProcedure( + String.format( + "CALL sys.delete_branch(`table` => '%s.%s', branch => 'empty_branch_named_argument')", + database, tableName)); + assertThat(branchManager.branchExists("empty_branch_named_argument")).isFalse(); + createAction( CreateBranchAction.class, "create_branch", @@ -303,7 +327,8 @@ void testFastForward() throws Exception { // Fast-forward branch branch_name again callProcedure( String.format( - "CALL sys.fast_forward('%s.%s', 'branch_name')", database, tableName)); + "CALL sys.fast_forward(`table` => '%s.%s', branch => 'branch_name')", + database, tableName)); // Check main branch data result = readTableData(table); diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CloneActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CloneActionITCase.java index 74cb44cbe33d..82ac7e58eb66 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CloneActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CloneActionITCase.java @@ -23,17 +23,18 @@ import org.apache.paimon.catalog.CatalogFactory; import org.apache.paimon.catalog.Identifier; import org.apache.paimon.flink.clone.PickFilesUtil; -import org.apache.paimon.flink.util.AbstractTestBase; import org.apache.paimon.fs.Path; import org.apache.paimon.table.FileStoreTable; import org.apache.paimon.utils.Pair; import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; import org.apache.flink.table.api.TableEnvironment; +import org.apache.flink.table.api.config.TableConfigOptions; import org.apache.flink.types.Row; import org.apache.flink.util.CloseableIterator; -import org.junit.jupiter.api.Test; import org.junit.jupiter.api.Timeout; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.util.ArrayList; import java.util.Collections; @@ -47,35 +48,58 @@ import static org.assertj.core.api.Assertions.assertThat; /** IT cases for {@link CloneAction}. */ -public class CloneActionITCase extends AbstractTestBase { +public class CloneActionITCase extends ActionITCaseBase { // ------------------------------------------------------------------------ // Constructed Tests // ------------------------------------------------------------------------ - @Test - public void testCloneTable() throws Exception { + @ParameterizedTest(name = "invoker = {0}") + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testCloneTable(String invoker) throws Exception { String sourceWarehouse = getTempDirPath("source-ware"); prepareData(sourceWarehouse); String targetWarehouse = getTempDirPath("target-ware"); - String[] args = - new String[] { - "clone", - "--warehouse", - sourceWarehouse, - "--database", - "db1", - "--table", - "t1", - "--target_warehouse", - targetWarehouse, - "--target_database", - "mydb", - "--target_table", - "myt" - }; - ActionFactory.createAction(args).get().run(); + switch (invoker) { + case "action": + String[] args = + new String[] { + "clone", + "--warehouse", + sourceWarehouse, + "--database", + "db1", + "--table", + "t1", + "--target_warehouse", + targetWarehouse, + "--target_database", + "mydb", + "--target_table", + "myt" + }; + ActionFactory.createAction(args).get().run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.clone('%s', 'db1', 't1', '', '%s', 'mydb', 'myt')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.clone(warehouse => '%s', database => 'db1', `table` => 't1', target_warehouse => '%s', target_database => 'mydb', target_table => 'myt')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + default: + throw new UnsupportedOperationException(invoker); + } // check result TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); @@ -93,25 +117,48 @@ public void testCloneTable() throws Exception { compareCloneFiles(sourceWarehouse, "db1", "t1", targetWarehouse, "mydb", "myt"); } - @Test - public void testCloneDatabase() throws Exception { + @ParameterizedTest(name = "invoker = {0}") + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testCloneDatabase(String invoker) throws Exception { String sourceWarehouse = getTempDirPath("source-ware"); prepareData(sourceWarehouse); String targetWarehouse = getTempDirPath("target-ware"); - String[] args = - new String[] { - "clone", - "--warehouse", - sourceWarehouse, - "--database", - "db1", - "--target_warehouse", - targetWarehouse, - "--target_database", - "mydb" - }; - ActionFactory.createAction(args).get().run(); + switch (invoker) { + case "action": + String[] args = + new String[] { + "clone", + "--warehouse", + sourceWarehouse, + "--database", + "db1", + "--target_warehouse", + targetWarehouse, + "--target_database", + "mydb" + }; + ActionFactory.createAction(args).get().run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.clone('%s', 'db1', '', '', '%s', 'mydb')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.clone(warehouse => '%s', database => 'db1', target_warehouse => '%s', target_database => 'mydb')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + default: + throw new UnsupportedOperationException(invoker); + } // check result TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); @@ -134,17 +181,44 @@ public void testCloneDatabase() throws Exception { compareCloneFiles(sourceWarehouse, "db1", "t2", targetWarehouse, "mydb", "t2"); } - @Test - public void testCloneWarehouse() throws Exception { + @ParameterizedTest(name = "invoker = {0}") + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testCloneWarehouse(String invoker) throws Exception { String sourceWarehouse = getTempDirPath("source-ware"); prepareData(sourceWarehouse); String targetWarehouse = getTempDirPath("target-ware"); - String[] args = - new String[] { - "clone", "--warehouse", sourceWarehouse, "--target_warehouse", targetWarehouse - }; - ActionFactory.createAction(args).get().run(); + switch (invoker) { + case "action": + String[] args = + new String[] { + "clone", + "--warehouse", + sourceWarehouse, + "--target_warehouse", + targetWarehouse + }; + ActionFactory.createAction(args).get().run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.clone('%s', '', '', '', '%s')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.clone(warehouse => '%s', target_warehouse => '%s')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + default: + throw new UnsupportedOperationException(invoker); + } // check result TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); @@ -285,8 +359,9 @@ private void prepareData(String sourceWarehouse) throws Exception { .await(); } - @Test - public void testCloneWithSchemaEvolution() throws Exception { + @ParameterizedTest(name = "invoker = {0}") + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testCloneWithSchemaEvolution(String invoker) throws Exception { String sourceWarehouse = getTempDirPath("source-ware"); TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql( @@ -320,11 +395,37 @@ public void testCloneWithSchemaEvolution() throws Exception { .await(); String targetWarehouse = getTempDirPath("target-ware"); - String[] args = - new String[] { - "clone", "--warehouse", sourceWarehouse, "--target_warehouse", targetWarehouse - }; - ActionFactory.createAction(args).get().run(); + switch (invoker) { + case "action": + String[] args = + new String[] { + "clone", + "--warehouse", + sourceWarehouse, + "--target_warehouse", + targetWarehouse + }; + ActionFactory.createAction(args).get().run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.clone('%s', '', '', '', '%s')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.clone(warehouse => '%s', target_warehouse => '%s')", + sourceWarehouse, targetWarehouse), + true, + true); + break; + default: + throw new UnsupportedOperationException(invoker); + } // check result tEnv.executeSql( @@ -400,9 +501,10 @@ private FileStoreTable getFileStoreTable(String warehouse, String db, String tab // Random Tests // ------------------------------------------------------------------------ - @Test + @ParameterizedTest(name = "invoker = {0}") + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) @Timeout(180) - public void testCloneTableWithExpiration() throws Exception { + public void testCloneTableWithExpiration(String invoker) throws Exception { String sourceWarehouse = getTempDirPath("source-ware"); TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().parallelism(1).build(); @@ -478,24 +580,47 @@ public void testCloneTableWithExpiration() throws Exception { Thread.sleep(ThreadLocalRandom.current().nextInt(2000)); String targetWarehouse = getTempDirPath("target-ware"); - String[] args = - new String[] { - "clone", - "--warehouse", - // special file io to make cloning slower, thus more likely to face - // FileNotFoundException, see CloneActionSlowFileIO - "clone-slow://" + sourceWarehouse, - "--target_warehouse", - "clone-slow://" + targetWarehouse, - "--parallelism", - "1" - }; - CloneAction action = (CloneAction) ActionFactory.createAction(args).get(); + switch (invoker) { + case "action": + String[] args = + new String[] { + "clone", + "--warehouse", + // special file io to make cloning slower, thus more likely to face + // FileNotFoundException, see CloneActionSlowFileIO + "clone-slow://" + sourceWarehouse, + "--target_warehouse", + "clone-slow://" + targetWarehouse, + "--parallelism", + "1" + }; + CloneAction action = (CloneAction) ActionFactory.createAction(args).get(); + + StreamExecutionEnvironment env = + streamExecutionEnvironmentBuilder().streamingMode().allowRestart().build(); + action.withStreamExecutionEnvironment(env).build(); + env.execute(); + break; + case "procedure_indexed": + callProcedureWithRestartAllowed( + String.format( + "CALL sys.clone('clone-slow://%s', '', '', '', 'clone-slow://%s', '', '', '', 1)", + sourceWarehouse, targetWarehouse), + true, + true); + break; + case "procedure_named": + callProcedureWithRestartAllowed( + String.format( + "CALL sys.clone(warehouse => 'clone-slow://%s', target_warehouse => 'clone-slow://%s', parallelism => 1)", + sourceWarehouse, targetWarehouse), + true, + true); + break; + default: + throw new UnsupportedOperationException(invoker); + } - StreamExecutionEnvironment env = - streamExecutionEnvironmentBuilder().streamingMode().allowRestart().build(); - action.withStreamExecutionEnvironment(env).build(); - env.execute(); running.set(false); thread.join(); @@ -519,6 +644,31 @@ public void testCloneTableWithExpiration() throws Exception { // Utils // ------------------------------------------------------------------------ + private CloseableIterator callProcedureWithRestartAllowed( + String procedureStatement, boolean isStreaming, boolean dmlSync) { + TableEnvironment tEnv; + if (isStreaming) { + tEnv = + tableEnvironmentBuilder() + .streamingMode() + .allowRestart() + .checkpointIntervalMs(500) + .build(); + } else { + tEnv = tableEnvironmentBuilder().batchMode().allowRestart().build(); + } + + tEnv.getConfig().set(TableConfigOptions.TABLE_DML_SYNC, dmlSync); + + tEnv.executeSql( + String.format( + "CREATE CATALOG PAIMON WITH ('type'='paimon', 'warehouse'='%s');", + warehouse)); + tEnv.useCatalog("PAIMON"); + + return tEnv.executeSql(procedureStatement).collect(); + } + private List collect(TableEnvironment tEnv, String sql) throws Exception { List actual = new ArrayList<>(); try (CloseableIterator it = tEnv.executeSql(sql).collect()) { diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CompactDatabaseActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CompactDatabaseActionITCase.java index c5008b216c7d..5e346499ef75 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CompactDatabaseActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/CompactDatabaseActionITCase.java @@ -44,7 +44,8 @@ import org.junit.jupiter.api.Test; import org.junit.jupiter.api.Timeout; import org.junit.jupiter.params.ParameterizedTest; -import org.junit.jupiter.params.provider.ValueSource; +import org.junit.jupiter.params.provider.Arguments; +import org.junit.jupiter.params.provider.MethodSource; import javax.annotation.Nullable; @@ -56,6 +57,7 @@ import java.util.List; import java.util.Map; import java.util.concurrent.ThreadLocalRandom; +import java.util.stream.Stream; import static org.apache.paimon.utils.CommonTestUtils.waitUtil; import static org.assertj.core.api.Assertions.assertThat; @@ -74,6 +76,16 @@ public class CompactDatabaseActionITCase extends CompactActionITCaseBase { }, new String[] {"k", "v", "hh", "dt"}); + private static Stream testData() { + return Stream.of( + Arguments.of("combined", "action"), + Arguments.of("divided", "action"), + Arguments.of("combined", "procedure_indexed"), + Arguments.of("divided", "procedure_indexed"), + Arguments.of("combined", "procedure_named"), + Arguments.of("divided", "procedure_named")); + } + private FileStoreTable createTable( String databaseName, String tableName, @@ -90,10 +102,10 @@ private FileStoreTable createTable( return (FileStoreTable) catalog.getTable(identifier); } - @ParameterizedTest(name = "mode = {0}") - @ValueSource(strings = {"combined", "divided"}) + @ParameterizedTest(name = "mode = {0}, invoker = {1}") + @MethodSource("testData") @Timeout(6000) - public void testStreamCompactForUnawareTable(String mode) throws Exception { + public void testStreamCompactForUnawareTable(String mode, String invoker) throws Exception { // step0. create tables Map tableToCompaction = new HashMap<>(); @@ -115,21 +127,33 @@ public void testStreamCompactForUnawareTable(String mode) throws Exception { } // step1. run streaming compaction task for tables - if (ThreadLocalRandom.current().nextBoolean()) { - StreamExecutionEnvironment env = - streamExecutionEnvironmentBuilder().streamingMode().build(); - createAction( - CompactDatabaseAction.class, - "compact_database", - "--warehouse", - warehouse, - "--mode", - mode) - .withStreamExecutionEnvironment(env) - .build(); - env.executeAsync(); - } else { - callProcedure(String.format("CALL sys.compact_database('', '%s')", mode), true, false); + switch (invoker) { + case "action": + StreamExecutionEnvironment env = + streamExecutionEnvironmentBuilder().streamingMode().build(); + createAction( + CompactDatabaseAction.class, + "compact_database", + "--warehouse", + warehouse, + "--mode", + mode) + .withStreamExecutionEnvironment(env) + .build(); + env.executeAsync(); + break; + case "procedure_indexed": + callProcedure( + String.format("CALL sys.compact_database('', '%s')", mode), true, false); + break; + case "procedure_named": + callProcedure( + String.format("CALL sys.compact_database(mode => '%s')", mode), + true, + false); + break; + default: + throw new UnsupportedOperationException(invoker); } // step3. write datas to table wait for compaction @@ -185,10 +209,10 @@ public void testStreamCompactForUnawareTable(String mode) throws Exception { } } - @ParameterizedTest(name = "mode = {0}") - @ValueSource(strings = {"divided", "combined"}) + @ParameterizedTest(name = "mode = {0}, invoker = {1}") + @MethodSource("testData") @Timeout(60) - public void testBatchCompact(String mode) throws Exception { + public void testBatchCompact(String mode, String invoker) throws Exception { List tables = new ArrayList<>(); for (String dbName : DATABASE_NAMES) { @@ -232,21 +256,33 @@ public void testBatchCompact(String mode) throws Exception { } } - if (ThreadLocalRandom.current().nextBoolean()) { - StreamExecutionEnvironment env = - streamExecutionEnvironmentBuilder().batchMode().build(); - createAction( - CompactDatabaseAction.class, - "compact_database", - "--warehouse", - warehouse, - "--mode", - mode) - .withStreamExecutionEnvironment(env) - .build(); - env.execute(); - } else { - callProcedure(String.format("CALL sys.compact_database('', '%s')", mode), false, true); + switch (invoker) { + case "action": + StreamExecutionEnvironment env = + streamExecutionEnvironmentBuilder().batchMode().build(); + createAction( + CompactDatabaseAction.class, + "compact_database", + "--warehouse", + warehouse, + "--mode", + mode) + .withStreamExecutionEnvironment(env) + .build(); + env.execute(); + break; + case "procedure_indexed": + callProcedure( + String.format("CALL sys.compact_database('', '%s')", mode), false, true); + break; + case "procedure_named": + callProcedure( + String.format("CALL sys.compact_database(mode => '%s')", mode), + false, + true); + break; + default: + throw new UnsupportedOperationException(invoker); } for (FileStoreTable table : tables) { @@ -264,9 +300,9 @@ public void testBatchCompact(String mode) throws Exception { } } - @ParameterizedTest(name = "mode = {0}") - @ValueSource(strings = {"divided", "combined"}) - public void testStreamingCompact(String mode) throws Exception { + @ParameterizedTest(name = "mode = {0}, invoker = {1}") + @MethodSource("testData") + public void testStreamingCompact(String mode, String invoker) throws Exception { Map options = new HashMap<>(); options.put(CoreOptions.CHANGELOG_PRODUCER.key(), "full-compaction"); options.put( @@ -315,42 +351,58 @@ public void testStreamingCompact(String mode) throws Exception { } } - if (ThreadLocalRandom.current().nextBoolean()) { - CompactDatabaseAction action; - if (mode.equals("divided")) { - action = - createAction( - CompactDatabaseAction.class, - "compact_database", - "--warehouse", - warehouse); - } else { - // if CoreOptions.CONTINUOUS_DISCOVERY_INTERVAL.key() use default value, the cost - // time in combined mode will be over 1 min - action = - createAction( - CompactDatabaseAction.class, - "compact_database", - "--warehouse", - warehouse, - "--mode", - "combined", - "--table_conf", - CoreOptions.CONTINUOUS_DISCOVERY_INTERVAL.key() + "=1s"); - } - StreamExecutionEnvironment env = - streamExecutionEnvironmentBuilder().streamingMode().build(); - action.withStreamExecutionEnvironment(env).build(); - env.executeAsync(); - } else { - if (mode.equals("divided")) { - callProcedure("CALL sys.compact_database()", true, false); - } else { - callProcedure( - "CALL sys.compact_database('', 'combined', '', '', 'continuous.discovery-interval=1s')", - true, - false); - } + switch (invoker) { + case "action": + CompactDatabaseAction action; + if (mode.equals("divided")) { + action = + createAction( + CompactDatabaseAction.class, + "compact_database", + "--warehouse", + warehouse); + } else { + // if CoreOptions.CONTINUOUS_DISCOVERY_INTERVAL.key() use default value, the + // cost + // time in combined mode will be over 1 min + action = + createAction( + CompactDatabaseAction.class, + "compact_database", + "--warehouse", + warehouse, + "--mode", + "combined", + "--table_conf", + CoreOptions.CONTINUOUS_DISCOVERY_INTERVAL.key() + "=1s"); + } + StreamExecutionEnvironment env = + streamExecutionEnvironmentBuilder().streamingMode().build(); + action.withStreamExecutionEnvironment(env).build(); + env.executeAsync(); + break; + case "procedure_indexed": + if (mode.equals("divided")) { + callProcedure("CALL sys.compact_database()", true, false); + } else { + callProcedure( + "CALL sys.compact_database('', 'combined', '', '', 'continuous.discovery-interval=1s')", + true, + false); + } + break; + case "procedure_named": + if (mode.equals("divided")) { + callProcedure("CALL sys.compact_database()", true, false); + } else { + callProcedure( + "CALL sys.compact_database(mode => 'combined', table_options => 'continuous.discovery-interval=1s')", + true, + false); + } + break; + default: + throw new UnsupportedOperationException(invoker); } for (FileStoreTable table : tables) { @@ -500,10 +552,10 @@ public void testStreamingCompact(String mode) throws Exception { } } - @ParameterizedTest(name = "mode = {0}") - @ValueSource(strings = {"divided", "combined"}) + @ParameterizedTest(name = "mode = {0}, invoker = {1}") + @MethodSource("testData") @Timeout(60) - public void testHistoryPartitionCompact(String mode) throws Exception { + public void testHistoryPartitionCompact(String mode, String invoker) throws Exception { List tables = new ArrayList<>(); String partitionIdleTime = "10s"; @@ -558,28 +610,41 @@ public void testHistoryPartitionCompact(String mode) throws Exception { writeData(rowData(3, 100, 16, BinaryString.fromString("20221208"))); } - if (ThreadLocalRandom.current().nextBoolean()) { - StreamExecutionEnvironment env = - streamExecutionEnvironmentBuilder().batchMode().build(); - createAction( - CompactDatabaseAction.class, - "compact_database", - "--warehouse", - warehouse, - "--mode", - mode, - "--partition_idle_time", - partitionIdleTime) - .withStreamExecutionEnvironment(env) - .build(); - env.execute(); - } else { - callProcedure( - String.format( - "CALL sys.compact_database('', '%s','','','','%s')", - mode, partitionIdleTime), - false, - true); + switch (invoker) { + case "action": + StreamExecutionEnvironment env = + streamExecutionEnvironmentBuilder().batchMode().build(); + createAction( + CompactDatabaseAction.class, + "compact_database", + "--warehouse", + warehouse, + "--mode", + mode, + "--partition_idle_time", + partitionIdleTime) + .withStreamExecutionEnvironment(env) + .build(); + env.execute(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.compact_database('', '%s','','','','%s')", + mode, partitionIdleTime), + false, + true); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.compact_database(mode => '%s', partition_idle_time => '%s')", + mode, partitionIdleTime), + false, + true); + break; + default: + throw new UnsupportedOperationException(invoker); } for (FileStoreTable table : tables) { @@ -602,11 +667,12 @@ public void testHistoryPartitionCompact(String mode) throws Exception { } @ParameterizedTest(name = "mode = {0}") - @ValueSource(strings = {"divided", "combined"}) + @MethodSource("testData") @Timeout(60) - public void includeTableCompaction(String mode) throws Exception { + public void includeTableCompaction(String mode, String invoker) throws Exception { includingAndExcludingTablesImpl( mode, + invoker, "db1.t1", null, Collections.singletonList(Identifier.fromString("db1.t1")), @@ -616,12 +682,13 @@ public void includeTableCompaction(String mode) throws Exception { Identifier.fromString("db2.t2"))); } - @ParameterizedTest(name = "mode = {0}") - @ValueSource(strings = {"divided", "combined"}) + @ParameterizedTest(name = "mode = {0}, invoker = {1}") + @MethodSource("testData") @Timeout(60) - public void excludeTableCompaction(String mode) throws Exception { + public void excludeTableCompaction(String mode, String invoker) throws Exception { includingAndExcludingTablesImpl( mode, + invoker, null, "db2.t2", Arrays.asList( @@ -631,12 +698,13 @@ public void excludeTableCompaction(String mode) throws Exception { Collections.singletonList(Identifier.fromString("db2.t2"))); } - @ParameterizedTest(name = "mode = {0}") - @ValueSource(strings = {"divided", "combined"}) + @ParameterizedTest(name = "mode = {0}, invoker = {1}") + @MethodSource("testData") @Timeout(60) - public void includeAndExcludeTableCompaction(String mode) throws Exception { + public void includeAndExcludeTableCompaction(String mode, String invoker) throws Exception { includingAndExcludingTablesImpl( mode, + invoker, "db1.+|db2.t1", "db1.t2", Arrays.asList(Identifier.fromString("db1.t1"), Identifier.fromString("db2.t1")), @@ -645,6 +713,7 @@ public void includeAndExcludeTableCompaction(String mode) throws Exception { private void includingAndExcludingTablesImpl( String mode, + String invoker, @Nullable String includingPattern, @Nullable String excludesPattern, List includeTables, @@ -696,48 +765,70 @@ private void includingAndExcludingTablesImpl( } } - if (ThreadLocalRandom.current().nextBoolean()) { - List args = new ArrayList<>(); - args.add("compact_database"); - args.add("--warehouse"); - args.add(warehouse); - if (includingPattern != null) { - args.add("--including_tables"); - args.add(includingPattern); - } - if (excludesPattern != null) { - args.add("--excluding_tables"); - args.add(excludesPattern); - } - args.add("--mode"); - args.add(mode); - if (mode.equals("combined")) { - args.add("--table_conf"); - args.add(CoreOptions.CONTINUOUS_DISCOVERY_INTERVAL.key() + "=1s"); - } + switch (invoker) { + case "action": + List args = new ArrayList<>(); + args.add("compact_database"); + args.add("--warehouse"); + args.add(warehouse); + if (includingPattern != null) { + args.add("--including_tables"); + args.add(includingPattern); + } + if (excludesPattern != null) { + args.add("--excluding_tables"); + args.add(excludesPattern); + } + args.add("--mode"); + args.add(mode); + if (mode.equals("combined")) { + args.add("--table_conf"); + args.add(CoreOptions.CONTINUOUS_DISCOVERY_INTERVAL.key() + "=1s"); + } - StreamExecutionEnvironment env = - streamExecutionEnvironmentBuilder().batchMode().build(); - createAction(CompactDatabaseAction.class, args) - .withStreamExecutionEnvironment(env) - .build(); - env.execute(); - } else { - if (mode.equals("divided")) { - callProcedure( - String.format( - "CALL sys.compact_database('', 'divided', '%s', '%s')", - nonNull(includingPattern), nonNull(excludesPattern)), - false, - true); - } else { - callProcedure( - String.format( - "CALL sys.compact_database('', 'combined', '%s', '%s', 'continuous.discovery-interval=1s')", - nonNull(includingPattern), nonNull(excludesPattern)), - false, - true); - } + StreamExecutionEnvironment env = + streamExecutionEnvironmentBuilder().batchMode().build(); + createAction(CompactDatabaseAction.class, args) + .withStreamExecutionEnvironment(env) + .build(); + env.execute(); + break; + case "procedure_indexed": + if (mode.equals("divided")) { + callProcedure( + String.format( + "CALL sys.compact_database('', 'divided', '%s', '%s')", + nonNull(includingPattern), nonNull(excludesPattern)), + false, + true); + } else { + callProcedure( + String.format( + "CALL sys.compact_database('', 'combined', '%s', '%s', 'continuous.discovery-interval=1s')", + nonNull(includingPattern), nonNull(excludesPattern)), + false, + true); + } + break; + case "procedure_named": + if (mode.equals("divided")) { + callProcedure( + String.format( + "CALL sys.compact_database(mode => 'divided', including_tables => '%s', excluding_tables => '%s')", + nonNull(includingPattern), nonNull(excludesPattern)), + false, + true); + } else { + callProcedure( + String.format( + "CALL sys.compact_database(mode => 'combined', including_tables => '%s', excluding_tables => '%s', table_options => 'continuous.discovery-interval=1s')", + nonNull(includingPattern), nonNull(excludesPattern)), + false, + true); + } + break; + default: + throw new UnsupportedOperationException(invoker); } for (FileStoreTable table : compactionTables) { diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/ConsumerActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/ConsumerActionITCase.java index 9d507b52f0fa..c2cb190a80b6 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/ConsumerActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/ConsumerActionITCase.java @@ -27,13 +27,13 @@ import org.apache.paimon.types.DataTypes; import org.apache.paimon.types.RowType; -import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.util.Arrays; import java.util.Collections; import java.util.List; import java.util.Optional; -import java.util.concurrent.ThreadLocalRandom; import static org.apache.flink.table.planner.factories.TestValuesTableFactory.changelogRow; import static org.apache.paimon.flink.util.ReadWriteTableTestUtil.init; @@ -43,8 +43,9 @@ /** IT cases for consumer management actions. */ public class ConsumerActionITCase extends ActionITCaseBase { - @Test - public void testResetConsumer() throws Exception { + @ParameterizedTest + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testResetConsumer(String invoker) throws Exception { init(warehouse); RowType rowType = @@ -99,30 +100,55 @@ public void testResetConsumer() throws Exception { "--next_snapshot", "1"); // reset consumer - if (ThreadLocalRandom.current().nextBoolean()) { - createAction(ResetConsumerAction.class, args).run(); - } else { - callProcedure( - String.format( - "CALL sys.reset_consumer('%s.%s', 'myid', 1)", database, tableName)); + switch (invoker) { + case "action": + createAction(ResetConsumerAction.class, args).run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.reset_consumer('%s.%s', 'myid', 1)", + database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.reset_consumer(`table` => '%s.%s', consumer_id => 'myid', next_snapshot_id => cast(1 as bigint))", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } Optional consumer2 = consumerManager.consumer("myid"); assertThat(consumer2).isPresent(); assertThat(consumer2.get().nextSnapshot()).isEqualTo(1); // delete consumer - if (ThreadLocalRandom.current().nextBoolean()) { - createAction(ResetConsumerAction.class, args.subList(0, 9)).run(); - } else { - callProcedure( - String.format("CALL sys.reset_consumer('%s.%s', 'myid')", database, tableName)); + switch (invoker) { + case "action": + createAction(ResetConsumerAction.class, args.subList(0, 9)).run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.reset_consumer('%s.%s', 'myid')", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.reset_consumer(`table` => '%s.%s', consumer_id => 'myid')", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } Optional consumer3 = consumerManager.consumer("myid"); assertThat(consumer3).isNotPresent(); } - @Test - public void testResetBranchConsumer() throws Exception { + @ParameterizedTest + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testResetBranchConsumer(String invoker) throws Exception { init(warehouse); RowType rowType = @@ -182,25 +208,48 @@ public void testResetBranchConsumer() throws Exception { "--next_snapshot", "1"); // reset consumer - if (ThreadLocalRandom.current().nextBoolean()) { - createAction(ResetConsumerAction.class, args).run(); - } else { - callProcedure( - String.format( - "CALL sys.reset_consumer('%s.%s', 'myid', 1)", - database, branchTableName)); + switch (invoker) { + case "action": + createAction(ResetConsumerAction.class, args).run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.reset_consumer('%s.%s', 'myid', 1)", + database, branchTableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.reset_consumer(`table` => '%s.%s', consumer_id => 'myid', next_snapshot_id => cast(1 as bigint))", + database, branchTableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } Optional consumer2 = consumerManager.consumer("myid"); assertThat(consumer2).isPresent(); assertThat(consumer2.get().nextSnapshot()).isEqualTo(1); // delete consumer - if (ThreadLocalRandom.current().nextBoolean()) { - createAction(ResetConsumerAction.class, args.subList(0, 9)).run(); - } else { - callProcedure( - String.format( - "CALL sys.reset_consumer('%s.%s', 'myid')", database, branchTableName)); + switch (invoker) { + case "action": + createAction(ResetConsumerAction.class, args.subList(0, 9)).run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.reset_consumer('%s.%s', 'myid')", + database, branchTableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.reset_consumer(`table` => '%s.%s', consumer_id => 'myid')", + database, branchTableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } Optional consumer3 = consumerManager.consumer("myid"); assertThat(consumer3).isNotPresent(); diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MarkPartitionDoneActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MarkPartitionDoneActionITCase.java index e6c2c8678ea8..1c0ec8587816 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MarkPartitionDoneActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MarkPartitionDoneActionITCase.java @@ -30,12 +30,13 @@ import org.apache.paimon.utils.SnapshotManager; import org.junit.jupiter.params.ParameterizedTest; -import org.junit.jupiter.params.provider.ValueSource; +import org.junit.jupiter.params.provider.Arguments; +import org.junit.jupiter.params.provider.MethodSource; import java.util.Arrays; import java.util.Collections; import java.util.HashMap; -import java.util.concurrent.ThreadLocalRandom; +import java.util.stream.Stream; import static org.assertj.core.api.Assertions.assertThat; @@ -48,28 +49,51 @@ public class MarkPartitionDoneActionITCase extends ActionITCaseBase { private static final RowType ROW_TYPE = RowType.of(FIELD_TYPES, new String[] {"partKey0", "partKey1", "dt", "value"}); + private static Stream testArguments() { + return Stream.of( + Arguments.of(true, "action"), + Arguments.of(false, "action"), + Arguments.of(true, "procedure_indexed"), + Arguments.of(false, "procedure_indexed"), + Arguments.of(true, "procedure_named"), + Arguments.of(false, "procedure_named")); + } + @ParameterizedTest - @ValueSource(booleans = {true, false}) - public void testPartitionMarkDoneWithSinglePartitionKey(boolean hasPk) throws Exception { + @MethodSource("testArguments") + public void testPartitionMarkDoneWithSinglePartitionKey(boolean hasPk, String invoker) + throws Exception { FileStoreTable table = prepareTable(hasPk); - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - MarkPartitionDoneAction.class, - "mark_partition_done", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--partition", - "partKey0=0") - .run(); - } else { - callProcedure( - String.format( - "CALL sys.mark_partition_done('%s.%s', 'partKey0 = 0')", - database, tableName)); + + switch (invoker) { + case "action": + createAction( + MarkPartitionDoneAction.class, + "mark_partition_done", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--partition", + "partKey0=0") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.mark_partition_done('%s.%s', 'partKey0 = 0')", + database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.mark_partition_done(`table` => '%s.%s', partitions => 'partKey0 = 0')", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } Path successPath = new Path(table.location(), "partKey0=0/_SUCCESS"); @@ -78,30 +102,42 @@ public void testPartitionMarkDoneWithSinglePartitionKey(boolean hasPk) throws Ex } @ParameterizedTest - @ValueSource(booleans = {true, false}) - public void testDropPartitionWithMultiplePartitionKey(boolean hasPk) throws Exception { + @MethodSource("testArguments") + public void testDropPartitionWithMultiplePartitionKey(boolean hasPk, String invoker) + throws Exception { FileStoreTable table = prepareTable(hasPk); - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - MarkPartitionDoneAction.class, - "mark_partition_done", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--partition", - "partKey0=0,partKey1=1", - "--partition", - "partKey0=1,partKey1=0") - .run(); - } else { - callProcedure( - String.format( - "CALL sys.mark_partition_done('%s.%s', 'partKey0=0,partKey1=1', 'partKey0=1,partKey1=0')", - database, tableName)); + switch (invoker) { + case "action": + createAction( + MarkPartitionDoneAction.class, + "mark_partition_done", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--partition", + "partKey0=0,partKey1=1", + "--partition", + "partKey0=1,partKey1=0") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.mark_partition_done('%s.%s', 'partKey0=0,partKey1=1;partKey0=1,partKey1=0')", + database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.mark_partition_done(`table` => '%s.%s', partitions => 'partKey0=0,partKey1=1;partKey0=1,partKey1=0')", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } Path successPath1 = new Path(table.location(), "partKey0=0/partKey1=1/_SUCCESS"); diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MergeIntoActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MergeIntoActionITCase.java index 1da8030e1308..78f83fcfa3e0 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MergeIntoActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/MergeIntoActionITCase.java @@ -29,7 +29,6 @@ import org.junit.jupiter.params.ParameterizedTest; import org.junit.jupiter.params.provider.Arguments; import org.junit.jupiter.params.provider.MethodSource; -import org.junit.jupiter.params.provider.ValueSource; import javax.annotation.Nullable; @@ -38,7 +37,7 @@ import java.util.Collections; import java.util.HashMap; import java.util.List; -import java.util.concurrent.ThreadLocalRandom; +import java.util.stream.Stream; import static org.apache.flink.table.planner.factories.TestValuesTableFactory.changelogRow; import static org.apache.paimon.CoreOptions.CHANGELOG_PRODUCER; @@ -132,9 +131,19 @@ public void testVariousChangelogProducer( changelogRow("+I", 12, "v_12", "insert", "02-29"))); } - @ParameterizedTest(name = "in-default = {0}") - @ValueSource(booleans = {true, false}) - public void testTargetAlias(boolean inDefault) throws Exception { + private static Stream testArguments() { + return Stream.of( + Arguments.of(true, "action"), + Arguments.of(false, "action"), + Arguments.of(true, "procedure_indexed"), + Arguments.of(false, "procedure_indexed"), + Arguments.of(true, "procedure_named"), + Arguments.of(false, "procedure_named")); + } + + @ParameterizedTest + @MethodSource("testArguments") + public void testTargetAlias(boolean inDefault, String invoker) throws Exception { MergeIntoActionBuilder action; if (!inDefault) { @@ -155,10 +164,23 @@ public void testTargetAlias(boolean inDefault) throws Exception { .withMergeCondition("TT.k = S.k AND TT.dt = S.dt") .withMatchedDelete("S.v IS NULL"); - String procedureStatement = - String.format( - "CALL sys.merge_into('%s.T', 'TT', '', 'S', 'TT.k = S.k AND TT.dt = S.dt', 'S.v IS NULL')", - inDefault ? database : "test_db"); + String procedureStatement = ""; + if ("procedure_indexed".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into('%s.T', 'TT', '', 'S', 'TT.k = S.k AND TT.dt = S.dt', '', '', '', '', 'S.v IS NULL')", + inDefault ? database : "test_db"); + } else if ("procedure_named".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into(" + + "target_table => '%s.T', " + + "target_alias => 'TT', " + + "source_table => 'S', " + + "merge_condition => 'TT.k = S.k AND TT.dt = S.dt', " + + "matched_delete_condition => 'S.v IS NULL')", + inDefault ? database : "test_db"); + } List streamingExpected = Arrays.asList( @@ -176,16 +198,16 @@ public void testTargetAlias(boolean inDefault) throws Exception { changelogRow("+I", 9, "v_9", "creation", "02-28"), changelogRow("+I", 10, "v_10", "creation", "02-28")); - if (ThreadLocalRandom.current().nextBoolean()) { + if ("action".equals(invoker)) { validateActionRunResult(action.build(), streamingExpected, batchExpected); } else { validateProcedureResult(procedureStatement, streamingExpected, batchExpected); } } - @ParameterizedTest(name = "in-default = {0}") - @ValueSource(booleans = {true, false}) - public void testSourceName(boolean inDefault) throws Exception { + @ParameterizedTest + @MethodSource("testArguments") + public void testSourceName(boolean inDefault, String invoker) throws Exception { MergeIntoActionBuilder action = new MergeIntoActionBuilder(warehouse, "default", "T"); String sourceTableName = "S"; @@ -203,10 +225,22 @@ public void testSourceName(boolean inDefault) throws Exception { .withMergeCondition("T.k = S.k AND T.dt = S.dt") .withMatchedDelete("S.v IS NULL"); - String procedureStatement = - String.format( - "CALL sys.merge_into('default.T', '', '', '%s', 'T.k = S.k AND T.dt = S.dt', 'S.v IS NULL')", - sourceTableName); + String procedureStatement = ""; + if ("procedure_indexed".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into('default.T', '', '', '%s', 'T.k = S.k AND T.dt = S.dt', '', '', '', '', 'S.v IS NULL')", + sourceTableName); + } else if ("procedure_named".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into(" + + "target_table => 'default.T', " + + "source_table => '%s', " + + "merge_condition => 'T.k = S.k AND T.dt = S.dt', " + + "matched_delete_condition => 'S.v IS NULL')", + sourceTableName); + } if (!inDefault) { sEnv.executeSql("USE `default`"); @@ -229,16 +263,16 @@ public void testSourceName(boolean inDefault) throws Exception { changelogRow("+I", 9, "v_9", "creation", "02-28"), changelogRow("+I", 10, "v_10", "creation", "02-28")); - if (ThreadLocalRandom.current().nextBoolean()) { + if ("action".equals(invoker)) { validateActionRunResult(action.build(), streamingExpected, batchExpected); } else { validateProcedureResult(procedureStatement, streamingExpected, batchExpected); } } - @ParameterizedTest(name = "useCatalog = {0}") - @ValueSource(booleans = {true, false}) - public void testSqls(boolean useCatalog) throws Exception { + @ParameterizedTest + @MethodSource("testArguments") + public void testSqls(boolean useCatalog, String invoker) throws Exception { // drop table S sEnv.executeSql("DROP TABLE S"); @@ -273,16 +307,28 @@ public void testSqls(boolean useCatalog) throws Exception { action.withMergeCondition("T.k = S.k AND T.dt = S.dt").withMatchedDelete("S.v IS NULL"); - String procedureStatement = - String.format( - "CALL sys.merge_into('%s.T', '', '%s', '%s', 'T.k = S.k AND T.dt = S.dt', 'S.v IS NULL')", - database, - useCatalog - ? String.format( - "%s;%s;%s", - escapeCatalog, "USE CATALOG test_cat", escapeDdl) - : String.format("%s;%s", escapeCatalog, escapeDdl), - useCatalog ? "S" : "test_cat.default.S"); + String procedureStatement = ""; + String sourceSqls = + useCatalog + ? String.format( + "%s;%s;%s", escapeCatalog, "USE CATALOG test_cat", escapeDdl) + : String.format("%s;%s", escapeCatalog, escapeDdl); + if ("procedure_indexed".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into('%s.T', '', '%s', '%s', 'T.k = S.k AND T.dt = S.dt', '', '', '', '', 'S.v IS NULL')", + database, sourceSqls, useCatalog ? "S" : "test_cat.default.S"); + } else if ("procedure_named".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into(" + + "target_table => '%s.T', " + + "source_sqls => '%s', " + + "source_table => '%s', " + + "merge_condition => 'T.k = S.k AND T.dt = S.dt', " + + "matched_delete_condition => 'S.v IS NULL')", + database, sourceSqls, useCatalog ? "S" : "test_cat.default.S"); + } List streamingExpected = Arrays.asList( @@ -300,16 +346,16 @@ public void testSqls(boolean useCatalog) throws Exception { changelogRow("+I", 9, "v_9", "creation", "02-28"), changelogRow("+I", 10, "v_10", "creation", "02-28")); - if (ThreadLocalRandom.current().nextBoolean()) { + if ("action".equals(invoker)) { validateActionRunResult(action.build(), streamingExpected, batchExpected); } else { validateProcedureResult(procedureStatement, streamingExpected, batchExpected); } } - @ParameterizedTest(name = "source-qualified = {0}") - @ValueSource(booleans = {true, false}) - public void testMatchedUpsertSetAll(boolean qualified) throws Exception { + @ParameterizedTest + @MethodSource("testArguments") + public void testMatchedUpsertSetAll(boolean qualified, String invoker) throws Exception { // build MergeIntoAction MergeIntoActionBuilder action = new MergeIntoActionBuilder(warehouse, database, "T"); action.withSourceSqls("CREATE TEMPORARY VIEW SS AS SELECT k, v, 'unknown', dt FROM S") @@ -317,12 +363,27 @@ public void testMatchedUpsertSetAll(boolean qualified) throws Exception { .withMergeCondition("T.k = SS.k AND T.dt = SS.dt") .withMatchedUpsert(null, "*"); - String procedureStatement = - String.format( - "CALL sys.merge_into('%s.T', '', '%s', '%s', 'T.k = SS.k AND T.dt = SS.dt', '', '*')", - database, - "CREATE TEMPORARY VIEW SS AS SELECT k, v, ''unknown'', dt FROM S", - qualified ? "default.SS" : "SS"); + String procedureStatement = ""; + if ("procedure_indexed".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into('%s.T', '', '%s', '%s', 'T.k = SS.k AND T.dt = SS.dt', '', '*')", + database, + "CREATE TEMPORARY VIEW SS AS SELECT k, v, ''unknown'', dt FROM S", + qualified ? "default.SS" : "SS"); + } else if ("procedure_named".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into(" + + "target_table => '%s.T', " + + "source_sqls => '%s', " + + "source_table => '%s', " + + "merge_condition => 'T.k = SS.k AND T.dt = SS.dt', " + + "matched_upsert_setting => '*')", + database, + "CREATE TEMPORARY VIEW SS AS SELECT k, v, ''unknown'', dt FROM S", + qualified ? "default.SS" : "SS"); + } List streamingExpected = Arrays.asList( @@ -348,16 +409,16 @@ public void testMatchedUpsertSetAll(boolean qualified) throws Exception { changelogRow("+I", 9, "v_9", "creation", "02-28"), changelogRow("+I", 10, "v_10", "creation", "02-28")); - if (ThreadLocalRandom.current().nextBoolean()) { + if ("action".equals(invoker)) { validateActionRunResult(action.build(), streamingExpected, batchExpected); } else { validateProcedureResult(procedureStatement, streamingExpected, batchExpected); } } - @ParameterizedTest(name = "source-qualified = {0}") - @ValueSource(booleans = {true, false}) - public void testNotMatchedInsertAll(boolean qualified) throws Exception { + @ParameterizedTest + @MethodSource("testArguments") + public void testNotMatchedInsertAll(boolean qualified, String invoker) throws Exception { // build MergeIntoAction MergeIntoActionBuilder action = new MergeIntoActionBuilder(warehouse, database, "T"); action.withSourceSqls("CREATE TEMPORARY VIEW SS AS SELECT k, v, 'unknown', dt FROM S") @@ -365,12 +426,28 @@ public void testNotMatchedInsertAll(boolean qualified) throws Exception { .withMergeCondition("T.k = SS.k AND T.dt = SS.dt") .withNotMatchedInsert("SS.k < 12", "*"); - String procedureStatement = - String.format( - "CALL sys.merge_into('%s.T', '', '%s', '%s', 'T.k = SS.k AND T.dt = SS.dt', '', '', 'SS.k < 12', '*', '')", - database, - "CREATE TEMPORARY VIEW SS AS SELECT k, v, ''unknown'', dt FROM S", - qualified ? "default.SS" : "SS"); + String procedureStatement = ""; + if ("procedure_indexed".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into('%s.T', '', '%s', '%s', 'T.k = SS.k AND T.dt = SS.dt', '', '', 'SS.k < 12', '*', '')", + database, + "CREATE TEMPORARY VIEW SS AS SELECT k, v, ''unknown'', dt FROM S", + qualified ? "default.SS" : "SS"); + } else if ("procedure_named".equals(invoker)) { + procedureStatement = + String.format( + "CALL sys.merge_into(" + + "target_table => '%s.T', " + + "source_sqls => '%s', " + + "source_table => '%s', " + + "merge_condition => 'T.k = SS.k AND T.dt = SS.dt', " + + "not_matched_insert_condition => 'SS.k < 12'," + + "not_matched_insert_values => '*')", + database, + "CREATE TEMPORARY VIEW SS AS SELECT k, v, ''unknown'', dt FROM S", + qualified ? "default.SS" : "SS"); + } List streamingExpected = Arrays.asList( @@ -392,7 +469,7 @@ public void testNotMatchedInsertAll(boolean qualified) throws Exception { changelogRow("+I", 8, "v_8", "unknown", "02-29"), changelogRow("+I", 11, "v_11", "unknown", "02-29")); - if (ThreadLocalRandom.current().nextBoolean()) { + if ("action".equals(invoker)) { validateActionRunResult(action.build(), streamingExpected, batchExpected); } else { validateProcedureResult(procedureStatement, streamingExpected, batchExpected); @@ -403,7 +480,7 @@ public void testNotMatchedInsertAll(boolean qualified) throws Exception { public void testProcedureWithDeleteConditionTrue() throws Exception { String procedureStatement = String.format( - "CALL sys.merge_into('%s.T', '', '', 'S', 'T.k = S.k AND T.dt = S.dt', 'TRUE')", + "CALL sys.merge_into('%s.T', '', '', 'S', 'T.k = S.k AND T.dt = S.dt', '', '', '', '', 'TRUE')", database); validateProcedureResult( diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RemoveOrphanFilesActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RemoveOrphanFilesActionITCase.java index 653e97cc6f50..844ade6949d0 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RemoveOrphanFilesActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RemoveOrphanFilesActionITCase.java @@ -40,7 +40,8 @@ import org.apache.flink.types.Row; import org.apache.flink.util.CloseableIterator; -import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.util.ArrayList; import java.util.Arrays; @@ -93,8 +94,9 @@ private Path getOrphanFilePath(FileStoreTable table, String orphanFile) { return new Path(table.location(), orphanFile); } - @Test - public void testRunWithoutException() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testRunWithoutException(boolean isNamedArgument) throws Exception { FileStoreTable table = createTableAndWriteData(tableName); Path orphanFile1 = getOrphanFilePath(table, ORPHAN_FILE_1); Path orphanFile2 = getOrphanFilePath(table, ORPHAN_FILE_2); @@ -118,14 +120,22 @@ public void testRunWithoutException() throws Exception { assertThatCode(action2::run).doesNotThrowAnyException(); String withoutOlderThan = - String.format("CALL sys.remove_orphan_files('%s.%s')", database, tableName); + String.format( + isNamedArgument + ? "CALL sys.remove_orphan_files(`table` => '%s.%s')" + : "CALL sys.remove_orphan_files('%s.%s')", + database, + tableName); CloseableIterator withoutOlderThanCollect = callProcedure(withoutOlderThan); assertThat(ImmutableList.copyOf(withoutOlderThanCollect).size()).isEqualTo(0); String withDryRun = String.format( - "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59', true)", - database, tableName); + isNamedArgument + ? "CALL sys.remove_orphan_files(`table` => '%s.%s', older_than => '2999-12-31 23:59:59', dry_run => true)" + : "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59', true)", + database, + tableName); ImmutableList actualDryRunDeleteFile = ImmutableList.copyOf(callProcedure(withDryRun)); assertThat(actualDryRunDeleteFile) .containsExactlyInAnyOrder( @@ -134,8 +144,11 @@ public void testRunWithoutException() throws Exception { String withOlderThan = String.format( - "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59')", - database, tableName); + isNamedArgument + ? "CALL sys.remove_orphan_files(`table` => '%s.%s', older_than => '2999-12-31 23:59:59')" + : "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59')", + database, + tableName); ImmutableList actualDeleteFile = ImmutableList.copyOf(callProcedure(withOlderThan)); assertThat(actualDeleteFile) @@ -144,8 +157,9 @@ public void testRunWithoutException() throws Exception { Row.of(orphanFile2.toUri().getPath())); } - @Test - public void testRemoveDatabaseOrphanFilesITCase() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testRemoveDatabaseOrphanFilesITCase(boolean isNamedArgument) throws Exception { FileStoreTable table1 = createTableAndWriteData("tableName1"); Path orphanFile11 = getOrphanFilePath(table1, ORPHAN_FILE_1); Path orphanFile12 = getOrphanFilePath(table1, ORPHAN_FILE_2); @@ -172,14 +186,22 @@ public void testRemoveDatabaseOrphanFilesITCase() throws Exception { assertThatCode(action2::run).doesNotThrowAnyException(); String withoutOlderThan = - String.format("CALL sys.remove_orphan_files('%s.%s')", database, "*"); + String.format( + isNamedArgument + ? "CALL sys.remove_orphan_files(`table` => '%s.%s')" + : "CALL sys.remove_orphan_files('%s.%s')", + database, + "*"); CloseableIterator withoutOlderThanCollect = callProcedure(withoutOlderThan); assertThat(ImmutableList.copyOf(withoutOlderThanCollect).size()).isEqualTo(0); String withDryRun = String.format( - "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59', true)", - database, "*"); + isNamedArgument + ? "CALL sys.remove_orphan_files(`table` => '%s.%s', older_than => '2999-12-31 23:59:59', dry_run => true)" + : "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59', true)", + database, + "*"); ImmutableList actualDryRunDeleteFile = ImmutableList.copyOf(callProcedure(withDryRun)); assertThat(actualDryRunDeleteFile) .containsExactlyInAnyOrder( @@ -190,8 +212,11 @@ public void testRemoveDatabaseOrphanFilesITCase() throws Exception { String withOlderThan = String.format( - "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59')", - database, "*"); + isNamedArgument + ? "CALL sys.remove_orphan_files(`table` => '%s.%s', older_than => '2999-12-31 23:59:59')" + : "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59')", + database, + "*"); ImmutableList actualDeleteFile = ImmutableList.copyOf(callProcedure(withOlderThan)); assertThat(actualDeleteFile) @@ -202,8 +227,9 @@ public void testRemoveDatabaseOrphanFilesITCase() throws Exception { Row.of(orphanFile22.toUri().getPath())); } - @Test - public void testCleanWithBranch() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testCleanWithBranch(boolean isNamedArgument) throws Exception { // create main branch FileStoreTable table = createTableAndWriteData(tableName); Path orphanFile1 = getOrphanFilePath(table, ORPHAN_FILE_1); @@ -241,8 +267,11 @@ public void testCleanWithBranch() throws Exception { String procedure = String.format( - "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59')", - database, "*"); + isNamedArgument + ? "CALL sys.remove_orphan_files(`table` => '%s.%s', older_than => '2999-12-31 23:59:59')" + : "CALL sys.remove_orphan_files('%s.%s', '2999-12-31 23:59:59')", + database, + "*"); ImmutableList actualDeleteFile = ImmutableList.copyOf(callProcedure(procedure)); assertThat(actualDeleteFile) .containsExactlyInAnyOrder( diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RollbackToActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RollbackToActionITCase.java index ceab42480ed6..a9e0726e7d15 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RollbackToActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/RollbackToActionITCase.java @@ -27,11 +27,11 @@ import org.apache.flink.types.Row; import org.junit.jupiter.api.BeforeEach; -import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.util.Arrays; import java.util.Collections; -import java.util.concurrent.ThreadLocalRandom; import static org.apache.paimon.flink.util.ReadWriteTableTestUtil.init; import static org.apache.paimon.flink.util.ReadWriteTableTestUtil.testBatchRead; @@ -49,8 +49,9 @@ public void setUp() { init(warehouse); } - @Test - public void rollbackToSnapshotTest() throws Exception { + @ParameterizedTest + @ValueSource(strings = {"action", "procedure_named", "procedure_indexed"}) + public void rollbackToSnapshotTest(String invoker) throws Exception { FileStoreTable table = createFileStoreTable( ROW_TYPE, @@ -67,21 +68,35 @@ public void rollbackToSnapshotTest() throws Exception { writeData(rowData(2L, BinaryString.fromString("World"))); writeData(rowData(2L, BinaryString.fromString("Flink"))); - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - RollbackToAction.class, - "rollback_to", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--version", - "2") - .run(); - } else { - callProcedure(String.format("CALL sys.rollback_to('%s.%s', 2)", database, tableName)); + switch (invoker) { + case "action": + createAction( + RollbackToAction.class, + "rollback_to", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--version", + "2") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.rollback_to('%s.%s', '', cast(2 as bigint))", + database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.rollback_to(`table` => '%s.%s', snapshot_id => cast(2 as bigint))", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } testBatchRead( @@ -89,8 +104,9 @@ public void rollbackToSnapshotTest() throws Exception { Arrays.asList(Row.of(1L, "Hi"), Row.of(2L, "Hello"))); } - @Test - public void rollbackToTagTest() throws Exception { + @ParameterizedTest + @ValueSource(strings = {"action", "procedure_named", "procedure_indexed"}) + public void rollbackToTagTest(String invoker) throws Exception { FileStoreTable table = createFileStoreTable( ROW_TYPE, @@ -110,22 +126,34 @@ public void rollbackToTagTest() throws Exception { table.createTag("tag2", 2); table.createTag("tag3", 3); - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - RollbackToAction.class, - "rollback_to", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--version", - "tag2") - .run(); - } else { - callProcedure( - String.format("CALL sys.rollback_to('%s.%s', 'tag2')", database, tableName)); + switch (invoker) { + case "action": + createAction( + RollbackToAction.class, + "rollback_to", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--version", + "tag2") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.rollback_to('%s.%s', 'tag2')", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.rollback_to(`table` => '%s.%s', tag => 'tag2')", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } testBatchRead( diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/TagActionITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/TagActionITCase.java index b873a1bb68a0..9603d9085dfb 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/TagActionITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/action/TagActionITCase.java @@ -27,11 +27,12 @@ import org.apache.paimon.utils.TagManager; import org.apache.flink.types.Row; -import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.util.Arrays; import java.util.Collections; -import java.util.concurrent.ThreadLocalRandom; +import java.util.stream.Stream; import static org.apache.paimon.flink.util.ReadWriteTableTestUtil.init; import static org.apache.paimon.flink.util.ReadWriteTableTestUtil.testBatchRead; @@ -40,8 +41,13 @@ /** IT cases for tag management actions. */ public class TagActionITCase extends ActionITCaseBase { - @Test - public void testCreateAndDeleteTag() throws Exception { + private static Stream testData() { + return Stream.of("action", "procedure_indexed", "procedure_named"); + } + + @ParameterizedTest(name = "{0}") + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testCreateAndDeleteTag(String invoker) throws Exception { init(warehouse); RowType rowType = @@ -67,24 +73,36 @@ public void testCreateAndDeleteTag() throws Exception { TagManager tagManager = new TagManager(table.fileIO(), table.location()); - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - CreateTagAction.class, - "create_tag", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--tag_name", - "tag2", - "--snapshot", - "2") - .run(); - } else { - callProcedure( - String.format("CALL sys.create_tag('%s.%s', 'tag2', 2)", database, tableName)); + switch (invoker) { + case "action": + createAction( + CreateTagAction.class, + "create_tag", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--tag_name", + "tag2", + "--snapshot", + "2") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.create_tag('%s.%s', 'tag2', 2)", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.create_tag(`table` => '%s.%s', tag => 'tag2', snapshot_id => cast(2 as bigint))", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } assertThat(tagManager.tagExists("tag2")).isTrue(); @@ -93,91 +111,138 @@ public void testCreateAndDeleteTag() throws Exception { "SELECT * FROM `" + tableName + "` /*+ OPTIONS('scan.tag-name'='tag2') */", Arrays.asList(Row.of(1L, "Hi"), Row.of(2L, "Hello"))); - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - DeleteTagAction.class, - "delete_tag", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--tag_name", - "tag2") - .run(); - } else { - callProcedure( - String.format("CALL sys.delete_tag('%s.%s', 'tag2')", database, tableName)); + switch (invoker) { + case "action": + createAction( + DeleteTagAction.class, + "delete_tag", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--tag_name", + "tag2") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format("CALL sys.delete_tag('%s.%s', 'tag2')", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.delete_tag(`table` => '%s.%s', tag => 'tag2')", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } assertThat(tagManager.tagExists("tag2")).isFalse(); // create tag1 - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - CreateTagAction.class, - "create_tag", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--tag_name", - "tag1", - "--snapshot", - "1") - .run(); - } else { - callProcedure( - String.format("CALL sys.create_tag('%s.%s', 'tag1', 1)", database, tableName)); + switch (invoker) { + case "action": + createAction( + CreateTagAction.class, + "create_tag", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--tag_name", + "tag1", + "--snapshot", + "1") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.create_tag('%s.%s', 'tag1', 1)", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.create_tag(`table` => '%s.%s', tag => 'tag1', snapshot_id => cast(1 as bigint))", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } // create tag3 - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - CreateTagAction.class, - "create_tag", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--tag_name", - "tag3", - "--snapshot", - "3") - .run(); - } else { - callProcedure( - String.format("CALL sys.create_tag('%s.%s', 'tag3', 3)", database, tableName)); + switch (invoker) { + case "action": + createAction( + CreateTagAction.class, + "create_tag", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--tag_name", + "tag3", + "--snapshot", + "3") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.create_tag('%s.%s', 'tag3', 3)", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.create_tag(`table` => '%s.%s', tag => 'tag3', snapshot_id => cast(3 as bigint))", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - DeleteTagAction.class, - "delete_tag", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--tag_name", - "tag1,tag3") - .run(); - } else { - callProcedure( - String.format( - "CALL sys.delete_tag('%s.%s', 'tag1,tag3')", database, tableName)); + switch (invoker) { + case "action": + createAction( + DeleteTagAction.class, + "delete_tag", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--tag_name", + "tag1,tag3") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.delete_tag('%s.%s', 'tag1,tag3')", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.delete_tag(`table` => '%s.%s', tag => 'tag1,tag3')", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } assertThat(tagManager.tagExists("tag1")).isFalse(); assertThat(tagManager.tagExists("tag3")).isFalse(); } - @Test - public void testCreateLatestTag() throws Exception { + @ParameterizedTest(name = "{0}") + @ValueSource(strings = {"action", "procedure_indexed", "procedure_named"}) + public void testCreateLatestTag(String invoker) throws Exception { init(warehouse); RowType rowType = @@ -203,22 +268,34 @@ public void testCreateLatestTag() throws Exception { TagManager tagManager = new TagManager(table.fileIO(), table.location()); - if (ThreadLocalRandom.current().nextBoolean()) { - createAction( - CreateTagAction.class, - "create_tag", - "--warehouse", - warehouse, - "--database", - database, - "--table", - tableName, - "--tag_name", - "tag2") - .run(); - } else { - callProcedure( - String.format("CALL sys.create_tag('%s.%s', 'tag2', 2)", database, tableName)); + switch (invoker) { + case "action": + createAction( + CreateTagAction.class, + "create_tag", + "--warehouse", + warehouse, + "--database", + database, + "--table", + tableName, + "--tag_name", + "tag2") + .run(); + break; + case "procedure_indexed": + callProcedure( + String.format( + "CALL sys.create_tag('%s.%s', 'tag2', 2)", database, tableName)); + break; + case "procedure_named": + callProcedure( + String.format( + "CALL sys.create_tag(`table` => '%s.%s', tag => 'tag2', snapshot_id => cast(2 as bigint))", + database, tableName)); + break; + default: + throw new UnsupportedOperationException(invoker); } assertThat(tagManager.tagExists("tag2")).isTrue(); } diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/ProcedureTest.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/ProcedureTest.java new file mode 100644 index 000000000000..c24d4105a557 --- /dev/null +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/ProcedureTest.java @@ -0,0 +1,113 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.paimon.flink.procedure; + +import org.apache.paimon.factories.FactoryException; +import org.apache.paimon.factories.FactoryUtil; +import org.apache.paimon.flink.action.ActionFactory; + +import org.apache.flink.table.annotation.ProcedureHint; +import org.junit.jupiter.api.Test; + +import java.lang.reflect.Method; +import java.util.HashSet; +import java.util.List; +import java.util.Set; + +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; + +/** Unit tests for procedures. */ +public class ProcedureTest { + @Test + public void testProcedureCoverAllActions() { + Set expectedExclusions = new HashSet<>(); + // Can be covered by `DELETE FROM` syntax. No procedure needed. + expectedExclusions.add("delete"); + List actionIdentifiers = + FactoryUtil.discoverIdentifiers( + ActionFactory.class.getClassLoader(), ActionFactory.class); + assertThat(actionIdentifiers.size()).isNotZero(); + for (String identifier : actionIdentifiers) { + if (expectedExclusions.contains(identifier)) { + assertThatThrownBy( + () -> + FactoryUtil.discoverFactory( + ProcedureBase.class.getClassLoader(), + ProcedureBase.class, + identifier)) + .isInstanceOf(FactoryException.class) + .hasMessageContaining("Could not find any factory for identifier"); + } else { + ProcedureBase procedure = + FactoryUtil.discoverFactory( + ProcedureBase.class.getClassLoader(), + ProcedureBase.class, + identifier); + assertThat(procedure).isNotNull(); + } + } + } + + @Test + public void testProcedureHasNamedArgument() { + Set expectedExclusions = new HashSet<>(); + expectedExclusions.add("drop_partition"); // Has been deprecated. + List identifiers = + FactoryUtil.discoverIdentifiers( + ProcedureBase.class.getClassLoader(), ProcedureBase.class); + assertThat(identifiers.size()).isNotZero(); + for (String identifier : identifiers) { + ProcedureBase procedure = + FactoryUtil.discoverFactory( + ProcedureBase.class.getClassLoader(), ProcedureBase.class, identifier); + Method method = getMethodFromName(procedure.getClass(), "call"); + if (expectedExclusions.contains(identifier)) { + assertThat(method) + .matches( + x -> !x.isAnnotationPresent(ProcedureHint.class), + String.format( + "Procedure %s has supported named argument. Should be removed from exclusion list.", + procedure.identifier())); + } else { + assertThat(method) + .matches( + x -> x.isAnnotationPresent(ProcedureHint.class), + String.format( + "Procedure %s should have its call method decorated by %s.", + procedure.identifier(), + ProcedureHint.class.getSimpleName())); + } + } + } + + private Method getMethodFromName(Class clazz, String methodName) { + Method[] methods = clazz.getDeclaredMethods(); + + for (Method method : methods) { + if (method.getName().equals(methodName)) { + return method; + } + } + throw new IllegalStateException( + String.format( + "Procedure class %s does not have a method named %s.", + clazz.getSimpleName(), methodName)); + } +} diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedureITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedureITCase.java index 18b623cc8297..1abfe355a566 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedureITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/RewriteFileIndexProcedureITCase.java @@ -33,7 +33,8 @@ import org.apache.flink.table.api.config.TableConfigOptions; import org.assertj.core.api.Assertions; -import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.util.List; import java.util.Set; @@ -43,8 +44,9 @@ /** IT Case for {@link RewriteFileIndexProcedure}. */ public class RewriteFileIndexProcedureITCase extends CatalogITCaseBase { - @Test - public void testFileIndexProcedureSchemaEvolution() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testFileIndexProcedureSchemaEvolution(boolean isNamedArgument) throws Exception { sql( "CREATE TABLE T (" + " k INT," @@ -82,7 +84,11 @@ public void testFileIndexProcedureSchemaEvolution() throws Exception { tEnv.getConfig().set(TableConfigOptions.TABLE_DML_SYNC, true); sql("ALTER TABLE T SET ('file-index.bloom-filter.columns'='order_id,v')"); - sql("CALL sys.rewrite_file_index('default.T')"); + if (isNamedArgument) { + sql("CALL sys.rewrite_file_index(`table` => 'default.T')"); + } else { + sql("CALL sys.rewrite_file_index('default.T')"); + } reader = table.newRead() @@ -95,8 +101,9 @@ public void testFileIndexProcedureSchemaEvolution() throws Exception { Assertions.assertThat(count.get()).isEqualTo(0); } - @Test - public void testPartitionFilter() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testPartitionFilter(boolean isNamedArgument) throws Exception { sql( "CREATE TABLE T (" + " k INT," @@ -132,7 +139,11 @@ public void testPartitionFilter() throws Exception { tEnv.getConfig().set(TableConfigOptions.TABLE_DML_SYNC, true); sql("ALTER TABLE T SET ('file-index.bloom-filter.columns'='order_id,v')"); - sql("CALL sys.rewrite_file_index('default.T', 'dt=20221208')"); + if (isNamedArgument) { + sql("CALL sys.rewrite_file_index(`table` => 'default.T', partitions => 'dt=20221208')"); + } else { + sql("CALL sys.rewrite_file_index('default.T', 'dt=20221208')"); + } reader = table.newRead() @@ -145,8 +156,9 @@ public void testPartitionFilter() throws Exception { Assertions.assertThat(count.get()).isEqualTo(2); } - @Test - public void testFileIndexProcedureDropIndex() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testFileIndexProcedureDropIndex(boolean isNamedArgument) throws Exception { sql( "CREATE TABLE T (" + " k INT," @@ -164,7 +176,11 @@ public void testFileIndexProcedureDropIndex() throws Exception { tEnv.getConfig().set(TableConfigOptions.TABLE_DML_SYNC, true); sql("ALTER TABLE T SET ('file-index.bloom-filter.columns'='k')"); - sql("CALL sys.rewrite_file_index('default.T')"); + if (isNamedArgument) { + sql("CALL sys.rewrite_file_index(`table` => 'default.T')"); + } else { + sql("CALL sys.rewrite_file_index('default.T')"); + } FileStoreTable table = paimonTable("T"); List list = table.store().newScan().plan().files(); @@ -195,7 +211,11 @@ public void testFileIndexProcedureDropIndex() throws Exception { sql("ALTER TABLE T RESET ('file-index.bloom-filter.columns')"); - sql("CALL sys.rewrite_file_index('default.T')"); + if (isNamedArgument) { + sql("CALL sys.rewrite_file_index('default.T')"); + } else { + sql("CALL sys.rewrite_file_index(`table` => 'default.T')"); + } table = paimonTable("T"); list = table.store().newScan().plan().files(); diff --git a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/privilege/PrivilegeProcedureITCase.java b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/privilege/PrivilegeProcedureITCase.java index db973b39c8ca..6350ccc3d38d 100644 --- a/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/privilege/PrivilegeProcedureITCase.java +++ b/paimon-flink/paimon-flink-common/src/test/java/org/apache/paimon/flink/procedure/privilege/PrivilegeProcedureITCase.java @@ -29,8 +29,9 @@ import org.apache.flink.types.Row; import org.apache.flink.util.CloseableIterator; import org.junit.jupiter.api.BeforeEach; -import org.junit.jupiter.api.Test; import org.junit.jupiter.api.function.Executable; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; import java.util.ArrayList; import java.util.Arrays; @@ -49,8 +50,9 @@ public void beforeEach() { path = getTempDirPath(); } - @Test - public void testUserPrivileges() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testUserPrivileges(boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql( String.format( @@ -69,7 +71,11 @@ public void testUserPrivileges() throws Exception { + " PRIMARY KEY (k) NOT ENFORCED\n" + ")"); tEnv.executeSql("INSERT INTO mydb.T1 VALUES (1, 10), (2, 20), (3, 30)").await(); - tEnv.executeSql("CALL sys.init_file_based_privilege('root-passwd')"); + if (isNamedArgument) { + tEnv.executeSql("CALL sys.init_file_based_privilege(root_password => 'root-passwd')"); + } else { + tEnv.executeSql("CALL sys.init_file_based_privilege('root-passwd')"); + } tEnv.executeSql( String.format( @@ -97,8 +103,17 @@ public void testUserPrivileges() throws Exception { assertNoPrivilege(() -> tEnv.executeSql("ALTER TABLE mydb.T1 RENAME TO mydb.T2")); assertNoPrivilege(() -> tEnv.executeSql("CREATE DATABASE anotherdb")); assertNoPrivilege(() -> tEnv.executeSql("DROP DATABASE mydb CASCADE")); - assertNoPrivilege( - () -> tEnv.executeSql("CALL sys.create_privileged_user('test2', 'test2-passwd')")); + if (isNamedArgument) { + assertNoPrivilege( + () -> + tEnv.executeSql( + "CALL sys.create_privileged_user(username => 'test2', password => 'test2-passwd')")); + } else { + assertNoPrivilege( + () -> + tEnv.executeSql( + "CALL sys.create_privileged_user('test2', 'test2-passwd')")); + } tEnv.executeSql( String.format( @@ -118,10 +133,21 @@ public void testUserPrivileges() throws Exception { + ")"); tEnv.executeSql("INSERT INTO mydb2.T2 VALUES (100, 1000), (200, 2000), (300, 3000)") .await(); - tEnv.executeSql("CALL sys.create_privileged_user('test', 'test-passwd')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'CREATE_TABLE', 'mydb')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'SELECT', 'mydb')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'INSERT', 'mydb')"); + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.create_privileged_user(username => 'test', password => 'test-passwd')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test', privilege => 'CREATE_TABLE', database => 'mydb')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test', privilege => 'SELECT', database => 'mydb')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test', privilege => 'INSERT', database => 'mydb')"); + } else { + tEnv.executeSql("CALL sys.create_privileged_user('test', 'test-passwd')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'CREATE_TABLE', 'mydb')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'SELECT', 'mydb')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'INSERT', 'mydb')"); + } tEnv.executeSql( String.format( @@ -146,15 +172,36 @@ public void testUserPrivileges() throws Exception { assertNoPrivilege(() -> tEnv.executeSql("ALTER TABLE mydb.S1 RENAME TO mydb.S2")); assertNoPrivilege(() -> tEnv.executeSql("CREATE DATABASE anotherdb")); assertNoPrivilege(() -> tEnv.executeSql("DROP DATABASE mydb CASCADE")); - assertNoPrivilege( - () -> tEnv.executeSql("CALL sys.create_privileged_user('test2', 'test2-passwd')")); - - tEnv.executeSql("USE CATALOG rootcat"); - tEnv.executeSql("CALL sys.create_privileged_user('test2', 'test2-passwd')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'SELECT', 'mydb2')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'INSERT', 'mydb', 'T1')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'SELECT', 'mydb', 'S1')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'CREATE_DATABASE')"); + if (isNamedArgument) { + assertNoPrivilege( + () -> + tEnv.executeSql( + "CALL sys.create_privileged_user(username => 'test2', password => 'test2-passwd')")); + + tEnv.executeSql("USE CATALOG rootcat"); + tEnv.executeSql( + "CALL sys.create_privileged_user(username => 'test2', password => 'test2-passwd')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test2', privilege => 'SELECT', database => 'mydb2')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test2', privilege => 'INSERT', database => 'mydb', `table` => 'T1')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test2', privilege => 'SELECT', database => 'mydb', `table` => 'S1')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test2', privilege => 'CREATE_DATABASE')"); + } else { + assertNoPrivilege( + () -> + tEnv.executeSql( + "CALL sys.create_privileged_user('test2', 'test2-passwd')")); + + tEnv.executeSql("USE CATALOG rootcat"); + tEnv.executeSql("CALL sys.create_privileged_user('test2', 'test2-passwd')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'SELECT', 'mydb2')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'INSERT', 'mydb', 'T1')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'SELECT', 'mydb', 'S1')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test2', 'CREATE_DATABASE')"); + } tEnv.executeSql( String.format( @@ -188,14 +235,30 @@ public void testUserPrivileges() throws Exception { assertNoPrivilege(() -> tEnv.executeSql("DROP TABLE mydb.S1")); assertNoPrivilege(() -> tEnv.executeSql("ALTER TABLE mydb.S1 RENAME TO mydb.S2")); assertNoPrivilege(() -> tEnv.executeSql("DROP DATABASE mydb CASCADE")); - assertNoPrivilege( - () -> tEnv.executeSql("CALL sys.create_privileged_user('test3', 'test3-passwd')")); - - tEnv.executeSql("USE CATALOG rootcat"); - assertThat(collect(tEnv, "SELECT * FROM mydb.T1 ORDER BY k")) - .isEqualTo(Arrays.asList(Row.of(1, 13), Row.of(2, 23), Row.of(3, 30))); - tEnv.executeSql("CALL sys.revoke_privilege_from_user('test2', 'SELECT')"); - tEnv.executeSql("CALL sys.drop_privileged_user('test')"); + if (isNamedArgument) { + assertNoPrivilege( + () -> + tEnv.executeSql( + "CALL sys.create_privileged_user(username => 'test3', password => 'test3-passwd')")); + + tEnv.executeSql("USE CATALOG rootcat"); + assertThat(collect(tEnv, "SELECT * FROM mydb.T1 ORDER BY k")) + .isEqualTo(Arrays.asList(Row.of(1, 13), Row.of(2, 23), Row.of(3, 30))); + tEnv.executeSql( + "CALL sys.revoke_privilege_from_user(username => 'test2', privilege => 'SELECT')"); + tEnv.executeSql("CALL sys.drop_privileged_user(username => 'test')"); + } else { + assertNoPrivilege( + () -> + tEnv.executeSql( + "CALL sys.create_privileged_user('test3', 'test3-passwd')")); + + tEnv.executeSql("USE CATALOG rootcat"); + assertThat(collect(tEnv, "SELECT * FROM mydb.T1 ORDER BY k")) + .isEqualTo(Arrays.asList(Row.of(1, 13), Row.of(2, 23), Row.of(3, 30))); + tEnv.executeSql("CALL sys.revoke_privilege_from_user('test2', 'SELECT')"); + tEnv.executeSql("CALL sys.drop_privileged_user('test')"); + } tEnv.executeSql("USE CATALOG testcat"); Exception e = @@ -215,14 +278,21 @@ public void testUserPrivileges() throws Exception { tEnv.executeSql("DROP DATABASE mydb2 CASCADE"); } - @Test - public void testDropUser() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testDropUser(boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); - initializeSingleUserTest(tEnv); + initializeSingleUserTest(tEnv, isNamedArgument); tEnv.executeSql("USE CATALOG rootcat"); - tEnv.executeSql("CALL sys.drop_privileged_user('test')"); - tEnv.executeSql("CALL sys.create_privileged_user('test', 'test-passwd')"); + if (isNamedArgument) { + tEnv.executeSql("CALL sys.drop_privileged_user(username => 'test')"); + tEnv.executeSql( + "CALL sys.create_privileged_user(username => 'test', password => 'test-passwd')"); + } else { + tEnv.executeSql("CALL sys.drop_privileged_user('test')"); + tEnv.executeSql("CALL sys.create_privileged_user('test', 'test-passwd')"); + } tEnv.executeSql("USE CATALOG testcat"); assertNoPrivilege(() -> collect(tEnv, "SELECT * FROM mydb.T1 ORDER BY k")); @@ -230,10 +300,11 @@ public void testDropUser() throws Exception { () -> tEnv.executeSql("INSERT INTO mydb.T1 VALUES (1, 12), (2, 22)").await()); } - @Test - public void testDropObject() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testDropObject(boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); - initializeSingleUserTest(tEnv); + initializeSingleUserTest(tEnv, isNamedArgument); tEnv.executeSql("USE CATALOG rootcat"); tEnv.executeSql("DROP TABLE mydb.T1"); @@ -250,10 +321,11 @@ public void testDropObject() throws Exception { () -> tEnv.executeSql("INSERT INTO mydb.T1 VALUES (1, 12), (2, 22)").await()); } - @Test - public void testRenameObject() throws Exception { + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testRenameObject(boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); - initializeSingleUserTest(tEnv); + initializeSingleUserTest(tEnv, isNamedArgument); tEnv.executeSql("USE CATALOG rootcat"); tEnv.executeSql("ALTER TABLE mydb.T1 RENAME TO mydb.T2"); @@ -266,7 +338,8 @@ public void testRenameObject() throws Exception { .isEqualTo(Arrays.asList(Row.of(1, 12), Row.of(2, 22), Row.of(3, 30))); } - private void initializeSingleUserTest(TableEnvironment tEnv) throws Exception { + private void initializeSingleUserTest(TableEnvironment tEnv, boolean isNamedArgument) + throws Exception { tEnv.executeSql( String.format( "CREATE CATALOG mycat WITH (\n" @@ -283,7 +356,11 @@ private void initializeSingleUserTest(TableEnvironment tEnv) throws Exception { + " PRIMARY KEY (k) NOT ENFORCED\n" + ")"); tEnv.executeSql("INSERT INTO mydb.T1 VALUES (1, 10), (2, 20), (3, 30)").await(); - tEnv.executeSql("CALL sys.init_file_based_privilege('root-passwd')"); + if (isNamedArgument) { + tEnv.executeSql("CALL sys.init_file_based_privilege(root_password => 'root-passwd')"); + } else { + tEnv.executeSql("CALL sys.init_file_based_privilege('root-passwd')"); + } tEnv.executeSql( String.format( @@ -295,9 +372,18 @@ private void initializeSingleUserTest(TableEnvironment tEnv) throws Exception { + ")", path)); tEnv.executeSql("USE CATALOG rootcat"); - tEnv.executeSql("CALL sys.create_privileged_user('test', 'test-passwd')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'SELECT', 'mydb', 'T1')"); - tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'INSERT', 'mydb', 'T1')"); + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.create_privileged_user(username => 'test', password => 'test-passwd')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test', privilege => 'SELECT', database => 'mydb', `table` => 'T1')"); + tEnv.executeSql( + "CALL sys.grant_privilege_to_user(username => 'test', privilege => 'INSERT', database => 'mydb', `table` => 'T1')"); + } else { + tEnv.executeSql("CALL sys.create_privileged_user('test', 'test-passwd')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'SELECT', 'mydb', 'T1')"); + tEnv.executeSql("CALL sys.grant_privilege_to_user('test', 'INSERT', 'mydb', 'T1')"); + } tEnv.executeSql( String.format( diff --git a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/HiveCatalogITCaseBase.java b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/HiveCatalogITCaseBase.java index c103564db95f..aa3062ec7db3 100644 --- a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/HiveCatalogITCaseBase.java +++ b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/HiveCatalogITCaseBase.java @@ -1623,14 +1623,75 @@ public void testRepairDatabasesOrTables() throws Exception { .containsExactlyInAnyOrder("dt=2020-01-02/hh=09", "dt=2020-01-03/hh=10"); } + @Test + public void testRepairDatabasesOrTablesWithNamedArgument() throws Exception { + TableEnvironment fileCatalog = useFileCatalog("test_db"); + TableEnvironment fileCatalog01 = useFileCatalog("test_db_02"); + // Database test_db exists in hive metastore + hiveShell.execute("use test_db"); + tEnv.executeSql("USE test_db").await(); + // When the Hive table does not exist, specify the paimon table to create hive table in hive + // metastore. + tEnv.executeSql("CALL sys.repair(`table` => 'test_db.t_repair_hive,test_db_02')"); + + assertThat(hiveShell.executeQuery("SHOW PARTITIONS test_db.t_repair_hive")) + .containsExactlyInAnyOrder("dt=2020-01-02/hh=09"); + + alterTableInFileSystem(fileCatalog); + // When the Hive table exists, specify the paimon table to update hive table in hive + // metastore. + tEnv.executeSql("CALL sys.repair(`table` => 'test_db.t_repair_hive')"); + + assertThat( + hiveShell + .executeQuery("DESC FORMATTED test_db.t_repair_hive") + .contains("item_id\tbigint\titem id")) + .isTrue(); + assertThat(hiveShell.executeQuery("SHOW PARTITIONS test_db.t_repair_hive")) + .containsExactlyInAnyOrder("dt=2020-01-02/hh=09", "dt=2020-01-03/hh=10"); + + // Database test_db_02 exists in hive metastore + hiveShell.execute("use test_db_02"); + tEnv.executeSql("USE test_db_02").await(); + assertThat(hiveShell.executeQuery("SHOW PARTITIONS test_db_02.t_repair_hive")) + .containsExactlyInAnyOrder("dt=2020-01-02/hh=09"); + + alterTableInFileSystem(fileCatalog01); + + // When the Hive table exists, specify the paimon table to update hive table in hive + // metastore. + tEnv.executeSql("CALL sys.repair(`table` => 'test_db_02.t_repair_hive')"); + assertThat( + hiveShell + .executeQuery("DESC FORMATTED test_db_02.t_repair_hive") + .contains("item_id\tbigint\titem id")) + .isTrue(); + assertThat(hiveShell.executeQuery("SHOW PARTITIONS test_db_02.t_repair_hive")) + .containsExactlyInAnyOrder("dt=2020-01-02/hh=09", "dt=2020-01-03/hh=10"); + hiveShell.execute("DROP TABLE test_db.t_repair_hive"); + hiveShell.execute("DROP TABLE test_db_02.t_repair_hive"); + } + @Test public void testRepairTable() throws Exception { + testRepairTable(false); + } + + @Test + public void testRepairTableWithNamedArgument() throws Exception { + testRepairTable(true); + } + + private void testRepairTable(boolean isNamedArgument) throws Exception { TableEnvironment fileCatalog = useFileCatalog("test_db"); // Database test_db exists in hive metastore hiveShell.execute("use test_db"); // When the Hive table does not exist, specify the paimon table to create hive table in hive // metastore. - tEnv.executeSql("CALL sys.repair('test_db.t_repair_hive')"); + tEnv.executeSql( + isNamedArgument + ? "CALL sys.repair(`table` => 'test_db.t_repair_hive')" + : "CALL sys.repair('test_db.t_repair_hive')"); assertThat(hiveShell.executeQuery("SHOW PARTITIONS t_repair_hive")) .containsExactlyInAnyOrder("dt=2020-01-02/hh=09"); @@ -1639,7 +1700,10 @@ public void testRepairTable() throws Exception { // When the Hive table exists, specify the paimon table to update hive table in hive // metastore. - tEnv.executeSql("CALL sys.repair('test_db.t_repair_hive')"); + tEnv.executeSql( + isNamedArgument + ? "CALL sys.repair(`table` => 'test_db.t_repair_hive')" + : "CALL sys.repair('test_db.t_repair_hive')"); assertThat( hiveShell .executeQuery("DESC FORMATTED t_repair_hive") @@ -1651,6 +1715,15 @@ public void testRepairTable() throws Exception { @Test public void testRepairTableWithCustomLocation() throws Exception { + testRepairTableWithCustomLocation(false); + } + + @Test + public void testRepairTableWithCustomLocationAndNamedArgument() throws Exception { + testRepairTableWithCustomLocation(true); + } + + private void testRepairTableWithCustomLocation(boolean isNamedArgument) throws Exception { TableEnvironment fileCatalog = useFileCatalog("test_db"); // Database exists in hive metastore and uses custom location. String databaseLocation = path + "test_db.db"; @@ -1659,7 +1732,11 @@ public void testRepairTableWithCustomLocation() throws Exception { // When the Hive table does not exist, specify the paimon table to create hive table in hive // metastore. - tEnv.executeSql("CALL sys.repair('my_database.t_repair_hive')").await(); + tEnv.executeSql( + isNamedArgument + ? "CALL sys.repair(`table` => 'my_database.t_repair_hive')" + : "CALL sys.repair('my_database.t_repair_hive')") + .await(); String tableLocation = databaseLocation + "/t_repair_hive"; assertThat( @@ -1674,7 +1751,10 @@ public void testRepairTableWithCustomLocation() throws Exception { // When the Hive table exists, specify the paimon table to update hive table in hive // metastore. - tEnv.executeSql("CALL sys.repair('my_database.t_repair_hive')"); + tEnv.executeSql( + isNamedArgument + ? "CALL sys.repair(`table` => 'my_database.t_repair_hive')" + : "CALL sys.repair('my_database.t_repair_hive')"); assertThat( hiveShell .executeQuery("DESC FORMATTED t_repair_hive") @@ -1687,6 +1767,8 @@ public void testRepairTableWithCustomLocation() throws Exception { .isTrue(); assertThat(hiveShell.executeQuery("SHOW PARTITIONS t_repair_hive")) .containsExactlyInAnyOrder("dt=2020-01-02/hh=09", "dt=2020-01-03/hh=10"); + hiveShell.execute("DROP TABLE my_database.t_repair_hive"); + hiveShell.execute("DROP DATABASE my_database"); } @Test diff --git a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateDatabaseProcedureITCase.java b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateDatabaseProcedureITCase.java index ff02a352e248..7829a9981ede 100644 --- a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateDatabaseProcedureITCase.java +++ b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateDatabaseProcedureITCase.java @@ -32,13 +32,15 @@ import org.assertj.core.api.Assertions; import org.junit.jupiter.api.AfterEach; import org.junit.jupiter.api.BeforeEach; -import org.junit.jupiter.api.Test; import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.Arguments; +import org.junit.jupiter.params.provider.MethodSource; import org.junit.jupiter.params.provider.ValueSource; import java.util.HashMap; import java.util.List; import java.util.Map; +import java.util.stream.Stream; import static org.apache.paimon.hive.procedure.MigrateTableProcedureITCase.data; @@ -59,25 +61,23 @@ public void afterEach() throws Exception { TEST_HIVE_METASTORE.stop(); } - @Test - public void testOrc() throws Exception { - testUpgradeNonPartitionTable("orc"); - resetMetastore(); - testUpgradePartitionTable("orc"); - } - - @Test - public void testAvro() throws Exception { - testUpgradeNonPartitionTable("avro"); - resetMetastore(); - testUpgradePartitionTable("avro"); + private static Stream testArguments() { + return Stream.of( + Arguments.of("orc", true), + Arguments.of("avro", true), + Arguments.of("parquet", true), + Arguments.of("orc", false), + Arguments.of("avro", false), + Arguments.of("parquet", false)); } - @Test - public void testParquet() throws Exception { - testUpgradeNonPartitionTable("parquet"); + @ParameterizedTest + @MethodSource("testArguments") + public void testMigrateDatabaseProcedure(String format, boolean isNamedArgument) + throws Exception { + testUpgradeNonPartitionTable(format, isNamedArgument); resetMetastore(); - testUpgradePartitionTable("parquet"); + testUpgradePartitionTable(format, isNamedArgument); } private void resetMetastore() throws Exception { @@ -86,7 +86,7 @@ private void resetMetastore() throws Exception { TEST_HIVE_METASTORE.start(PORT); } - public void testUpgradePartitionTable(String format) throws Exception { + public void testUpgradePartitionTable(String format, boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql("CREATE CATALOG HIVE WITH ('type'='hive')"); tEnv.useCatalog("HIVE"); @@ -126,11 +126,19 @@ public void testUpgradePartitionTable(String format) throws Exception { + System.getProperty(HiveConf.ConfVars.METASTOREWAREHOUSE.varname) + "')"); tEnv.useCatalog("PAIMON"); - tEnv.executeSql( - "CALL sys.migrate_database('hive', 'my_database', 'file.format=" - + format - + "')") - .await(); + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.migrate_database(connector => 'hive', source_database => 'my_database', options => 'file.format=" + + format + + "')") + .await(); + } else { + tEnv.executeSql( + "CALL sys.migrate_database('hive', 'my_database', 'file.format=" + + format + + "')") + .await(); + } List r2 = ImmutableList.copyOf( tEnv.executeSql("SELECT * FROM my_database.hivetable1").collect()); @@ -142,7 +150,8 @@ public void testUpgradePartitionTable(String format) throws Exception { Assertions.assertThatList(r3).containsExactlyInAnyOrderElementsOf(r4); } - public void testUpgradeNonPartitionTable(String format) throws Exception { + public void testUpgradeNonPartitionTable(String format, boolean isNamedArgument) + throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql("CREATE CATALOG HIVE WITH ('type'='hive')"); tEnv.useCatalog("HIVE"); @@ -180,11 +189,19 @@ public void testUpgradeNonPartitionTable(String format) throws Exception { + System.getProperty(HiveConf.ConfVars.METASTOREWAREHOUSE.varname) + "')"); tEnv.useCatalog("PAIMON"); - tEnv.executeSql( - "CALL sys.migrate_database('hive', 'my_database', 'file.format=" - + format - + "')") - .await(); + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.migrate_database(connector => 'hive', source_database => 'my_database', options => 'file.format=" + + format + + "')") + .await(); + } else { + tEnv.executeSql( + "CALL sys.migrate_database('hive', 'my_database', 'file.format=" + + format + + "')") + .await(); + } List r2 = ImmutableList.copyOf( tEnv.executeSql("SELECT * FROM my_database.hivetable1").collect()); diff --git a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateFileProcedureITCase.java b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateFileProcedureITCase.java index 721578631ddc..3f9aebb044aa 100644 --- a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateFileProcedureITCase.java +++ b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateFileProcedureITCase.java @@ -32,12 +32,15 @@ import org.assertj.core.api.Assertions; import org.junit.jupiter.api.AfterEach; import org.junit.jupiter.api.BeforeEach; -import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.Arguments; +import org.junit.jupiter.params.provider.MethodSource; import java.util.HashMap; import java.util.List; import java.util.Map; import java.util.Random; +import java.util.stream.Stream; /** Tests for {@link MigrateFileProcedure}. */ public class MigrateFileProcedureITCase extends ActionITCaseBase { @@ -56,25 +59,24 @@ public void afterEach() throws Exception { TEST_HIVE_METASTORE.stop(); } - @Test - public void testOrc() throws Exception { - test("orc"); - testMigrateFileAction("orc"); + private static Stream testArguments() { + return Stream.of( + Arguments.of("orc", true), + Arguments.of("avro", true), + Arguments.of("parquet", true), + Arguments.of("orc", false), + Arguments.of("avro", false), + Arguments.of("parquet", false)); } - @Test - public void testAvro() throws Exception { - test("avro"); - testMigrateFileAction("avro"); + @ParameterizedTest + @MethodSource("testArguments") + public void testMigrateFile(String format, boolean isNamedArgument) throws Exception { + test(format, isNamedArgument); + testMigrateFileAction(format, isNamedArgument); } - @Test - public void testParquet() throws Exception { - test("parquet"); - testMigrateFileAction("parquet"); - } - - public void test(String format) throws Exception { + public void test(String format, boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql("CREATE CATALOG HIVE WITH ('type'='hive')"); tEnv.useCatalog("HIVE"); @@ -99,14 +101,21 @@ public void test(String format) throws Exception { tEnv.useCatalog("PAIMON"); tEnv.executeSql( "CREATE TABLE paimontable (id STRING, id2 INT, id3 INT) PARTITIONED BY (id2, id3) with ('bucket' = '-1');"); - tEnv.executeSql("CALL sys.migrate_file('hive', 'default.hivetable', 'default.paimontable')") - .await(); + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.migrate_file(connector => 'hive', source_table => 'default.hivetable', target_table => 'default.paimontable')") + .await(); + } else { + tEnv.executeSql( + "CALL sys.migrate_file('hive', 'default.hivetable', 'default.paimontable')") + .await(); + } List r2 = ImmutableList.copyOf(tEnv.executeSql("SELECT * FROM paimontable").collect()); Assertions.assertThatList(r1).containsExactlyInAnyOrderElementsOf(r2); } - public void testMigrateFileAction(String format) throws Exception { + public void testMigrateFileAction(String format, boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql("CREATE CATALOG HIVE WITH ('type'='hive')"); tEnv.useCatalog("HIVE"); @@ -137,9 +146,16 @@ public void testMigrateFileAction(String format) throws Exception { "CREATE TABLE paimontable01 (id STRING, id2 INT, id3 INT) PARTITIONED BY (id2, id3) with ('bucket' = '-1');"); tEnv.executeSql( "CREATE TABLE paimontable02 (id STRING, id2 INT, id3 INT) PARTITIONED BY (id2, id3) with ('bucket' = '-1');"); - tEnv.executeSql( - "CALL sys.migrate_file('hive', 'default.hivetable01', 'default.paimontable01', false)") - .await(); + + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.migrate_file(connector => 'hive', source_table => 'default.hivetable01', target_table => 'default.paimontable01', delete_origin => false)") + .await(); + } else { + tEnv.executeSql( + "CALL sys.migrate_file('hive', 'default.hivetable01', 'default.paimontable01', false)") + .await(); + } tEnv.useCatalog("PAIMON_GE"); Map catalogConf = new HashMap<>(); diff --git a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateTableProcedureITCase.java b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateTableProcedureITCase.java index 634f10a60be7..b88e2b95b998 100644 --- a/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateTableProcedureITCase.java +++ b/paimon-hive/paimon-hive-connector-common/src/test/java/org/apache/paimon/hive/procedure/MigrateTableProcedureITCase.java @@ -32,14 +32,16 @@ import org.assertj.core.api.Assertions; import org.junit.jupiter.api.AfterEach; import org.junit.jupiter.api.BeforeEach; -import org.junit.jupiter.api.Test; import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.Arguments; +import org.junit.jupiter.params.provider.MethodSource; import org.junit.jupiter.params.provider.ValueSource; import java.util.HashMap; import java.util.List; import java.util.Map; import java.util.Random; +import java.util.stream.Stream; /** Tests for {@link MigrateFileProcedure}. */ public class MigrateTableProcedureITCase extends ActionITCaseBase { @@ -58,25 +60,22 @@ public void afterEach() throws Exception { TEST_HIVE_METASTORE.stop(); } - @Test - public void testOrc() throws Exception { - testUpgradeNonPartitionTable("orc"); - resetMetastore(); - testUpgradePartitionTable("orc"); + private static Stream testArguments() { + return Stream.of( + Arguments.of("orc", true), + Arguments.of("avro", true), + Arguments.of("parquet", true), + Arguments.of("orc", false), + Arguments.of("avro", false), + Arguments.of("parquet", false)); } - @Test - public void testAvro() throws Exception { - testUpgradeNonPartitionTable("avro"); - resetMetastore(); - testUpgradePartitionTable("avro"); - } - - @Test - public void testParquet() throws Exception { - testUpgradeNonPartitionTable("parquet"); + @ParameterizedTest + @MethodSource("testArguments") + public void testMigrateProcedure(String format, boolean isNamedArgument) throws Exception { + testUpgradeNonPartitionTable(format, isNamedArgument); resetMetastore(); - testUpgradePartitionTable("parquet"); + testUpgradePartitionTable(format, isNamedArgument); } private void resetMetastore() throws Exception { @@ -85,7 +84,7 @@ private void resetMetastore() throws Exception { TEST_HIVE_METASTORE.start(PORT); } - public void testUpgradePartitionTable(String format) throws Exception { + public void testUpgradePartitionTable(String format, boolean isNamedArgument) throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql("CREATE CATALOG HIVE WITH ('type'='hive')"); tEnv.useCatalog("HIVE"); @@ -108,17 +107,26 @@ public void testUpgradePartitionTable(String format) throws Exception { + System.getProperty(HiveConf.ConfVars.METASTOREWAREHOUSE.varname) + "')"); tEnv.useCatalog("PAIMON"); - tEnv.executeSql( - "CALL sys.migrate_table('hive', 'default.hivetable', 'file.format=" - + format - + "')") - .await(); + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.migrate_table(connector => 'hive', source_table => 'default.hivetable', options => 'file.format=" + + format + + "')") + .await(); + } else { + tEnv.executeSql( + "CALL sys.migrate_table('hive', 'default.hivetable', 'file.format=" + + format + + "')") + .await(); + } List r2 = ImmutableList.copyOf(tEnv.executeSql("SELECT * FROM hivetable").collect()); Assertions.assertThatList(r1).containsExactlyInAnyOrderElementsOf(r2); } - public void testUpgradeNonPartitionTable(String format) throws Exception { + public void testUpgradeNonPartitionTable(String format, boolean isNamedArgument) + throws Exception { TableEnvironment tEnv = tableEnvironmentBuilder().batchMode().build(); tEnv.executeSql("CREATE CATALOG HIVE WITH ('type'='hive')"); tEnv.useCatalog("HIVE"); @@ -139,11 +147,19 @@ public void testUpgradeNonPartitionTable(String format) throws Exception { + System.getProperty(HiveConf.ConfVars.METASTOREWAREHOUSE.varname) + "')"); tEnv.useCatalog("PAIMON"); - tEnv.executeSql( - "CALL sys.migrate_table('hive', 'default.hivetable', 'file.format=" - + format - + "')") - .await(); + if (isNamedArgument) { + tEnv.executeSql( + "CALL sys.migrate_table(connector => 'hive', source_table => 'default.hivetable', options => 'file.format=" + + format + + "')") + .await(); + } else { + tEnv.executeSql( + "CALL sys.migrate_table('hive', 'default.hivetable', 'file.format=" + + format + + "')") + .await(); + } List r2 = ImmutableList.copyOf(tEnv.executeSql("SELECT * FROM hivetable").collect()); Assertions.assertThatList(r1).containsExactlyInAnyOrderElementsOf(r2);