trait 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: add an array of partitions and any data they contain to the table#dropPartitions: remove an array of partitions and any data they contain from the table#purgePartitions: remove an array of partitions and any data they contain from the table by skipping a trash even if it is supported#truncatePartitions: truncate an array of partitions by removing partitions data
- Annotations
- @Experimental()
- Source
- SupportsAtomicPartitionManagement.java
- Since
3.1.0
- Alphabetic
- By Inheritance
- SupportsAtomicPartitionManagement
- SupportsPartitionManagement
- Table
- AnyRef
- Any
- Hide All
- Show All
- Public
- Protected
Abstract Value Members
- abstract def capabilities(): Set[TableCapability]
Returns the set of capabilities for this table.
Returns the set of capabilities for this table.
- Definition Classes
- Table
- abstract def createPartitions(idents: Array[InternalRow], properties: Array[Map[String, String]]): Unit
Create an array of partitions atomically in table.
Create an array of partitions atomically in table.
If any partition already exists, the operation of createPartitions need to be safely rolled back.
- idents
an array of new partition identifiers
- properties
the metadata of the partitions
- Exceptions thrown
PartitionsAlreadyExistExceptionIf any partition already exists for the identifierUnsupportedOperationExceptionIf partition property is not supported
- abstract def dropPartitions(idents: Array[InternalRow]): Boolean
Drop an array of partitions atomically from table.
Drop an array of partitions atomically from table.
If any partition doesn't exists, the operation of dropPartitions need to be safely rolled back.
- idents
an array of partition identifiers
- returns
true if partitions were deleted, false if any partition not exists
- abstract def listPartitionIdentifiers(names: Array[String], ident: InternalRow): Array[InternalRow]
List the identifiers of all partitions that match to the ident by names.
List the identifiers of all partitions that match to the ident by names.
- names
the names of partition values in the identifier.
- ident
a partition identifier values.
- returns
an array of Identifiers for the partitions
- Definition Classes
- SupportsPartitionManagement
- abstract def loadPartitionMetadata(ident: InternalRow): Map[String, String]
Retrieve the partition metadata of the existing partition.
Retrieve the partition metadata of the existing partition.
- ident
a partition identifier
- returns
the metadata of the partition
- Definition Classes
- SupportsPartitionManagement
- Exceptions thrown
UnsupportedOperationExceptionIf partition property is not supported
- abstract def name(): String
A name to identify this table.
A name to identify this table. Implementations should provide a meaningful name, like the database and table name from catalog, or the location of files for this table.
- Definition Classes
- Table
- abstract def partitionSchema(): StructType
Get the partition schema of table, this must be consistent with $
Table#partitioning().Get the partition schema of table, this must be consistent with $
Table#partitioning().- returns
the partition schema of table
- Definition Classes
- SupportsPartitionManagement
- abstract def replacePartitionMetadata(ident: InternalRow, properties: Map[String, String]): Unit
Replace the partition metadata of the existing partition.
Replace the partition metadata of the existing partition.
- ident
the partition identifier of the existing partition
- properties
the new metadata of the partition
- Definition Classes
- SupportsPartitionManagement
- Exceptions thrown
NoSuchPartitionExceptionIf the partition identifier to alter doesn't existUnsupportedOperationExceptionIf partition property is not supported
- abstract def schema(): StructType
Returns the schema of this table.
Returns the schema of this table. If the table is not readable and doesn't have a schema, an empty schema can be returned here.
- Definition Classes
- Table
- Annotations
- @Deprecated
- Deprecated
(Since version 3.4.0)
Concrete Value Members
- final def !=(arg0: Any): Boolean
- Definition Classes
- AnyRef → Any
- final def ##: Int
- Definition Classes
- AnyRef → Any
- final def ==(arg0: Any): Boolean
- Definition Classes
- AnyRef → Any
- final def asInstanceOf[T0]: T0
- Definition Classes
- Any
- def clone(): AnyRef
- Attributes
- protected[lang]
- Definition Classes
- AnyRef
- Annotations
- @throws(classOf[java.lang.CloneNotSupportedException]) @IntrinsicCandidate() @native()
- def columns(): Array[Column]
Returns the columns of this table.
Returns the columns of this table. If the table is not readable and doesn't have a schema, an empty array can be returned here.
- Definition Classes
- Table
- def createPartition(ident: InternalRow, properties: Map[String, String]): Unit
Create a partition in table.
Create a partition in table.
- ident
a new partition identifier
- properties
the metadata of a partition
- Definition Classes
- SupportsAtomicPartitionManagement → SupportsPartitionManagement
- Annotations
- @SuppressWarnings() @Override()
- Exceptions thrown
PartitionsAlreadyExistExceptionIf a partition already exists for the identifierUnsupportedOperationExceptionIf partition property is not supported
- def dropPartition(ident: InternalRow): Boolean
Drop a partition from table.
Drop a partition from table.
- ident
a partition identifier
- returns
true if a partition was deleted, false if no partition exists for the identifier
- Definition Classes
- SupportsAtomicPartitionManagement → SupportsPartitionManagement
- Annotations
- @Override()
- final def eq(arg0: AnyRef): Boolean
- Definition Classes
- AnyRef
- def equals(arg0: AnyRef): Boolean
- Definition Classes
- AnyRef → Any
- final def getClass(): Class[_ <: AnyRef]
- Definition Classes
- AnyRef → Any
- Annotations
- @IntrinsicCandidate() @native()
- def hashCode(): Int
- Definition Classes
- AnyRef → Any
- Annotations
- @IntrinsicCandidate() @native()
- final def isInstanceOf[T0]: Boolean
- Definition Classes
- Any
- final def ne(arg0: AnyRef): Boolean
- Definition Classes
- AnyRef
- final def notify(): Unit
- Definition Classes
- AnyRef
- Annotations
- @IntrinsicCandidate() @native()
- final def notifyAll(): Unit
- Definition Classes
- AnyRef
- Annotations
- @IntrinsicCandidate() @native()
- def partitionExists(ident: InternalRow): Boolean
Test whether a partition exists using an
identfrom the table.Test whether a partition exists using an
identfrom the table.- ident
a partition identifier which must contain all partition fields in order
- returns
true if the partition exists, false otherwise
- Definition Classes
- SupportsPartitionManagement
- def partitioning(): Array[Transform]
Returns the physical partitioning of this table.
Returns the physical partitioning of this table.
- Definition Classes
- Table
- def properties(): Map[String, String]
Returns the string map of table properties.
Returns the string map of table properties.
- Definition Classes
- Table
- def purgePartition(ident: InternalRow): Boolean
Drop a partition from the table and completely remove partition data by skipping a trash even if it is supported.
Drop a partition from the table and completely remove partition data by skipping a trash even if it is supported.
- ident
a partition identifier
- returns
true if a partition was deleted, false if no partition exists for the identifier
- Definition Classes
- SupportsPartitionManagement
- Since
3.2.0
- Exceptions thrown
NoSuchPartitionExceptionIf the partition identifier to alter doesn't existUnsupportedOperationExceptionIf partition purging is not supported
- def purgePartitions(idents: Array[InternalRow]): Boolean
Drop an array of partitions atomically from table, and completely remove partitions data by skipping a trash even if it is supported.
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.
- idents
an array of partition identifiers
- returns
true if partitions were deleted, false if any partition not exists
- Since
3.2.0
- Exceptions thrown
NoSuchPartitionExceptionIf any partition identifier to alter doesn't existUnsupportedOperationExceptionIf partition purging is not supported
- def renamePartition(from: InternalRow, to: InternalRow): Boolean
Rename an existing partition of the table.
Rename an existing partition of the table.
- from
an existing partition identifier to rename
- to
new partition identifier
- returns
true if renaming completes successfully otherwise false
- Definition Classes
- SupportsPartitionManagement
- Since
3.2.0
- Exceptions thrown
NoSuchPartitionExceptionIf thefrompartition does not existPartitionsAlreadyExistExceptionIf thetopartition exists alreadyUnsupportedOperationExceptionIf partition renaming is not supported
- final def synchronized[T0](arg0: => T0): T0
- Definition Classes
- AnyRef
- def toString(): String
- Definition Classes
- AnyRef → Any
- def truncatePartition(ident: InternalRow): Boolean
Truncate a partition in the table by completely removing partition data.
Truncate a partition in the table by completely removing partition data.
- ident
a partition identifier
- returns
true if the partition was truncated successfully otherwise false
- Definition Classes
- SupportsPartitionManagement
- Since
3.2.0
- Exceptions thrown
NoSuchPartitionExceptionIf the partition identifier to alter doesn't existUnsupportedOperationExceptionIf partition truncation is not supported
- def truncatePartitions(idents: Array[InternalRow]): Boolean
Truncate an array of partitions atomically from table, and completely remove partitions data.
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.
- idents
an array of partition identifiers
- returns
true if partitions were truncated successfully otherwise false
- Since
3.2.0
- Exceptions thrown
NoSuchPartitionExceptionIf any partition identifier to truncate doesn't existUnsupportedOperationExceptionIf partition truncate is not supported
- final def wait(arg0: Long, arg1: Int): Unit
- Definition Classes
- AnyRef
- Annotations
- @throws(classOf[java.lang.InterruptedException])
- final def wait(arg0: Long): Unit
- Definition Classes
- AnyRef
- Annotations
- @throws(classOf[java.lang.InterruptedException]) @native()
- final def wait(): Unit
- Definition Classes
- AnyRef
- Annotations
- @throws(classOf[java.lang.InterruptedException])
Deprecated Value Members
- def finalize(): Unit
- Attributes
- protected[lang]
- Definition Classes
- AnyRef
- Annotations
- @throws(classOf[java.lang.Throwable]) @Deprecated
- Deprecated
(Since version 9)