2020-02-03 12:54:36 +00:00
|
|
|
#pragma once
|
|
|
|
#include <Storages/IStorage_fwd.h>
|
2020-03-13 10:30:55 +00:00
|
|
|
#include <Interpreters/StorageID.h>
|
2020-03-10 19:36:17 +00:00
|
|
|
#include <Parsers/IAST_fwd.h>
|
2020-02-03 12:54:36 +00:00
|
|
|
#include <Core/UUID.h>
|
2020-02-12 16:54:26 +00:00
|
|
|
#include <Poco/Logger.h>
|
2020-02-03 12:54:36 +00:00
|
|
|
#include <boost/noncopyable.hpp>
|
|
|
|
#include <memory>
|
|
|
|
#include <map>
|
2020-02-12 16:54:26 +00:00
|
|
|
#include <set>
|
2020-02-03 12:54:36 +00:00
|
|
|
#include <unordered_map>
|
|
|
|
#include <mutex>
|
|
|
|
#include <array>
|
2020-03-19 21:14:52 +00:00
|
|
|
#include <list>
|
2020-02-03 12:54:36 +00:00
|
|
|
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
class Context;
|
|
|
|
class IDatabase;
|
|
|
|
class Exception;
|
2020-03-13 15:41:36 +00:00
|
|
|
class ColumnsDescription;
|
2020-02-12 16:54:26 +00:00
|
|
|
|
2020-02-03 12:54:36 +00:00
|
|
|
using DatabasePtr = std::shared_ptr<IDatabase>;
|
|
|
|
using DatabaseAndTable = std::pair<DatabasePtr, StoragePtr>;
|
2020-02-10 18:19:35 +00:00
|
|
|
using Databases = std::map<String, std::shared_ptr<IDatabase>>;
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-02-12 16:54:26 +00:00
|
|
|
/// Table -> set of table-views that make SELECT from it.
|
|
|
|
using ViewDependencies = std::map<StorageID, std::set<StorageID>>;
|
|
|
|
using Dependencies = std::vector<StorageID>;
|
2020-02-10 18:31:52 +00:00
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
|
2020-02-10 18:31:52 +00:00
|
|
|
/// Allows executing DDL query only in one thread.
|
|
|
|
/// Puts an element into the map, locks tables's mutex, counts how much threads run parallel query on the table,
|
|
|
|
/// when counter is 0 erases element in the destructor.
|
|
|
|
/// If the element already exists in the map, waits, when ddl query will be finished in other thread.
|
|
|
|
class DDLGuard
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
struct Entry
|
|
|
|
{
|
|
|
|
std::unique_ptr<std::mutex> mutex;
|
|
|
|
UInt32 counter;
|
|
|
|
};
|
|
|
|
|
|
|
|
/// Element name -> (mutex, counter).
|
|
|
|
/// NOTE: using std::map here (and not std::unordered_map) to avoid iterator invalidation on insertion.
|
|
|
|
using Map = std::map<String, Entry>;
|
|
|
|
|
|
|
|
DDLGuard(Map & map_, std::unique_lock<std::mutex> guards_lock_, const String & elem);
|
|
|
|
~DDLGuard();
|
|
|
|
|
|
|
|
private:
|
|
|
|
Map & map;
|
|
|
|
Map::iterator it;
|
|
|
|
std::unique_lock<std::mutex> guards_lock;
|
|
|
|
std::unique_lock<std::mutex> table_lock;
|
|
|
|
};
|
|
|
|
|
2020-03-10 19:36:17 +00:00
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
/// Creates temporary table in `_temporary_and_external_tables` with randomly generated unique StorageID.
|
|
|
|
/// Such table can be accessed from everywhere by its ID.
|
|
|
|
/// Removes the table from database on destruction.
|
|
|
|
/// TemporaryTableHolder object can be attached to a query or session Context, so table will be accessible through the context.
|
2020-03-10 19:36:17 +00:00
|
|
|
struct TemporaryTableHolder : boost::noncopyable
|
|
|
|
{
|
|
|
|
typedef std::function<StoragePtr(const StorageID &)> Creator;
|
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
TemporaryTableHolder() = default;
|
2020-03-10 19:36:17 +00:00
|
|
|
TemporaryTableHolder(const Context & context, const Creator & creator, const ASTPtr & query = {});
|
|
|
|
|
|
|
|
/// Creates temporary table with Engine=Memory
|
|
|
|
TemporaryTableHolder(const Context & context, const ColumnsDescription & columns, const ASTPtr & query = {});
|
|
|
|
|
|
|
|
TemporaryTableHolder(TemporaryTableHolder && rhs);
|
|
|
|
TemporaryTableHolder & operator = (TemporaryTableHolder && rhs);
|
|
|
|
|
|
|
|
~TemporaryTableHolder();
|
|
|
|
|
|
|
|
StorageID getGlobalTableID() const;
|
|
|
|
|
|
|
|
StoragePtr getTable() const;
|
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
operator bool () const { return id != UUIDHelpers::Nil; }
|
|
|
|
|
|
|
|
const Context * global_context = nullptr;
|
|
|
|
IDatabase * temporary_tables = nullptr;
|
|
|
|
UUID id = UUIDHelpers::Nil;
|
2020-03-10 19:36:17 +00:00
|
|
|
};
|
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
///TODO maybe remove shared_ptr from here?
|
2020-03-10 19:36:17 +00:00
|
|
|
using TemporaryTablesMapping = std::map<String, std::shared_ptr<TemporaryTableHolder>>;
|
2020-02-10 18:31:52 +00:00
|
|
|
|
2020-03-19 21:14:52 +00:00
|
|
|
class BackgroundSchedulePoolTaskHolder;
|
2020-03-13 15:41:36 +00:00
|
|
|
|
2020-02-03 12:54:36 +00:00
|
|
|
class DatabaseCatalog : boost::noncopyable
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
static constexpr const char * TEMPORARY_DATABASE = "_temporary_and_external_tables";
|
|
|
|
static constexpr const char * SYSTEM_DATABASE = "system";
|
|
|
|
|
2020-03-19 21:14:52 +00:00
|
|
|
static DatabaseCatalog & init(Context * global_context_);
|
2020-02-10 13:10:17 +00:00
|
|
|
static DatabaseCatalog & instance();
|
2020-02-03 12:54:36 +00:00
|
|
|
|
|
|
|
void loadDatabases();
|
|
|
|
void shutdown();
|
|
|
|
|
2020-02-10 18:31:52 +00:00
|
|
|
/// Get an object that protects the table from concurrently executing multiple DDL operations.
|
|
|
|
std::unique_ptr<DDLGuard> getDDLGuard(const String & database, const String & table);
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
|
2020-02-03 12:54:36 +00:00
|
|
|
void assertDatabaseExists(const String & database_name) const;
|
|
|
|
void assertDatabaseDoesntExist(const String & database_name) const;
|
|
|
|
|
|
|
|
DatabasePtr getDatabaseForTemporaryTables() const;
|
2020-02-10 13:10:17 +00:00
|
|
|
DatabasePtr getSystemDatabase() const;
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-02-17 13:52:59 +00:00
|
|
|
void attachDatabase(const String & database_name, const DatabasePtr & database);
|
2020-02-17 19:28:25 +00:00
|
|
|
DatabasePtr detachDatabase(const String & database_name, bool drop = false, bool check_empty = true);
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
/// database_name must be not empty
|
2020-02-17 13:52:59 +00:00
|
|
|
DatabasePtr getDatabase(const String & database_name) const;
|
|
|
|
DatabasePtr tryGetDatabase(const String & database_name) const;
|
|
|
|
bool isDatabaseExist(const String & database_name) const;
|
2020-02-03 12:54:36 +00:00
|
|
|
Databases getDatabases() const;
|
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
/// Same as getDatabase(const String & database_name), but if database_name is empty, current database of local_context is used
|
|
|
|
DatabasePtr getDatabase(const String & database_name, const Context & local_context) const;
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
/// For all of the following methods database_name in table_id must be not empty (even for temporary tables).
|
|
|
|
void assertTableDoesntExist(const StorageID & table_id) const;
|
|
|
|
bool isTableExist(const StorageID & table_id) const;
|
|
|
|
bool isDictionaryExist(const StorageID & table_id) const;
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-02-17 13:52:59 +00:00
|
|
|
StoragePtr getTable(const StorageID & table_id) const;
|
|
|
|
StoragePtr tryGetTable(const StorageID & table_id) const;
|
2020-03-04 20:29:52 +00:00
|
|
|
DatabaseAndTable getDatabaseAndTable(const StorageID & table_id) const { return getTableImpl(table_id, *global_context); }
|
|
|
|
DatabaseAndTable tryGetDatabaseAndTable(const StorageID & table_id) const;
|
|
|
|
DatabaseAndTable getTableImpl(const StorageID & table_id, const Context & local_context, std::optional<Exception> * exception = nullptr) const;
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-02-12 16:54:26 +00:00
|
|
|
void addDependency(const StorageID & from, const StorageID & where);
|
|
|
|
void removeDependency(const StorageID & from, const StorageID & where);
|
|
|
|
Dependencies getDependencies(const StorageID & from) const;
|
|
|
|
|
|
|
|
/// For Materialized and Live View
|
2020-03-11 19:10:55 +00:00
|
|
|
void updateDependency(const StorageID & old_from, const StorageID & old_where,const StorageID & new_from, const StorageID & new_where);
|
2020-02-12 16:54:26 +00:00
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
/// If table has UUID, addUUIDMapping(...) must be called when table attached to some database
|
|
|
|
/// and removeUUIDMapping(...) must be called when it detached.
|
|
|
|
/// Such tables can be accessed by persistent UUID instead of database and table name.
|
|
|
|
void addUUIDMapping(const UUID & uuid, DatabasePtr database, StoragePtr table);
|
|
|
|
void removeUUIDMapping(const UUID & uuid);
|
2020-03-23 00:12:13 +00:00
|
|
|
/// For moving table between databases
|
|
|
|
void updateUUIDMapping(const UUID & uuid, DatabasePtr database, StoragePtr table);
|
|
|
|
|
2020-04-06 23:22:44 +00:00
|
|
|
static String getPathForUUID(const UUID & uuid);
|
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
DatabaseAndTable tryGetByUUID(const UUID & uuid) const;
|
|
|
|
|
2020-03-19 21:14:52 +00:00
|
|
|
String getPathForDroppedMetadata(const StorageID & table_id) const;
|
|
|
|
void enqueueDroppedTableCleanup(StorageID table_id, StoragePtr table, String dropped_metadata_path, bool ignore_delay = false);
|
|
|
|
|
2020-02-03 12:54:36 +00:00
|
|
|
private:
|
2020-03-19 21:14:52 +00:00
|
|
|
DatabaseCatalog(Context * global_context_);
|
2020-02-03 12:54:36 +00:00
|
|
|
void assertDatabaseExistsUnlocked(const String & database_name) const;
|
|
|
|
void assertDatabaseDoesntExistUnlocked(const String & database_name) const;
|
|
|
|
|
2020-03-19 21:14:52 +00:00
|
|
|
|
2020-02-03 12:54:36 +00:00
|
|
|
struct UUIDToStorageMapPart
|
|
|
|
{
|
|
|
|
std::unordered_map<UUID, DatabaseAndTable> map;
|
|
|
|
mutable std::mutex mutex;
|
|
|
|
};
|
|
|
|
|
2020-03-12 18:04:29 +00:00
|
|
|
static constexpr UInt64 bits_for_first_level = 4;
|
2020-02-03 12:54:36 +00:00
|
|
|
using UUIDToStorageMap = std::array<UUIDToStorageMapPart, 1ull << bits_for_first_level>;
|
|
|
|
|
|
|
|
inline size_t getFirstLevelIdx(const UUID & uuid) const
|
|
|
|
{
|
|
|
|
return uuid.toUnderType().low >> (64 - bits_for_first_level);
|
|
|
|
}
|
|
|
|
|
2020-03-19 21:14:52 +00:00
|
|
|
struct TableMarkedAsDropped
|
|
|
|
{
|
|
|
|
StorageID table_id = StorageID::createEmpty();
|
|
|
|
StoragePtr table;
|
|
|
|
String metadata_path;
|
|
|
|
time_t drop_time;
|
|
|
|
};
|
|
|
|
using TablesMarkedAsDropped = std::list<TableMarkedAsDropped>;
|
|
|
|
|
|
|
|
void loadMarkedAsDroppedTables();
|
|
|
|
void dropTableDataTask();
|
|
|
|
void dropTableFinally(const TableMarkedAsDropped & table) const;
|
|
|
|
|
|
|
|
static constexpr time_t drop_delay_s = 10;
|
|
|
|
static constexpr size_t reschedule_time_ms = 100;
|
|
|
|
|
2020-02-03 12:54:36 +00:00
|
|
|
private:
|
2020-03-13 15:41:36 +00:00
|
|
|
/// For some reason Context is required to get Storage from Database object
|
2020-03-19 21:14:52 +00:00
|
|
|
Context * global_context;
|
2020-02-17 13:52:59 +00:00
|
|
|
mutable std::mutex databases_mutex;
|
2020-02-12 16:54:26 +00:00
|
|
|
|
2020-03-13 15:41:36 +00:00
|
|
|
ViewDependencies view_dependencies;
|
2020-02-12 16:54:26 +00:00
|
|
|
|
2020-02-03 12:54:36 +00:00
|
|
|
Databases databases;
|
|
|
|
UUIDToStorageMap uuid_map;
|
|
|
|
|
2020-02-12 16:54:26 +00:00
|
|
|
Poco::Logger * log;
|
2020-02-03 12:54:36 +00:00
|
|
|
|
2020-02-10 18:31:52 +00:00
|
|
|
/// Do not allow simultaneous execution of DDL requests on the same table.
|
|
|
|
/// database -> object -> (mutex, counter), counter: how many threads are running a query on the table at the same time
|
|
|
|
/// For the duration of the operation, an element is placed here, and an object is returned,
|
|
|
|
/// which deletes the element in the destructor when counter becomes zero.
|
|
|
|
/// In case the element already exists, waits, when query will be executed in other thread. See class DDLGuard below.
|
|
|
|
using DDLGuards = std::unordered_map<String, DDLGuard::Map>;
|
|
|
|
DDLGuards ddl_guards;
|
|
|
|
/// If you capture mutex and ddl_guards_mutex, then you need to grab them strictly in this order.
|
|
|
|
mutable std::mutex ddl_guards_mutex;
|
2020-03-19 21:14:52 +00:00
|
|
|
|
|
|
|
TablesMarkedAsDropped tables_marked_droped;
|
|
|
|
mutable std::mutex tables_marked_droped_mutex;
|
|
|
|
|
|
|
|
std::unique_ptr<BackgroundSchedulePoolTaskHolder> drop_task;
|
2020-02-03 12:54:36 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|