Interface SupportsPartitionManagement

All Superinterfaces:
Table
All Known Subinterfaces:
SupportsAtomicPartitionManagement

@Experimental public interface SupportsPartitionManagement extends Table
A partition interface of Table. A partition is composed of identifier and properties, and properties contains metadata information of the partition.

These APIs are used to modify table partition identifier or partition metadata. In some cases, they will change the table data as well.

Since:
3.1.0
  • Method Summary

    Modifier and Type
    Method
    Description
    void
    createPartition(org.apache.spark.sql.catalyst.InternalRow ident, Map<String,String> properties)
    Create a partition in table.
    boolean
    dropPartition(org.apache.spark.sql.catalyst.InternalRow ident)
    Drop a partition from table.
    org.apache.spark.sql.catalyst.InternalRow[]
    listPartitionIdentifiers(String[] names, org.apache.spark.sql.catalyst.InternalRow ident)
    List the identifiers of all partitions that match to the ident by names.
    loadPartitionMetadata(org.apache.spark.sql.catalyst.InternalRow ident)
    Retrieve the partition metadata of the existing partition.
    default boolean
    partitionExists(org.apache.spark.sql.catalyst.InternalRow ident)
    Test whether a partition exists using an ident from the table.
    Get the partition schema of table, this must be consistent with $Table.partitioning().
    default boolean
    purgePartition(org.apache.spark.sql.catalyst.InternalRow ident)
    Drop a partition from the table and completely remove partition data by skipping a trash even if it is supported.
    default boolean
    renamePartition(org.apache.spark.sql.catalyst.InternalRow from, org.apache.spark.sql.catalyst.InternalRow to)
    Rename an existing partition of the table.
    void
    replacePartitionMetadata(org.apache.spark.sql.catalyst.InternalRow ident, Map<String,String> properties)
    Replace the partition metadata of the existing partition.
    default boolean
    truncatePartition(org.apache.spark.sql.catalyst.InternalRow ident)
    Truncate a partition in the table by completely removing partition data.

    Methods inherited from interface org.apache.spark.sql.connector.catalog.Table

    capabilities, columns, name, partitioning, properties, schema
  • Method Details

    • partitionSchema

      StructType partitionSchema()
      Get the partition schema of table, this must be consistent with $Table.partitioning().
      Returns:
      the partition schema of table
    • createPartition

      void createPartition(org.apache.spark.sql.catalyst.InternalRow ident, Map<String,String> properties) throws org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException, UnsupportedOperationException
      Create a partition in table.
      Parameters:
      ident - a new partition identifier
      properties - the metadata of a partition
      Throws:
      org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException - If a partition already exists for the identifier
      UnsupportedOperationException - If partition property is not supported
    • dropPartition

      boolean dropPartition(org.apache.spark.sql.catalyst.InternalRow ident)
      Drop a partition from table.
      Parameters:
      ident - a partition identifier
      Returns:
      true if a partition was deleted, false if no partition exists for the identifier
    • purgePartition

      default boolean purgePartition(org.apache.spark.sql.catalyst.InternalRow ident) throws org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException, UnsupportedOperationException
      Drop a partition from the table and completely remove partition data by skipping a trash even if it is supported.
      Parameters:
      ident - a partition identifier
      Returns:
      true if a partition was deleted, false if no partition exists for the identifier
      Throws:
      org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException - If the partition identifier to alter doesn't exist
      UnsupportedOperationException - If partition purging is not supported
      Since:
      3.2.0
    • partitionExists

      default boolean partitionExists(org.apache.spark.sql.catalyst.InternalRow ident)
      Test whether a partition exists using an ident from the table.
      Parameters:
      ident - a partition identifier which must contain all partition fields in order
      Returns:
      true if the partition exists, false otherwise
    • replacePartitionMetadata

      void replacePartitionMetadata(org.apache.spark.sql.catalyst.InternalRow ident, Map<String,String> properties) throws org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException, UnsupportedOperationException
      Replace the partition metadata of the existing partition.
      Parameters:
      ident - the partition identifier of the existing partition
      properties - the new metadata of the partition
      Throws:
      org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException - If the partition identifier to alter doesn't exist
      UnsupportedOperationException - If partition property is not supported
    • loadPartitionMetadata

      Map<String,String> loadPartitionMetadata(org.apache.spark.sql.catalyst.InternalRow ident) throws UnsupportedOperationException
      Retrieve the partition metadata of the existing partition.
      Parameters:
      ident - a partition identifier
      Returns:
      the metadata of the partition
      Throws:
      UnsupportedOperationException - If partition property is not supported
    • listPartitionIdentifiers

      org.apache.spark.sql.catalyst.InternalRow[] listPartitionIdentifiers(String[] names, org.apache.spark.sql.catalyst.InternalRow ident)
      List the identifiers of all partitions that match to the ident by names.
      Parameters:
      names - the names of partition values in the identifier.
      ident - a partition identifier values.
      Returns:
      an array of Identifiers for the partitions
    • renamePartition

      default boolean renamePartition(org.apache.spark.sql.catalyst.InternalRow from, org.apache.spark.sql.catalyst.InternalRow to) throws UnsupportedOperationException, org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException, org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException
      Rename an existing partition of the table.
      Parameters:
      from - an existing partition identifier to rename
      to - new partition identifier
      Returns:
      true if renaming completes successfully otherwise false
      Throws:
      UnsupportedOperationException - If partition renaming is not supported
      org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException - If the `to` partition exists already
      org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException - If the `from` partition does not exist
      Since:
      3.2.0
    • truncatePartition

      default boolean truncatePartition(org.apache.spark.sql.catalyst.InternalRow ident) throws org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException, UnsupportedOperationException
      Truncate a partition in the table by completely removing partition data.
      Parameters:
      ident - a partition identifier
      Returns:
      true if the partition was truncated successfully otherwise false
      Throws:
      org.apache.spark.sql.catalyst.analysis.NoSuchPartitionException - If the partition identifier to alter doesn't exist
      UnsupportedOperationException - If partition truncation is not supported
      Since:
      3.2.0