2011-08-15 01:12:57 +00:00
# pragma once
2010-03-04 19:20:28 +00:00
2017-04-01 09:19:00 +00:00
# include <Core/Names.h>
# include <Common/Exception.h>
2017-09-01 15:05:23 +00:00
# include <Common/RWLockFIFO.h>
2017-04-01 09:19:00 +00:00
# include <Core/QueryProcessingStage.h>
# include <Storages/ITableDeclaration.h>
2017-07-15 03:48:36 +00:00
# include <Storages/SelectQueryInfo.h>
2017-07-28 17:34:02 +00:00
# include <shared_mutex>
2015-02-10 21:10:58 +00:00
# include <memory>
2017-11-20 04:15:43 +00:00
# include <optional>
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
{
2017-04-01 07:20:54 +00:00
extern const int TABLE_IS_DROPPED ;
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
class IBlockInputStream ;
class IBlockOutputStream ;
2017-09-01 18:22:03 +00:00
class RWLockFIFO ;
2017-09-01 15:05:23 +00:00
using RWLockFIFOPtr = std : : shared_ptr < RWLockFIFO > ;
2016-05-28 12:22:22 +00:00
using BlockOutputStreamPtr = std : : shared_ptr < IBlockOutputStream > ;
using BlockInputStreamPtr = std : : shared_ptr < IBlockInputStream > ;
2016-05-28 10:15:36 +00:00
using BlockInputStreams = std : : vector < BlockInputStreamPtr > ;
2013-06-17 07:01:31 +00:00
2017-09-17 18:49:43 +00:00
class ASTCreateQuery ;
2010-03-01 16:59:51 +00:00
2014-03-24 13:24:47 +00:00
class IStorage ;
2016-05-28 10:15:36 +00:00
using StoragePtr = std : : shared_ptr < IStorage > ;
2014-03-24 13:24:47 +00:00
2017-01-21 04:24:28 +00:00
struct Settings ;
class AlterCommands ;
2017-04-16 15:00:33 +00:00
/** Does not allow changing the table description (including rename and delete the table).
* If during any operation the table structure should remain unchanged , you need to hold such a lock for all of its time .
* For example , you need to hold such a lock for the duration of the entire SELECT or INSERT query and for the whole time the merge of the set of parts
* ( but between the selection of parts for the merge and their merging , the table structure can change ) .
* NOTE : This is a lock to " read " the table ' s description . To change the table description , you need to take the TableStructureWriteLock .
2017-01-21 04:24:28 +00:00
*/
class TableStructureReadLock
{
private :
2017-04-01 07:20:54 +00:00
friend class IStorage ;
2017-01-21 04:24:28 +00:00
2017-04-01 07:20:54 +00:00
StoragePtr storage ;
2017-04-16 15:00:33 +00:00
/// Order is important.
2017-09-01 15:05:23 +00:00
RWLockFIFO : : LockHandler data_lock ;
RWLockFIFO : : LockHandler structure_lock ;
2017-01-21 04:24:28 +00:00
public :
2017-09-01 15:05:23 +00:00
TableStructureReadLock ( StoragePtr storage_ , bool lock_structure , bool lock_data , const std : : string & who ) ;
2017-01-21 04:24:28 +00:00
} ;
2017-09-01 15:05:23 +00:00
2017-01-21 04:24:28 +00:00
using TableStructureReadLockPtr = std : : shared_ptr < TableStructureReadLock > ;
using TableStructureReadLocks = std : : vector < TableStructureReadLockPtr > ;
2017-09-01 15:05:23 +00:00
using TableStructureWriteLock = RWLockFIFO : : LockHandler ;
using TableDataWriteLock = RWLockFIFO : : LockHandler ;
2017-07-28 17:34:02 +00:00
using TableFullWriteLock = std : : pair < TableDataWriteLock , TableStructureWriteLock > ;
2017-01-21 04:24:28 +00:00
2014-03-24 13:24:47 +00:00
2017-04-16 15:00:33 +00:00
/** Storage. Responsible for
* - 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
*/
2016-08-26 21:25:05 +00:00
class IStorage : public std : : enable_shared_from_this < IStorage > , private boost : : noncopyable , public ITableDeclaration
2010-03-01 16:59:51 +00:00
{
public :
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 ;
2017-06-06 17:06:14 +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 ; }
2017-04-16 15:00:33 +00:00
/** Returns true if the storage supports queries with the SAMPLE section. */
2017-04-01 07:20:54 +00:00
virtual bool supportsSampling ( ) const { return false ; }
2017-04-16 15:00:33 +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 ; }
2017-04-16 15:00:33 +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 ; }
2017-06-06 17:06:14 +00:00
/** Returns true if the storage supports read from multiple replicas. Assumed isRemote. */
2017-04-01 07:20:54 +00:00
virtual bool supportsParallelReplicas ( ) const { return false ; }
2017-04-25 15:21:03 +00:00
/** Returns true if the storage replicates SELECT, INSERT and ALTER commands among replicas. */
virtual bool supportsReplication ( ) const { return false ; }
2017-04-16 15:00:33 +00:00
/** Does not allow you to change the structure or name of the table.
* If you change the data in the table , you will need to specify will_modify_data = true .
* This will take an extra lock that does not allow starting ALTER MODIFY .
2017-09-01 17:11:41 +00:00
* Parameter ' who ' identifies a client of the lock ( ALTER query , merge process , etc ) , used for diagnostic purposes .
2017-04-01 07:20:54 +00:00
*
2017-04-16 15:00:33 +00:00
* WARNING : You need to call methods from ITableDeclaration under such a lock . Without it , they are not thread safe .
2017-04-01 07:20:54 +00:00
* WARNING : To avoid deadlocks , this method must not be called under lock of Context .
*/
2017-09-01 17:11:41 +00:00
TableStructureReadLockPtr lockStructure ( bool will_modify_data , const std : : string & who )
2017-04-01 07:20:54 +00:00
{
2017-09-01 15:05:23 +00:00
TableStructureReadLockPtr res = std : : make_shared < TableStructureReadLock > ( shared_from_this ( ) , true , will_modify_data , who ) ;
2017-04-01 07:20:54 +00:00
if ( is_dropped )
throw Exception ( " Table is dropped " , ErrorCodes : : TABLE_IS_DROPPED ) ;
return res ;
}
2017-04-16 15:00:33 +00:00
/** Does not allow reading the table structure. It is taken for ALTER, RENAME and DROP.
2017-04-01 07:20:54 +00:00
*/
2017-09-01 15:05:23 +00:00
TableFullWriteLock lockForAlter ( const std : : string & who = " Alter " )
2017-04-01 07:20:54 +00:00
{
2017-04-16 15:00:33 +00:00
/// The calculation order is important.
2017-09-01 15:05:23 +00:00
auto data_lock = lockDataForAlter ( who ) ;
auto structure_lock = lockStructureForAlter ( who ) ;
2017-04-01 07:20:54 +00:00
return { std : : move ( data_lock ) , std : : move ( structure_lock ) } ;
}
2017-04-16 15:00:33 +00:00
/** Does not allow changing the data in the table. (Moreover, does not give a look at the structure of the table with the intention to change the data).
* It is taken during write temporary data in ALTER MODIFY .
* Under this lock , you can take lockStructureForAlter ( ) to change the structure of the table .
2017-04-01 07:20:54 +00:00
*/
2017-09-01 15:05:23 +00:00
TableDataWriteLock lockDataForAlter ( const std : : string & who = " Alter " )
2017-04-01 07:20:54 +00:00
{
2017-09-01 15:05:23 +00:00
auto res = data_lock - > getLock ( RWLockFIFO : : Write , who ) ;
2017-04-01 07:20:54 +00:00
if ( is_dropped )
throw Exception ( " Table is dropped " , ErrorCodes : : TABLE_IS_DROPPED ) ;
return res ;
}
2017-09-01 15:05:23 +00:00
TableStructureWriteLock lockStructureForAlter ( const std : : string & who = " Alter " )
2017-04-01 07:20:54 +00:00
{
2017-09-01 15:05:23 +00:00
auto res = structure_lock - > getLock ( RWLockFIFO : : Write , who ) ;
2017-04-01 07:20:54 +00:00
if ( is_dropped )
throw Exception ( " Table is dropped " , ErrorCodes : : TABLE_IS_DROPPED ) ;
return res ;
}
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 .
2017-07-15 03:48:36 +00:00
* The ` processed_stage ` info is also written to what stage the request was processed .
2017-04-16 15:00:33 +00:00
* ( 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 . )
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
*/
virtual BlockInputStreams read (
const Names & column_names ,
2017-07-15 03:48:36 +00:00
const SelectQueryInfo & query_info ,
2017-04-01 07:20:54 +00:00
const Context & context ,
QueryProcessingStage : : Enum & processed_stage ,
2017-06-02 15:54:39 +00:00
size_t max_block_size ,
unsigned num_streams )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method read is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
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-05-21 22:25:25 +00:00
const ASTPtr & query ,
2017-04-01 07:20:54 +00:00
const Settings & settings )
{
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.
* 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
*/
virtual void drop ( ) { }
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 .
2017-04-01 07:20:54 +00:00
*/
virtual void rename ( const String & new_path_to_db , const String & new_database_name , const String & new_table_name )
{
throw Exception ( " Method rename is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
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
*/
virtual void alter ( const AlterCommands & params , const String & database_name , const String & table_name , const Context & context )
{
throw Exception ( " Method alter is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-06-22 15:17:27 +00:00
/** Execute CLEAR COLUMN ... IN PARTITION query which removes column from given partition. */
2017-09-06 20:34:26 +00:00
virtual void clearColumnInPartition ( const ASTPtr & partition , const Field & column_name , const Context & context )
2017-04-14 12:40:48 +00:00
{
throw Exception ( " Method dropColumnFromPartition is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-04-16 15:00:33 +00:00
/** Run the query (DROP|DETACH) PARTITION.
2017-04-01 07:20:54 +00:00
*/
2017-09-06 20:34:26 +00:00
virtual void dropPartition ( const ASTPtr & query , const ASTPtr & partition , bool detach , const Context & context )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method dropPartition is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-05-24 21:38:56 +00:00
/** Run the ATTACH request (PART|PARTITION).
2017-04-01 07:20:54 +00:00
*/
2017-09-06 20:34:26 +00:00
virtual void attachPartition ( const ASTPtr & partition , bool part , const Context & context )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method attachPartition is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-04-16 15:00:33 +00:00
/** Run the FETCH PARTITION query.
2017-04-01 07:20:54 +00:00
*/
2017-09-06 20:34:26 +00:00
virtual void fetchPartition ( const ASTPtr & partition , const String & from , const Context & context )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method fetchPartition is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
2017-04-16 15:00:33 +00:00
/** Run the FREEZE PARTITION request. That is, create a local backup (snapshot) of data using the `localBackup` function (see localBackup.h)
2017-04-01 07:20:54 +00:00
*/
2017-09-06 20:34:26 +00:00
virtual void freezePartition ( const ASTPtr & partition , const String & with_name , const Context & context )
2017-04-01 07:20:54 +00:00
{
throw Exception ( " Method freezePartition is not supported by storage " + getName ( ) , ErrorCodes : : NOT_IMPLEMENTED ) ;
}
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-09-06 20:34:26 +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 ) ;
}
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 ( ) { }
bool is_dropped { false } ;
/// Does table support index for IN sections
virtual bool supportsIndexForIn ( ) const { return false ; }
/// Checks validity of the data
virtual bool checkData ( ) const { throw DB : : Exception ( " Check query is not supported for " + getName ( ) + " storage " ) ; }
/// Checks that table could be dropped right now
/// If it can - returns true
/// Otherwise - throws an exception with detailed information or returns false
virtual bool checkTableCanBeDropped ( ) const { return true ; }
2017-01-23 19:18:25 +00:00
2017-10-03 23:43:55 +00:00
/** Notify engine about updated dependencies for this storage. */
virtual void updateDependencies ( ) { }
2013-02-06 11:26:35 +00:00
protected :
2017-04-01 07:20:54 +00:00
using ITableDeclaration : : ITableDeclaration ;
using std : : enable_shared_from_this < IStorage > : : shared_from_this ;
2014-03-09 17:36:01 +00:00
2013-01-23 17:38:03 +00:00
private :
2017-04-01 07:20:54 +00:00
friend class TableStructureReadLock ;
2017-04-16 15:00:33 +00:00
/// You always need to take the next two locks in this order.
2017-04-01 07:20:54 +00:00
2017-04-16 15:00:33 +00:00
/** It is taken for read for the entire INSERT query and the entire merge of the parts (for MergeTree).
* It is taken for write for the entire time ALTER MODIFY .
2017-04-01 07:20:54 +00:00
*
2017-04-16 15:00:33 +00:00
* Formally :
* Taking a write lock ensures that :
* 1 ) the data in the table will not change while the lock is alive ,
* 2 ) all changes to the data after releasing the lock will be based on the structure of the table at the time after the lock was released .
* You need to take for read for the entire time of the operation that changes the data .
2017-04-01 07:20:54 +00:00
*/
2017-09-01 15:05:23 +00:00
mutable RWLockFIFOPtr data_lock = RWLockFIFO : : create ( ) ;
2017-04-01 07:20:54 +00:00
2017-04-16 15:00:33 +00:00
/** Lock for multiple columns and path to table. It is taken for write at RENAME, ALTER (for ALTER MODIFY for a while) and DROP.
* It is taken for read for the whole time of SELECT , INSERT and merge parts ( for MergeTree ) .
2017-04-01 07:20:54 +00:00
*
2017-04-16 15:00:33 +00:00
* Taking this lock for writing is a strictly " stronger " operation than taking parts_writing_lock for write record .
* That is , if this lock is taken for write , you should not worry about ` parts_writing_lock ` .
* parts_writing_lock is only needed for cases when you do not want to take ` table_structure_lock ` for long operations ( ALTER MODIFY ) .
2017-04-01 07:20:54 +00:00
*/
2017-09-01 15:05:23 +00:00
mutable RWLockFIFOPtr structure_lock = RWLockFIFO : : create ( ) ;
2013-01-23 17:38:03 +00:00
} ;
2010-03-01 16:59:51 +00:00
2017-04-16 15:00:33 +00:00
/// table name -> table
2015-09-04 20:52:00 +00:00
using Tables = std : : map < String , StoragePtr > ;
2011-08-15 01:12:57 +00:00
}