2011-08-15 01:12:57 +00:00
# pragma once
2010-03-04 19:20:28 +00:00
2018-11-19 14:31:16 +00:00
# include <Core/Names.h>
2017-04-01 09:19:00 +00:00
# include <Core/QueryProcessingStage.h>
2019-05-17 14:34:25 +00:00
# include <DataStreams/IBlockStream_fwd.h>
2018-10-14 15:30:06 +00:00
# include <Databases/IDatabase.h>
2019-02-04 13:04:02 +00:00
# include <Interpreters/CancellationCode.h>
2019-05-17 14:34:25 +00:00
# include <Storages/IStorage_fwd.h>
2020-03-13 10:30:55 +00:00
# include <Interpreters/StorageID.h>
2019-05-17 14:34:25 +00:00
# include <Storages/SelectQueryInfo.h>
# include <Storages/TableStructureLockHolder.h>
2019-07-03 13:17:19 +00:00
# include <Storages/CheckResults.h>
2019-08-27 20:43:08 +00:00
# include <Storages/ColumnsDescription.h>
# include <Storages/IndicesDescription.h>
# include <Storages/ConstraintsDescription.h>
2019-12-26 18:17:05 +00:00
# include <Storages/StorageInMemoryMetadata.h>
2020-02-17 20:39:24 +00:00
# include <Storages/ColumnDependency.h>
2018-05-21 13:49:54 +00:00
# include <Common/ActionLock.h>
2019-05-17 14:34:25 +00:00
# include <Common/Exception.h>
# include <Common/RWLock.h>
2019-10-17 11:12:35 +00:00
# include <Common/TypePromotion.h>
2019-05-17 14:34:25 +00:00
# include <optional>
# include <shared_mutex>
2013-01-23 17:38:03 +00:00
2010-03-01 16:59:51 +00:00
namespace DB
{
2016-01-11 21:46:36 +00:00
namespace ErrorCodes
{
2018-11-22 21:19:58 +00:00
extern const int NOT_IMPLEMENTED ;
2016-01-11 21:46:36 +00:00
}
2013-06-17 07:01:31 +00:00
class Context ;
2014-03-19 10:45:13 +00:00
2018-05-21 13:49:54 +00:00
using StorageActionBlockType = size_t ;
2017-09-17 18:49:43 +00:00
class ASTCreateQuery ;
2010-03-01 16:59:51 +00:00
2017-01-21 04:24:28 +00:00
struct Settings ;
2019-10-04 17:46:36 +00:00
struct SettingChange ;
using SettingsChanges = std : : vector < SettingChange > ;
2017-01-21 04:24:28 +00:00
class AlterCommands ;
2018-06-13 20:02:27 +00:00
class MutationCommands ;
2020-03-17 13:49:50 +00:00
struct PartitionCommand ;
using PartitionCommands = std : : vector < PartitionCommand > ;
2017-01-21 04:24:28 +00:00
2019-09-13 12:59:48 +00:00
class IProcessor ;
using ProcessorPtr = std : : shared_ptr < IProcessor > ;
using Processors = std : : vector < ProcessorPtr > ;
2019-10-20 09:12:42 +00:00
class Pipe ;
using Pipes = std : : vector < Pipe > ;
2019-09-13 12:59:48 +00:00
2020-02-10 15:50:12 +00:00
class StoragePolicy ;
using StoragePolicyPtr = std : : shared_ptr < const StoragePolicy > ;
2019-07-17 01:24:37 +00:00
struct ColumnSize
{
size_t marks = 0 ;
size_t data_compressed = 0 ;
size_t data_uncompressed = 0 ;
void add ( const ColumnSize & other )
{
marks + = other . marks ;
data_compressed + = other . data_compressed ;
data_uncompressed + = other . data_uncompressed ;
}
} ;
2017-01-21 04:24:28 +00:00
2019-05-17 14:34:25 +00:00
/** Storage. Describes the table. Responsible for
2017-04-16 15:00:33 +00:00
* - storage of the table data ;
2017-06-06 17:06:14 +00:00
* - the definition in which files ( or not in files ) the data is stored ;
* - data lookups and appends ;
2017-04-16 15:00:33 +00:00
* - data storage structure ( compression , etc . )
* - concurrent access to data ( locks , etc . )
2010-03-01 16:59:51 +00:00
*/
2019-10-17 11:12:35 +00:00
class IStorage : public std : : enable_shared_from_this < IStorage > , public TypePromotion < IStorage >
2010-03-01 16:59:51 +00:00
{
public :
2019-12-03 16:25:32 +00:00
IStorage ( ) = delete ;
2020-05-26 12:14:50 +00:00
/// Storage fields should be initialized in separate methods like setColumns
/// or setTableTTLs.
explicit IStorage ( StorageID storage_id_ ) : storage_id ( std : : move ( storage_id_ ) ) { } //-V730
2019-05-17 14:34:25 +00:00
virtual ~ IStorage ( ) = default ;
IStorage ( const IStorage & ) = delete ;
IStorage & operator = ( const IStorage & ) = delete ;
2017-04-16 15:00:33 +00:00
/// The main name of the table type (for example, StorageMergeTree).
2017-04-01 07:20:54 +00:00
virtual std : : string getName ( ) const = 0 ;
2019-03-29 20:31:06 +00:00
/// The name of the table.
2019-12-27 19:30:22 +00:00
StorageID getStorageID ( ) const ;
2018-03-06 20:18:34 +00:00
2019-05-17 14:34:25 +00:00
/// Returns true if the storage receives data from a remote server or servers.
2017-04-01 07:20:54 +00:00
virtual bool isRemote ( ) const { return false ; }
2020-01-24 16:20:36 +00:00
/// Returns true if the storage is a view of a table or another view.
virtual bool isView ( ) const { return false ; }
2019-05-17 14:34:25 +00:00
/// Returns true if the storage supports queries with the SAMPLE section.
2020-05-21 19:46:03 +00:00
virtual bool supportsSampling ( ) const { return hasSamplingKey ( ) ; }
2017-04-01 07:20:54 +00:00
2019-05-17 14:34:25 +00:00
/// Returns true if the storage supports queries with the FINAL section.
2017-04-01 07:20:54 +00:00
virtual bool supportsFinal ( ) const { return false ; }
2019-05-17 14:34:25 +00:00
/// Returns true if the storage supports queries with the PREWHERE section.
2017-04-01 07:20:54 +00:00
virtual bool supportsPrewhere ( ) const { return false ; }
2019-05-17 14:34:25 +00:00
/// Returns true if the storage replicates SELECT, INSERT and ALTER commands among replicas.
2017-04-25 15:21:03 +00:00
virtual bool supportsReplication ( ) const { return false ; }
2019-12-12 10:49:15 +00:00
/// Returns true if the storage supports parallel insert.
virtual bool supportsParallelInsert ( ) const { return false ; }
2019-05-17 14:34:25 +00:00
/// Returns true if the storage supports deduplication of inserted data blocks.
2018-05-21 23:17:57 +00:00
virtual bool supportsDeduplication ( ) const { return false ; }
2017-04-25 15:21:03 +00:00
2019-08-07 15:21:45 +00:00
/// Returns true if the storage supports settings.
virtual bool supportsSettings ( ) const { return false ; }
2019-10-22 10:31:28 +00:00
/// Returns true if the blocks shouldn't be pushed to associated views on insert.
virtual bool noPushingToViews ( ) const { return false ; }
2020-01-24 15:10:24 +00:00
/// Read query returns streams which automatically distribute data between themselves.
/// So, it's impossible for one stream run out of data when there is data in other streams.
/// Example is StorageSystemNumbers.
2020-01-20 07:24:28 +00:00
virtual bool hasEvenlyDistributedRead ( ) const { return false ; }
2020-02-18 19:03:40 +00:00
/// Returns true if there is set table TTL, any column TTL or any move TTL.
2020-05-25 17:57:08 +00:00
virtual bool hasAnyTTL ( ) const { return hasAnyColumnTTL ( ) | | hasAnyTableTTL ( ) ; }
2020-02-18 19:03:40 +00:00
2019-07-17 01:24:37 +00:00
/// Optional size information of each physical column.
/// Currently it's only used by the MergeTree family for query optimizations.
2019-07-16 17:13:12 +00:00
using ColumnSizeByName = std : : unordered_map < std : : string , ColumnSize > ;
virtual ColumnSizeByName getColumnSizes ( ) const { return { } ; }
2019-03-05 10:12:20 +00:00
2019-05-18 09:31:51 +00:00
public : /// thread-unsafe part. lockStructure must be acquired
2019-07-01 22:12:31 +00:00
virtual const ColumnsDescription & getColumns ( ) const ; /// returns combined set of columns
2019-07-07 23:57:58 +00:00
virtual void setColumns ( ColumnsDescription columns_ ) ; /// sets only real columns, possibly overwrites virtual ones.
2019-05-17 14:34:25 +00:00
const IndicesDescription & getIndices ( ) const ;
2019-05-19 06:08:25 +00:00
const ConstraintsDescription & getConstraints ( ) const ;
void setConstraints ( ConstraintsDescription constraints_ ) ;
2019-12-27 15:01:41 +00:00
/// Returns storage metadata copy. Direct modification of
/// result structure doesn't affect storage.
2019-12-26 18:17:05 +00:00
virtual StorageInMemoryMetadata getInMemoryMetadata ( ) const ;
2019-05-22 19:38:43 +00:00
Block getSampleBlock ( ) const ; /// ordinary + materialized.
2019-06-18 16:32:37 +00:00
Block getSampleBlockWithVirtuals ( ) const ; /// ordinary + materialized + virtuals.
2019-05-22 19:38:43 +00:00
Block getSampleBlockNonMaterialized ( ) const ; /// ordinary.
Block getSampleBlockForColumns ( const Names & column_names ) const ; /// ordinary + materialized + aliases + virtuals.
2019-05-17 14:34:25 +00:00
/// Verify that all the requested names are in the table and are set correctly:
/// list of names is not empty and the names do not repeat.
2019-08-07 16:10:14 +00:00
void check ( const Names & column_names , bool include_virtuals = false ) const ;
2019-05-17 14:34:25 +00:00
/// Check that all the requested names are in the table and have the correct types.
void check ( const NamesAndTypesList & columns ) const ;
/// Check that all names from the intersection of `names` and `columns` are in the table and have the same types.
void check ( const NamesAndTypesList & columns , const Names & column_names ) const ;
/// Check that the data block contains all the columns of the table with the correct types,
/// contains only the columns of the table, and all the columns are different.
/// If |need_all| is set, then checks that all the columns of the table are in the block.
void check ( const Block & block , bool need_all = false ) const ;
2020-04-27 17:46:51 +00:00
/// Return list of virtual columns (like _part, _table, etc). In the vast
/// majority of cases virtual columns are static constant part of Storage
/// class and don't depend on Storage object. But sometimes we have fake
/// storages, like Merge, which works as proxy for other storages and it's
/// virtual columns must contain virtual columns from underlying table.
///
/// User can create columns with the same name as virtual column. After that
/// virtual column will be overriden and inaccessible.
///
/// By default return empty list of columns.
2020-04-28 10:38:57 +00:00
virtual NamesAndTypesList getVirtuals ( ) const ;
2020-05-20 15:16:39 +00:00
2019-05-21 11:24:32 +00:00
protected : /// still thread-unsafe part.
void setIndices ( IndicesDescription indices_ ) ;
/// Returns whether the column is virtual - by default all columns are real.
/// Initially reserved virtual column name may be shadowed by real column.
2020-04-23 15:18:33 +00:00
bool isVirtualColumn ( const String & column_name ) const ;
2019-08-27 09:34:53 +00:00
2019-05-17 14:34:25 +00:00
private :
2019-12-12 12:30:31 +00:00
StorageID storage_id ;
2019-12-03 16:25:32 +00:00
mutable std : : mutex id_mutex ;
2020-05-20 12:16:55 +00:00
2020-05-01 10:05:37 +00:00
ColumnsDescription columns ;
2019-05-17 14:34:25 +00:00
IndicesDescription indices ;
2019-05-19 06:08:25 +00:00
ConstraintsDescription constraints ;
2019-05-17 14:34:25 +00:00
2020-05-20 12:16:55 +00:00
StorageMetadataKeyField partition_key ;
2020-05-20 18:11:38 +00:00
StorageMetadataKeyField primary_key ;
StorageMetadataKeyField sorting_key ;
2020-05-20 15:16:39 +00:00
StorageMetadataKeyField sampling_key ;
2020-05-20 12:16:55 +00:00
2020-05-25 17:57:08 +00:00
StorageMetadataTTLColumnFields column_ttls_by_name ;
2020-05-25 17:07:14 +00:00
StorageMetadataTableTTL table_ttl ;
2020-04-07 11:34:35 +00:00
private :
2020-04-27 15:21:37 +00:00
RWLockImpl : : LockHolder tryLockTimed (
const RWLock & rwlock , RWLockImpl : : Type type , const String & query_id , const SettingSeconds & acquire_timeout ) const ;
2020-04-07 11:34:35 +00:00
2019-05-17 14:34:25 +00:00
public :
2019-03-07 18:34:46 +00:00
/// Acquire this lock if you need the table structure to remain constant during the execution of
/// the query. If will_add_new_data is true, this means that the query will add new data to the table
/// (INSERT or a parts merge).
2020-04-27 15:17:08 +00:00
TableStructureReadLockHolder lockStructureForShare ( bool will_add_new_data , const String & query_id , const SettingSeconds & acquire_timeout ) ;
2017-04-01 07:20:54 +00:00
2019-03-07 18:34:46 +00:00
/// Acquire this lock at the start of ALTER to lock out other ALTERs and make sure that only you
/// can modify the table structure. It can later be upgraded to the exclusive lock.
2020-04-09 18:10:27 +00:00
TableStructureWriteLockHolder lockAlterIntention ( const String & query_id , const SettingSeconds & acquire_timeout ) ;
2017-04-01 07:20:54 +00:00
2019-03-07 18:34:46 +00:00
/// Upgrade alter intention lock to the full exclusive structure lock. This is done by ALTER queries
/// to ensure that no other query uses the table structure and it can be safely changed.
2020-04-27 15:17:08 +00:00
void lockStructureExclusively ( TableStructureWriteLockHolder & lock_holder , const String & query_id , const SettingSeconds & acquire_timeout ) ;
2019-03-05 10:12:20 +00:00
2019-03-07 18:34:46 +00:00
/// Acquire the full exclusive lock immediately. No other queries can run concurrently.
2020-04-09 18:10:27 +00:00
TableStructureWriteLockHolder lockExclusively ( const String & query_id , const SettingSeconds & acquire_timeout ) ;
2017-04-01 07:20:54 +00:00
2018-04-19 14:47:09 +00:00
/** Returns stage to which query is going to be processed in read() function.
* ( Normally , the function only reads the columns from the list , but in other cases ,
* for example , the request can be partially processed on a remote server . )
2020-03-22 17:43:01 +00:00
*
* SelectQueryInfo is required since the stage can depends on the query
* ( see Distributed ( ) engine and optimize_skip_unused_shards ) .
2020-04-01 18:38:01 +00:00
*
* QueryProcessingStage : : Enum required for Distributed over Distributed ,
* since it cannot return Complete for intermediate queries never .
2018-04-19 14:47:09 +00:00
*/
2020-04-01 18:38:01 +00:00
QueryProcessingStage : : Enum getQueryProcessingStage ( const Context & context ) const
{
return getQueryProcessingStage ( context , QueryProcessingStage : : Complete , { } ) ;
}
virtual QueryProcessingStage : : Enum getQueryProcessingStage ( const Context & , QueryProcessingStage : : Enum /*to_stage*/ , const ASTPtr & ) const
{
return QueryProcessingStage : : FetchColumns ;
}
2017-04-01 07:20:54 +00:00
2019-05-28 21:17:48 +00:00
/** Watch live changes to the table.
* Accepts a list of columns to read , as well as a description of the query ,
* from which information can be extracted about how to retrieve data
* ( indexes , locks , etc . )
* Returns a stream with which you can read data sequentially
* or multiple streams for parallel data reading .
* The ` processed_stage ` info is also written to what stage the request was processed .
* ( Normally , the function only reads the columns from the list , but in other cases ,
* for example , the request can be partially processed on a remote server . )
*
* context contains settings for one query .
* Usually Storage does not care about these settings , since they are used in the interpreter .
* But , for example , for distributed query processing , the settings are passed to the remote server .
*
* num_streams - a recommendation , how many streams to return ,
* if the storage can return a different number of streams .
*
* It is guaranteed that the structure of the table will not change over the lifetime of the returned streams ( that is , there will not be ALTER , RENAME and DROP ) .
*/
virtual BlockInputStreams watch (
const Names & /*column_names*/ ,
const SelectQueryInfo & /*query_info*/ ,
const Context & /*context*/ ,
QueryProcessingStage : : Enum & /*processed_stage*/ ,
size_t /*max_block_size*/ ,
unsigned /*num_streams*/ )
{
throw Exception ( " Method watch is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-04-16 15:00:33 +00:00
/** Read a set of columns from the table.
* Accepts a list of columns to read , as well as a description of the query ,
* from which information can be extracted about how to retrieve data
* ( indexes , locks , etc . )
* Returns a stream with which you can read data sequentially
* or multiple streams for parallel data reading .
2018-04-19 14:47:09 +00:00
* The ` processed_stage ` must be the result of getQueryProcessingStage ( ) function .
2017-04-01 07:20:54 +00:00
*
2017-05-24 21:06:29 +00:00
* context contains settings for one query .
2017-04-16 15:00:33 +00:00
* Usually Storage does not care about these settings , since they are used in the interpreter .
* But , for example , for distributed query processing , the settings are passed to the remote server .
2017-04-01 07:20:54 +00:00
*
2017-06-02 15:54:39 +00:00
* num_streams - a recommendation , how many streams to return ,
* if the storage can return a different number of streams .
2017-04-01 07:20:54 +00:00
*
2017-04-16 15:00:33 +00:00
* It is guaranteed that the structure of the table will not change over the lifetime of the returned streams ( that is , there will not be ALTER , RENAME and DROP ) .
2017-04-01 07:20:54 +00:00
*/
2020-02-19 16:02:37 +00:00
virtual Pipes read (
2019-09-13 12:59:48 +00:00
const Names & /*column_names*/ ,
2017-12-01 19:34:51 +00:00
const SelectQueryInfo & /*query_info*/ ,
const Context & /*context*/ ,
2018-04-19 14:47:09 +00:00
QueryProcessingStage : : Enum /*processed_stage*/ ,
2019-02-18 17:31:18 +00:00
size_t /*max_block_size*/ ,
2017-12-01 19:34:51 +00:00
unsigned /*num_streams*/ )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method read is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2020-02-19 16:02:37 +00:00
/** The same as read, but returns BlockInputStreams.
*/
BlockInputStreams readStreams (
const Names & /*column_names*/ ,
const SelectQueryInfo & /*query_info*/ ,
const Context & /*context*/ ,
QueryProcessingStage : : Enum /*processed_stage*/ ,
size_t /*max_block_size*/ ,
unsigned /*num_streams*/ ) ;
2019-11-05 17:33:03 +00:00
2017-04-16 15:00:33 +00:00
/** Writes the data to a table.
* Receives a description of the query , which can contain information about the data write method .
* Returns an object by which you can write data sequentially .
2017-04-01 07:20:54 +00:00
*
2017-04-16 15:00:33 +00:00
* It is guaranteed that the table structure will not change over the lifetime of the returned streams ( that is , there will not be ALTER , RENAME and DROP ) .
2017-04-01 07:20:54 +00:00
*/
virtual BlockOutputStreamPtr write (
2017-12-01 19:34:51 +00:00
const ASTPtr & /*query*/ ,
2019-02-27 18:26:24 +00:00
const Context & /*context*/ )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method write is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-04-16 15:00:33 +00:00
/** Delete the table data. Called before deleting the directory with the data.
2018-06-09 15:48:22 +00:00
* The method can be called only after detaching table from Context ( when no queries are performed with table ) .
* The table is not usable during and after call to this method .
2020-01-22 11:30:11 +00:00
* If some queries may still use the table , then it must be called under exclusive lock .
2017-04-16 15:00:33 +00:00
* If you do not need any action other than deleting the directory with data , you can leave this method blank .
2017-04-01 07:20:54 +00:00
*/
2020-01-22 11:30:11 +00:00
virtual void drop ( ) { }
2017-04-01 07:20:54 +00:00
2018-06-09 15:48:22 +00:00
/** Clear the table data and leave it empty.
2018-06-09 18:17:27 +00:00
* Must be called under lockForAlter .
2018-04-21 00:35:20 +00:00
*/
2019-08-27 20:43:08 +00:00
virtual void truncate ( const ASTPtr & /*query*/ , const Context & /* context */ , TableStructureWriteLockHolder & )
2018-04-21 00:35:20 +00:00
{
throw Exception ( " Truncate is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-04-16 15:00:33 +00:00
/** Rename the table.
* Renaming a name in a file with metadata , the name in the list of tables in the RAM , is done separately .
* In this function , you need to rename the directory with the data , if any .
* Called when the table structure is locked for write .
2020-04-07 14:05:51 +00:00
* Table UUID must remain unchanged , unless table moved between Ordinary and Atomic databases .
2017-04-01 07:20:54 +00:00
*/
2020-04-07 14:05:51 +00:00
virtual void rename ( const String & /*new_path_to_table_data*/ , const StorageID & new_table_id )
2017-04-01 07:20:54 +00:00
{
2020-04-07 14:05:51 +00:00
renameInMemory ( new_table_id ) ;
2017-04-01 07:20:54 +00:00
}
2019-12-03 16:25:32 +00:00
/**
* Just updates names of database and table without moving any data on disk
2019-12-12 12:30:31 +00:00
* Can be called directly only from DatabaseAtomic .
2019-12-03 16:25:32 +00:00
*/
2020-04-07 14:05:51 +00:00
virtual void renameInMemory ( const StorageID & new_table_id ) ;
2019-12-03 16:25:32 +00:00
2017-04-16 15:00:33 +00:00
/** ALTER tables in the form of column changes that do not affect the change to Storage or its parameters.
* This method must fully execute the ALTER query , taking care of the locks itself .
* To update the table metadata on disk , this method should call InterpreterAlterQuery : : updateMetadata .
2017-04-01 07:20:54 +00:00
*/
2019-08-26 14:50:34 +00:00
virtual void alter ( const AlterCommands & params , const Context & context , TableStructureWriteLockHolder & table_lock_holder ) ;
2018-05-21 13:49:54 +00:00
2019-12-27 15:07:53 +00:00
/** Checks that alter commands can be applied to storage. For example, columns can be modified,
* or primary key can be changes , etc .
*/
2019-12-26 18:17:05 +00:00
virtual void checkAlterIsPossible ( const AlterCommands & commands , const Settings & settings ) ;
2018-11-13 13:48:53 +00:00
/** ALTER tables with regard to its partitions.
* Should handle locks for each command on its own .
2017-04-01 07:20:54 +00:00
*/
2018-11-26 14:43:40 +00:00
virtual void alterPartition ( const ASTPtr & /* query */ , const PartitionCommands & /* commands */ , const Context & /* context */ )
2017-04-01 07:20:54 +00:00
{
2018-11-13 13:48:53 +00:00
throw Exception ( " Partition operations are not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
2017-04-01 07:20:54 +00:00
}
2017-04-16 15:00:33 +00:00
/** Perform any background work. For example, combining parts in a MergeTree type table.
* Returns whether any work has been done .
2017-04-01 07:20:54 +00:00
*/
2017-12-01 19:34:51 +00:00
virtual bool optimize ( const ASTPtr & /*query*/ , const ASTPtr & /*partition*/ , bool /*final*/ , bool /*deduplicate*/ , const Context & /*context*/ )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method optimize is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2018-02-02 16:02:43 +00:00
/// Mutate the table contents
virtual void mutate ( const MutationCommands & , const Context & )
{
throw Exception ( " Mutations are not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2019-01-10 18:19:29 +00:00
/// Cancel a mutation.
2019-02-04 13:04:02 +00:00
virtual CancellationCode killMutation ( const String & /*mutation_id*/ )
2019-01-10 18:19:29 +00:00
{
throw Exception ( " Mutations are not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-06-06 17:06:14 +00:00
/** If the table have to do some complicated work on startup,
* that must be postponed after creation of table object
* ( like launching some background threads ) ,
* do it in this method .
* You should call this method after creation of object .
* By default , does nothing .
* Cannot be called simultaneously by multiple threads .
*/
virtual void startup ( ) { }
/** If the table have to do some complicated work when destroying an object - do it in advance.
2017-04-16 15:00:33 +00:00
* For example , if the table contains any threads for background work - ask them to complete and wait for completion .
2017-06-06 17:06:14 +00:00
* By default , does nothing .
2017-04-16 15:00:33 +00:00
* Can be called simultaneously from different threads , even after a call to drop ( ) .
2017-04-01 07:20:54 +00:00
*/
virtual void shutdown ( ) { }
2018-05-21 13:49:54 +00:00
/// Asks table to stop executing some action identified by action_type
/// If table does not support such type of lock, and empty lock is returned
2018-05-28 15:37:30 +00:00
virtual ActionLock getActionLock ( StorageActionBlockType /* action_type */ )
2018-05-21 13:49:54 +00:00
{
return { } ;
}
2019-06-12 13:11:44 +00:00
std : : atomic < bool > is_dropped { false } ;
2017-04-01 07:20:54 +00:00
/// Does table support index for IN sections
virtual bool supportsIndexForIn ( ) const { return false ; }
2018-01-23 08:18:12 +00:00
/// Provides a hint that the storage engine may evaluate the IN-condition by using an index.
2019-02-27 18:26:24 +00:00
virtual bool mayBenefitFromIndexForIn ( const ASTPtr & /* left_in_operand */ , const Context & /* query_context */ ) const { return false ; }
2018-01-21 07:30:07 +00:00
2017-04-01 07:20:54 +00:00
/// Checks validity of the data
2019-07-03 13:17:19 +00:00
virtual CheckResults checkData ( const ASTPtr & /* query */ , const Context & /* context */ ) { throw Exception ( " Check query is not supported for " + getName ( ) + " storage " , ErrorCodes : : NOT_IMPLEMENTED ) ; }
2017-04-01 07:20:54 +00:00
/// Checks that table could be dropped right now
2018-08-03 09:54:46 +00:00
/// Otherwise - throws an exception with detailed information.
2018-08-03 13:19:53 +00:00
/// We do not use mutex because it is not very important that the size could change during the operation.
2018-08-03 09:39:01 +00:00
virtual void checkTableCanBeDropped ( ) const { }
2017-01-23 19:18:25 +00:00
2018-08-01 17:41:18 +00:00
/// Checks that Partition could be dropped right now
2018-08-03 09:54:46 +00:00
/// Otherwise - throws an exception with detailed information.
2018-08-03 13:19:53 +00:00
/// We do not use mutex because it is not very important that the size could change during the operation.
2018-08-03 09:39:01 +00:00
virtual void checkPartitionCanBeDropped ( const ASTPtr & /*partition*/ ) { }
2017-01-23 19:18:25 +00:00
2019-08-01 10:29:14 +00:00
/// Returns data paths if storage supports it, empty vector otherwise.
2019-04-04 13:13:59 +00:00
virtual Strings getDataPaths ( ) const { return { } ; }
2018-02-21 19:26:59 +00:00
2020-05-22 12:34:26 +00:00
/// Returns structure with partition key.
2020-05-21 19:46:03 +00:00
const StorageMetadataKeyField & getPartitionKey ( ) const ;
/// Set partition key for storage (methods bellow, are just wrappers for this
2020-05-22 12:34:26 +00:00
/// struct).
2020-05-21 19:46:03 +00:00
void setPartitionKey ( const StorageMetadataKeyField & partition_key_ ) ;
2018-11-27 17:07:10 +00:00
/// Returns ASTExpressionList of partition key expression for storage or nullptr if there is none.
2020-05-21 19:46:03 +00:00
ASTPtr getPartitionKeyAST ( ) const { return partition_key . definition_ast ; }
2020-05-22 12:34:26 +00:00
/// Storage has partition key.
2020-05-21 19:46:03 +00:00
bool hasPartitionKey ( ) const ;
2020-05-13 11:04:27 +00:00
/// Returns column names that need to be read to calculate partition key.
2020-05-21 19:46:03 +00:00
Names getColumnsRequiredForPartitionKey ( ) const ;
2018-11-19 14:31:16 +00:00
2020-05-21 19:46:03 +00:00
2020-05-22 12:34:26 +00:00
/// Returns structure with sorting key.
2020-05-21 19:46:03 +00:00
const StorageMetadataKeyField & getSortingKey ( ) const ;
/// Set sorting key for storage (methods bellow, are just wrappers for this
2020-05-22 12:34:26 +00:00
/// struct).
2020-05-21 19:46:03 +00:00
void setSortingKey ( const StorageMetadataKeyField & sorting_key_ ) ;
/// Returns ASTExpressionList of sorting key expression for storage or nullptr if there is none.
ASTPtr getSortingKeyAST ( ) const { return sorting_key . definition_ast ; }
2020-05-22 12:34:26 +00:00
/// Storage has sorting key.
2020-05-21 19:46:03 +00:00
bool hasSortingKey ( ) const ;
2020-05-13 11:04:27 +00:00
/// Returns column names that need to be read to calculate sorting key.
2020-05-21 19:46:03 +00:00
Names getColumnsRequiredForSortingKey ( ) const ;
/// Returns columns names in sorting key specified by user in ORDER BY
/// expression. For example: 'a', 'x * y', 'toStartOfMonth(date)', etc.
Names getSortingKeyColumns ( ) const ;
2018-11-19 14:31:16 +00:00
2020-05-22 12:34:26 +00:00
/// Returns structure with primary key.
2020-05-21 19:46:03 +00:00
const StorageMetadataKeyField & getPrimaryKey ( ) const ;
/// Set primary key for storage (methods bellow, are just wrappers for this
2020-05-22 12:34:26 +00:00
/// struct).
2020-05-21 19:46:03 +00:00
void setPrimaryKey ( const StorageMetadataKeyField & primary_key_ ) ;
/// Returns ASTExpressionList of primary key expression for storage or nullptr if there is none.
ASTPtr getPrimaryKeyAST ( ) const { return primary_key . definition_ast ; }
2020-05-22 12:34:26 +00:00
/// Storage has user-defined (in CREATE query) sorting key.
2020-05-21 19:46:03 +00:00
bool isPrimaryKeyDefined ( ) const ;
2020-05-22 12:34:26 +00:00
/// Storage has primary key (maybe part of some other key).
2020-05-21 19:46:03 +00:00
bool hasPrimaryKey ( ) const ;
2020-05-13 11:04:27 +00:00
/// Returns column names that need to be read to calculate primary key.
2020-05-21 19:46:03 +00:00
Names getColumnsRequiredForPrimaryKey ( ) const ;
/// Returns columns names in sorting key specified by. For example: 'a', 'x
/// * y', 'toStartOfMonth(date)', etc.
Names getPrimaryKeyColumns ( ) const ;
2018-11-19 14:31:16 +00:00
2020-05-22 12:34:26 +00:00
/// Returns structure with sampling key.
2020-05-21 19:46:03 +00:00
const StorageMetadataKeyField & getSamplingKey ( ) const ;
/// Set sampling key for storage (methods bellow, are just wrappers for this
2020-05-22 12:34:26 +00:00
/// struct).
2020-05-21 19:46:03 +00:00
void setSamplingKey ( const StorageMetadataKeyField & sampling_key_ ) ;
/// Returns sampling expression AST for storage or nullptr if there is none.
ASTPtr getSamplingKeyAST ( ) const { return sampling_key . definition_ast ; }
2020-05-22 12:34:26 +00:00
/// Storage has sampling key.
2020-05-21 19:46:03 +00:00
bool hasSamplingKey ( ) const ;
2020-05-13 11:04:27 +00:00
/// Returns column names that need to be read to calculate sampling key.
2020-05-21 19:46:03 +00:00
Names getColumnsRequiredForSampling ( ) const ;
2018-11-27 17:07:10 +00:00
2020-05-13 11:04:27 +00:00
/// Returns column names that need to be read for FINAL to work.
2020-05-21 19:46:03 +00:00
Names getColumnsRequiredForFinal ( ) const { return getColumnsRequiredForSortingKey ( ) ; }
2018-11-19 14:31:16 +00:00
2019-05-18 12:30:41 +00:00
2020-05-25 17:57:08 +00:00
/// Returns columns, which will be needed to calculate dependencies (skip
/// indices, TTL expressions) if we update @updated_columns set of columns.
2020-02-17 20:39:24 +00:00
virtual ColumnDependencies getColumnDependencies ( const NameSet & /* updated_columns */ ) const { return { } ; }
2020-05-25 17:57:08 +00:00
/// Returns storage policy if storage supports it.
2019-11-27 09:39:44 +00:00
virtual StoragePolicyPtr getStoragePolicy ( ) const { return { } ; }
2014-03-09 17:36:01 +00:00
2020-05-25 17:57:08 +00:00
/// Common tables TTLs (for rows and moves).
2020-05-25 17:07:14 +00:00
const StorageMetadataTableTTL & getTableTTLs ( ) const ;
void setTableTTLs ( const StorageMetadataTableTTL & table_ttl_ ) ;
bool hasAnyTableTTL ( ) const ;
2020-05-25 17:57:08 +00:00
/// Separate TTLs for columns.
const StorageMetadataTTLColumnFields & getColumnTTLs ( ) const ;
void setColumnTTLs ( const StorageMetadataTTLColumnFields & column_ttls_by_name_ ) ;
bool hasAnyColumnTTL ( ) const ;
2020-05-25 17:07:14 +00:00
2020-05-25 17:57:08 +00:00
/// Just wrapper for table TTLs, return rows part of table TTLs.
const StorageMetadataTTLField & getRowsTTL ( ) const ;
bool hasRowsTTL ( ) const ;
/// Just wrapper for table TTLs, return moves (to disks or volumes) parts of
/// table TTL.
const StorageMetadataTTLFields & getMoveTTLs ( ) const ;
bool hasAnyMoveTTL ( ) const ;
2020-05-25 17:07:14 +00:00
2020-03-29 08:38:38 +00:00
/// If it is possible to quickly determine exact number of rows in the table at this moment of time, then return it.
/// Used for:
/// - Simple count() opimization
/// - For total_rows column in system.tables
///
/// Does takes underlying Storage (if any) into account.
2019-10-28 17:27:43 +00:00
virtual std : : optional < UInt64 > totalRows ( ) const
{
return { } ;
}
2020-03-29 08:38:38 +00:00
/// If it is possible to quickly determine exact number of bytes for the table on storage:
/// - memory (approximated)
/// - disk (compressed)
///
/// Used for:
/// - For total_bytes column in system.tables
//
/// Does not takes underlying Storage (if any) into account
/// (since for Buffer we still need to know how much bytes it uses).
virtual std : : optional < UInt64 > totalBytes ( ) const
{
return { } ;
}
2013-01-23 17:38:03 +00:00
private :
2019-03-05 10:12:20 +00:00
/// You always need to take the next three locks in this order.
2017-04-01 07:20:54 +00:00
2019-03-07 18:34:46 +00:00
/// If you hold this lock exclusively, you can be sure that no other structure modifying queries
/// (e.g. ALTER, DROP) are concurrently executing. But queries that only read table structure
/// (e.g. SELECT, INSERT) can continue to execute.
2020-04-01 12:43:09 +00:00
mutable RWLock alter_intention_lock = RWLockImpl : : create ( ) ;
/// It is taken for share for the entire INSERT query and the entire merge of the parts (for MergeTree).
/// ALTER COLUMN queries acquire an exclusive lock to ensure that no new parts with the old structure
/// are added to the table and thus the set of parts to modify doesn't change.
mutable RWLock new_data_structure_lock = RWLockImpl : : create ( ) ;
2017-04-01 07:20:54 +00:00
2019-03-07 18:34:46 +00:00
/// Lock for the table column structure (names, types, etc.) and data path.
/// It is taken in exclusive mode by queries that modify them (e.g. RENAME, ALTER and DROP)
/// and in share mode by other queries.
2018-11-27 16:45:45 +00:00
mutable RWLock structure_lock = RWLockImpl : : create ( ) ;
2013-01-23 17:38:03 +00:00
} ;
2010-03-01 16:59:51 +00:00
2011-08-15 01:12:57 +00:00
}