Interface SupportsAtomicPartitionManagement
- All Superinterfaces:
SupportsPartitionManagement,Table
@Experimental
public interface SupportsAtomicPartitionManagement
extends SupportsPartitionManagement
An atomic partition interface of
Table to operate multiple partitions atomically.
These APIs are used to modify table partition or partition metadata, they will change the table data as well.
createPartitions(org.apache.spark.sql.catalyst.InternalRow[], java.util.Map<java.lang.String, java.lang.String>[]): add an array of partitions and any data they contain to the tabledropPartitions(org.apache.spark.sql.catalyst.InternalRow[]): remove an array of partitions and any data they contain from the tablepurgePartitions(org.apache.spark.sql.catalyst.InternalRow[]): remove an array of partitions and any data they contain from the table by skipping a trash even if it is supportedtruncatePartitions(org.apache.spark.sql.catalyst.InternalRow[]): truncate an array of partitions by removing partitions data
- Since:
- 3.1.0
-
Method Summary
Modifier and TypeMethodDescriptiondefault voidcreatePartition(org.apache.spark.sql.catalyst.InternalRow ident, Map<String, String> properties) Create a partition in table.voidcreatePartitions(org.apache.spark.sql.catalyst.InternalRow[] idents, Map<String, String>[] properties) Create an array of partitions atomically in table.default booleandropPartition(org.apache.spark.sql.catalyst.InternalRow ident) Drop a partition from table.booleandropPartitions(org.apache.spark.sql.catalyst.InternalRow[] idents) Drop an array of partitions atomically from table.default booleanpurgePartitions(org.apache.spark.sql.catalyst.InternalRow[] idents) Drop an array of partitions atomically from table, and completely remove partitions data by skipping a trash even if it is supported.default booleantruncatePartitions(org.apache.spark.sql.catalyst.InternalRow[] idents) Truncate an array of partitions atomically from table, and completely remove partitions data.Methods inherited from interface org.apache.spark.sql.connector.catalog.SupportsPartitionManagement
listPartitionIdentifiers, loadPartitionMetadata, partitionExists, partitionSchema, purgePartition, renamePartition, replacePartitionMetadata, truncatePartitionMethods inherited from interface org.apache.spark.sql.connector.catalog.Table
capabilities, columns, name, partitioning, properties, schema
-
Method Details
-
createPartition
default void createPartition(org.apache.spark.sql.catalyst.InternalRow ident, Map<String, String> properties) throws org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException, UnsupportedOperationExceptionDescription copied from interface:SupportsPartitionManagementCreate a partition in table.- Specified by:
createPartitionin interfaceSupportsPartitionManagement- Parameters:
ident- a new partition identifierproperties- the metadata of a partition- Throws:
org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException- If a partition already exists for the identifierUnsupportedOperationException- If partition property is not supported
-
dropPartition
default boolean dropPartition(org.apache.spark.sql.catalyst.InternalRow ident) Description copied from interface:SupportsPartitionManagementDrop a partition from table.- Specified by:
dropPartitionin interfaceSupportsPartitionManagement- Parameters:
ident- a partition identifier- Returns:
- true if a partition was deleted, false if no partition exists for the identifier
-
createPartitions
void createPartitions(org.apache.spark.sql.catalyst.InternalRow[] idents, Map<String, String>[] properties) throws org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException, UnsupportedOperationExceptionCreate an array of partitions atomically in table.If any partition already exists, the operation of createPartitions need to be safely rolled back.
- Parameters:
idents- an array of new partition identifiersproperties- the metadata of the partitions- Throws:
org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException- If any partition already exists for the identifierUnsupportedOperationException- If partition property is not supported
-
dropPartitions
boolean dropPartitions(org.apache.spark.sql.catalyst.InternalRow[] idents) Drop an array of partitions atomically from table.If any partition doesn't exists, the operation of dropPartitions need to be safely rolled back.
- Parameters:
idents- an array of partition identifiers- Returns:
- true if partitions were deleted, false if any partition not exists
-
purgePartitions
default boolean purgePartitions(org.apache.spark.sql.catalyst.InternalRow[] idents) throws org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException, UnsupportedOperationException Drop an array of partitions atomically from table, and completely remove partitions data by skipping a trash even if it is supported.If any partition doesn't exists, the operation of purgePartitions need to be safely rolled back.
- Parameters:
idents- an array of partition identifiers- Returns:
- true if partitions were deleted, false if any partition not exists
- Throws:
org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException- If any partition identifier to alter doesn't existUnsupportedOperationException- If partition purging is not supported- Since:
- 3.2.0
-
truncatePartitions
default boolean truncatePartitions(org.apache.spark.sql.catalyst.InternalRow[] idents) throws org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException, SparkUnsupportedOperationException Truncate an array of partitions atomically from table, and completely remove partitions data.If any partition doesn't exists, the operation of truncatePartitions need to be safely rolled back.
- Parameters:
idents- an array of partition identifiers- Returns:
- true if partitions were truncated successfully otherwise false
- Throws:
org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException- If any partition identifier to truncate doesn't existUnsupportedOperationException- If partition truncate is not supportedSparkUnsupportedOperationException- Since:
- 3.2.0
-