2015-04-16 06:12:35 +00:00
|
|
|
#include <map>
|
|
|
|
#include <set>
|
|
|
|
|
2017-06-02 18:48:33 +00:00
|
|
|
#include <boost/functional/hash/hash.hpp>
|
2015-04-16 06:12:35 +00:00
|
|
|
#include <Poco/Mutex.h>
|
2012-08-17 19:53:11 +00:00
|
|
|
#include <Poco/File.h>
|
2017-05-24 19:31:50 +00:00
|
|
|
#include <Poco/UUID.h>
|
2017-01-21 04:24:28 +00:00
|
|
|
#include <Poco/Net/IPAddress.h>
|
2012-08-17 19:53:11 +00:00
|
|
|
|
2015-09-29 19:19:54 +00:00
|
|
|
#include <common/logger_useful.h>
|
2017-09-09 23:17:38 +00:00
|
|
|
#include <pcg_random.hpp>
|
2015-04-16 06:12:35 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/Macros.h>
|
|
|
|
#include <Common/escapeForFileName.h>
|
2017-06-02 18:48:33 +00:00
|
|
|
#include <Common/setThreadName.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/Stopwatch.h>
|
|
|
|
#include <Common/formatReadable.h>
|
|
|
|
#include <DataStreams/FormatFactory.h>
|
2017-10-13 01:02:16 +00:00
|
|
|
#include <Databases/IDatabase.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/IStorage.h>
|
|
|
|
#include <Storages/MarkCache.h>
|
|
|
|
#include <Storages/MergeTree/BackgroundProcessingPool.h>
|
|
|
|
#include <Storages/MergeTree/ReshardingWorker.h>
|
|
|
|
#include <Storages/MergeTree/MergeList.h>
|
|
|
|
#include <Storages/MergeTree/MergeTreeSettings.h>
|
2017-10-13 01:02:16 +00:00
|
|
|
#include <Storages/CompressionSettingsSelector.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/Settings.h>
|
|
|
|
#include <Interpreters/Users.h>
|
|
|
|
#include <Interpreters/Quota.h>
|
|
|
|
#include <Interpreters/EmbeddedDictionaries.h>
|
|
|
|
#include <Interpreters/ExternalDictionaries.h>
|
2017-10-17 10:44:46 +00:00
|
|
|
#include <Interpreters/ExternalModels.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/ProcessList.h>
|
|
|
|
#include <Interpreters/Cluster.h>
|
|
|
|
#include <Interpreters/InterserverIOHandler.h>
|
|
|
|
#include <Interpreters/Compiler.h>
|
2017-06-05 13:59:38 +00:00
|
|
|
#include <Interpreters/SystemLog.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/QueryLog.h>
|
|
|
|
#include <Interpreters/PartLog.h>
|
|
|
|
#include <Interpreters/Context.h>
|
2017-08-07 17:01:04 +00:00
|
|
|
#include <Interpreters/DNSCache.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <IO/ReadBufferFromFile.h>
|
|
|
|
#include <IO/UncompressedCache.h>
|
|
|
|
#include <Parsers/ASTCreateQuery.h>
|
|
|
|
#include <Parsers/ParserCreateQuery.h>
|
|
|
|
#include <Parsers/parseQuery.h>
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/ConfigProcessor.h>
|
2017-06-19 20:06:35 +00:00
|
|
|
#include <Common/ZooKeeper/ZooKeeper.h>
|
2017-06-05 13:59:38 +00:00
|
|
|
#include <common/logger_useful.h>
|
2015-04-16 06:12:35 +00:00
|
|
|
|
2014-08-22 01:01:28 +00:00
|
|
|
|
2016-12-07 22:49:42 +00:00
|
|
|
namespace ProfileEvents
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
extern const Event ContextLock;
|
2016-12-07 22:49:42 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
namespace CurrentMetrics
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
extern const Metric ContextLockWait;
|
|
|
|
extern const Metric MemoryTrackingForMerges;
|
2016-12-07 22:49:42 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +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 DATABASE_ACCESS_DENIED;
|
|
|
|
extern const int UNKNOWN_DATABASE;
|
|
|
|
extern const int UNKNOWN_TABLE;
|
|
|
|
extern const int TABLE_ALREADY_EXISTS;
|
|
|
|
extern const int TABLE_WAS_NOT_DROPPED;
|
|
|
|
extern const int DATABASE_ALREADY_EXISTS;
|
|
|
|
extern const int TABLE_METADATA_DOESNT_EXIST;
|
|
|
|
extern const int THERE_IS_NO_SESSION;
|
|
|
|
extern const int NO_ELEMENTS_IN_CONFIG;
|
|
|
|
extern const int DDL_GUARD_IS_ACTIVE;
|
|
|
|
extern const int TABLE_SIZE_EXCEEDS_MAX_DROP_SIZE_LIMIT;
|
2017-06-02 18:48:33 +00:00
|
|
|
extern const int SESSION_NOT_FOUND;
|
|
|
|
extern const int SESSION_IS_LOCKED;
|
2016-01-11 21:46:36 +00:00
|
|
|
}
|
2015-04-16 06:12:35 +00:00
|
|
|
|
|
|
|
|
2016-11-23 20:41:39 +00:00
|
|
|
/** Set of known objects (environment), that could be used in query.
|
|
|
|
* Shared (global) part. Order of members (especially, order of destruction) is very important.
|
2015-04-16 06:12:35 +00:00
|
|
|
*/
|
|
|
|
struct ContextShared
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
Logger * log = &Logger::get("Context");
|
|
|
|
|
|
|
|
/// For access of most of shared objects. Recursive mutex.
|
|
|
|
mutable Poco::Mutex mutex;
|
|
|
|
/// Separate mutex for access of dictionaries. Separate mutex to avoid locks when server doing request to itself.
|
|
|
|
mutable std::mutex embedded_dictionaries_mutex;
|
|
|
|
mutable std::mutex external_dictionaries_mutex;
|
2017-10-17 10:44:46 +00:00
|
|
|
mutable std::mutex external_models_mutex;
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Separate mutex for re-initialization of zookeer session. This operation could take a long time and must not interfere with another operations.
|
|
|
|
mutable std::mutex zookeeper_mutex;
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
mutable zkutil::ZooKeeperPtr zookeeper; /// Client for ZooKeeper.
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
String interserver_io_host; /// The host name by which this server is available for other servers.
|
2017-09-07 21:04:48 +00:00
|
|
|
UInt16 interserver_io_port = 0; /// and port.
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
String path; /// Path to the data directory, with a slash at the end.
|
|
|
|
String tmp_path; /// The path to the temporary files that occur when processing the request.
|
2017-06-02 21:01:17 +00:00
|
|
|
String flags_path; /// Path to the directory with some control flags for server maintenance.
|
2017-08-24 14:51:13 +00:00
|
|
|
ConfigurationPtr config; /// Global configuration settings.
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
Databases databases; /// List of databases and tables in them.
|
|
|
|
FormatFactory format_factory; /// Formats.
|
2017-04-01 07:20:54 +00:00
|
|
|
mutable std::shared_ptr<EmbeddedDictionaries> embedded_dictionaries; /// Metrica's dictionaeis. Have lazy initialization.
|
|
|
|
mutable std::shared_ptr<ExternalDictionaries> external_dictionaries;
|
2017-10-17 10:44:46 +00:00
|
|
|
mutable std::shared_ptr<ExternalModels> external_models;
|
2017-04-01 07:20:54 +00:00
|
|
|
String default_profile_name; /// Default profile name used for default values.
|
|
|
|
Users users; /// Known users.
|
2017-04-02 17:37:49 +00:00
|
|
|
Quotas quotas; /// Known quotas for resource use.
|
|
|
|
mutable UncompressedCachePtr uncompressed_cache; /// The cache of decompressed blocks.
|
|
|
|
mutable MarkCachePtr mark_cache; /// Cache of marks in compressed files.
|
|
|
|
ProcessList process_list; /// Executing queries at the moment.
|
|
|
|
MergeList merge_list; /// The list of executable merge (for (Replicated)?MergeTree)
|
|
|
|
ViewDependencies view_dependencies; /// Current dependencies
|
|
|
|
ConfigurationPtr users_config; /// Config with the users, profiles and quotas sections.
|
|
|
|
InterserverIOHandler interserver_io_handler; /// Handler for interserver communication.
|
|
|
|
BackgroundProcessingPoolPtr background_pool; /// The thread pool for the background work performed by the tables.
|
2017-04-01 07:20:54 +00:00
|
|
|
ReshardingWorkerPtr resharding_worker;
|
2017-06-13 04:30:36 +00:00
|
|
|
Macros macros; /// Substitutions extracted from config.
|
2017-06-02 21:01:17 +00:00
|
|
|
std::unique_ptr<Compiler> compiler; /// Used for dynamic compilation of queries' parts if it necessary.
|
2017-06-13 04:30:36 +00:00
|
|
|
std::shared_ptr<DDLWorker> ddl_worker; /// Process ddl commands from zk.
|
2017-10-13 01:02:16 +00:00
|
|
|
/// Rules for selecting the compression settings, depending on the size of the part.
|
|
|
|
mutable std::unique_ptr<CompressionSettingsSelector> compression_settings_selector;
|
2017-06-02 21:01:17 +00:00
|
|
|
std::unique_ptr<MergeTreeSettings> merge_tree_settings; /// Settings of MergeTree* engines.
|
|
|
|
size_t max_table_size_to_drop = 50000000000lu; /// Protects MergeTree tables from accidental DROP (50GB by default)
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-06-07 12:54:35 +00:00
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
/// Named sessions. The user could specify session identifier to reuse settings and temporary tables in subsequent requests.
|
|
|
|
|
|
|
|
class SessionKeyHash
|
|
|
|
{
|
2017-06-02 18:48:33 +00:00
|
|
|
public:
|
|
|
|
size_t operator()(const Context::SessionKey & key) const
|
|
|
|
{
|
|
|
|
size_t seed = 0;
|
|
|
|
boost::hash_combine(seed, key.first);
|
|
|
|
boost::hash_combine(seed, key.second);
|
|
|
|
return seed;
|
|
|
|
}
|
|
|
|
};
|
|
|
|
|
|
|
|
using Sessions = std::unordered_map<Context::SessionKey, std::shared_ptr<Context>, SessionKeyHash>;
|
|
|
|
using CloseTimes = std::deque<std::vector<Context::SessionKey>>;
|
|
|
|
mutable Sessions sessions;
|
|
|
|
mutable CloseTimes close_times;
|
|
|
|
std::chrono::steady_clock::duration close_interval = std::chrono::seconds(1);
|
|
|
|
std::chrono::steady_clock::time_point close_cycle_time = std::chrono::steady_clock::now();
|
|
|
|
UInt64 close_cycle = 0;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Clusters for distributed tables
|
|
|
|
/// Initialized on demand (on distributed storages initialization) since Settings should be initialized
|
|
|
|
std::unique_ptr<Clusters> clusters;
|
|
|
|
ConfigurationPtr clusters_config; /// Soteres updated configs
|
|
|
|
mutable std::mutex clusters_mutex; /// Guards clusters and clusters_config
|
|
|
|
|
|
|
|
bool shutdown_called = false;
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Do not allow simultaneous execution of DDL requests on the same table.
|
2017-04-01 07:20:54 +00:00
|
|
|
/// database -> table -> exception_message
|
2017-04-02 17:37:49 +00:00
|
|
|
/// For the duration of the operation, an element is placed here, and an object is returned, which deletes the element in the destructor.
|
|
|
|
/// In case the element already exists, an exception is thrown. See class DDLGuard below.
|
2017-04-01 07:20:54 +00:00
|
|
|
using DDLGuards = std::unordered_map<String, DDLGuard::Map>;
|
|
|
|
DDLGuards ddl_guards;
|
2017-04-02 17:37:49 +00:00
|
|
|
/// If you capture mutex and ddl_guards_mutex, then you need to grab them strictly in this order.
|
2017-04-01 07:20:54 +00:00
|
|
|
mutable std::mutex ddl_guards_mutex;
|
|
|
|
|
|
|
|
Stopwatch uptime_watch;
|
|
|
|
|
|
|
|
Context::ApplicationType application_type = Context::ApplicationType::SERVER;
|
|
|
|
|
2017-09-09 23:17:38 +00:00
|
|
|
pcg64 rng{randomSeed()};
|
2017-05-24 19:31:50 +00:00
|
|
|
|
2017-06-19 20:35:53 +00:00
|
|
|
ContextShared()
|
|
|
|
{
|
2017-07-10 04:34:14 +00:00
|
|
|
/// TODO: make it singleton (?)
|
2017-06-19 20:35:53 +00:00
|
|
|
static std::atomic<size_t> num_calls{0};
|
|
|
|
if (++num_calls > 1)
|
|
|
|
{
|
|
|
|
std::cerr << "Attempting to create multiple ContextShared instances. Stack trace:\n" << StackTrace().toString();
|
|
|
|
std::cerr.flush();
|
|
|
|
std::terminate();
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
~ContextShared()
|
|
|
|
{
|
|
|
|
try
|
|
|
|
{
|
|
|
|
shutdown();
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
tryLogCurrentException(__PRETTY_FUNCTION__);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/** Perform a complex job of destroying objects in advance.
|
2017-04-01 07:20:54 +00:00
|
|
|
*/
|
|
|
|
void shutdown()
|
|
|
|
{
|
|
|
|
if (shutdown_called)
|
|
|
|
return;
|
|
|
|
shutdown_called = true;
|
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
/** At this point, some tables may have threads that block our mutex.
|
|
|
|
* To complete them correctly, we will copy the current list of tables,
|
|
|
|
* and ask them all to finish their work.
|
|
|
|
* Then delete all objects with tables.
|
2017-04-01 07:20:54 +00:00
|
|
|
*/
|
|
|
|
|
|
|
|
Databases current_databases;
|
|
|
|
|
|
|
|
{
|
|
|
|
Poco::ScopedLock<Poco::Mutex> lock(mutex);
|
|
|
|
current_databases = databases;
|
|
|
|
}
|
|
|
|
|
|
|
|
for (auto & database : current_databases)
|
|
|
|
database.second->shutdown();
|
|
|
|
|
|
|
|
{
|
|
|
|
Poco::ScopedLock<Poco::Mutex> lock(mutex);
|
|
|
|
databases.clear();
|
|
|
|
}
|
|
|
|
}
|
2015-04-16 06:12:35 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
|
2017-06-19 20:31:23 +00:00
|
|
|
Context::Context() = default;
|
|
|
|
|
|
|
|
|
|
|
|
Context Context::createGlobal()
|
2015-04-16 06:12:35 +00:00
|
|
|
{
|
2017-06-19 20:31:23 +00:00
|
|
|
Context res;
|
|
|
|
res.shared = std::make_shared<ContextShared>();
|
|
|
|
res.quota = std::make_shared<QuotaForIntervals>();
|
|
|
|
res.system_logs = std::make_shared<SystemLogs>();
|
|
|
|
return res;
|
2015-04-16 06:12:35 +00:00
|
|
|
}
|
|
|
|
|
2017-06-19 20:31:23 +00:00
|
|
|
|
2017-06-05 13:59:38 +00:00
|
|
|
Context::~Context()
|
|
|
|
{
|
|
|
|
try
|
|
|
|
{
|
|
|
|
/// Destroy system logs while at least one Context is alive
|
|
|
|
system_logs.reset();
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
tryLogCurrentException(__PRETTY_FUNCTION__);
|
|
|
|
}
|
|
|
|
}
|
2015-04-16 06:12:35 +00:00
|
|
|
|
|
|
|
|
2017-09-04 17:49:39 +00:00
|
|
|
InterserverIOHandler & Context::getInterserverIOHandler() { return shared->interserver_io_handler; }
|
2016-12-07 22:49:42 +00:00
|
|
|
|
|
|
|
std::unique_lock<Poco::Mutex> Context::getLock() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
ProfileEvents::increment(ProfileEvents::ContextLock);
|
|
|
|
CurrentMetrics::Increment increment{CurrentMetrics::ContextLockWait};
|
|
|
|
return std::unique_lock<Poco::Mutex>(shared->mutex);
|
2016-12-07 22:49:42 +00:00
|
|
|
}
|
|
|
|
|
2017-09-04 17:49:39 +00:00
|
|
|
ProcessList & Context::getProcessList() { return shared->process_list; }
|
|
|
|
const ProcessList & Context::getProcessList() const { return shared->process_list; }
|
2017-04-01 07:20:54 +00:00
|
|
|
MergeList & Context::getMergeList() { return shared->merge_list; }
|
|
|
|
const MergeList & Context::getMergeList() const { return shared->merge_list; }
|
2015-04-16 06:12:35 +00:00
|
|
|
|
|
|
|
|
2016-03-19 01:18:49 +00:00
|
|
|
const Databases Context::getDatabases() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->databases;
|
2016-03-19 01:18:49 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
Databases Context::getDatabases()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->databases;
|
2016-03-19 01:18:49 +00:00
|
|
|
}
|
|
|
|
|
2017-01-05 16:34:05 +00:00
|
|
|
|
2017-06-02 18:48:33 +00:00
|
|
|
Context::SessionKey Context::getSessionKey(const String & session_id) const
|
|
|
|
{
|
|
|
|
auto & user_name = client_info.current_user;
|
|
|
|
|
|
|
|
if (user_name.empty())
|
|
|
|
throw Exception("Empty user name.", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
return SessionKey(user_name, session_id);
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
void Context::scheduleCloseSession(const Context::SessionKey & key, std::chrono::steady_clock::duration timeout)
|
2017-06-02 18:48:33 +00:00
|
|
|
{
|
|
|
|
const UInt64 close_index = timeout / shared->close_interval + 1;
|
|
|
|
const auto new_close_cycle = shared->close_cycle + close_index;
|
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
if (session_close_cycle != new_close_cycle)
|
2017-06-02 18:48:33 +00:00
|
|
|
{
|
2017-06-02 21:01:17 +00:00
|
|
|
session_close_cycle = new_close_cycle;
|
2017-06-02 18:48:33 +00:00
|
|
|
if (shared->close_times.size() < close_index + 1)
|
|
|
|
shared->close_times.resize(close_index + 1);
|
|
|
|
shared->close_times[close_index].emplace_back(key);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
std::shared_ptr<Context> Context::acquireSession(const String & session_id, std::chrono::steady_clock::duration timeout, bool session_check) const
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
|
|
|
|
const auto & key = getSessionKey(session_id);
|
|
|
|
auto it = shared->sessions.find(key);
|
|
|
|
|
|
|
|
if (it == shared->sessions.end())
|
|
|
|
{
|
|
|
|
if (session_check)
|
|
|
|
throw Exception("Session not found.", ErrorCodes::SESSION_NOT_FOUND);
|
|
|
|
|
|
|
|
auto new_session = std::make_shared<Context>(*global_context);
|
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
new_session->scheduleCloseSession(key, timeout);
|
2017-06-02 18:48:33 +00:00
|
|
|
|
|
|
|
it = shared->sessions.insert(std::make_pair(key, std::move(new_session))).first;
|
|
|
|
}
|
|
|
|
else if (it->second->client_info.current_user != client_info.current_user)
|
|
|
|
{
|
|
|
|
throw Exception("Session belongs to a different user", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
}
|
|
|
|
|
|
|
|
const auto & session = it->second;
|
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
if (session->session_is_used)
|
2017-06-02 18:48:33 +00:00
|
|
|
throw Exception("Session is locked by a concurrent client.", ErrorCodes::SESSION_IS_LOCKED);
|
2017-06-02 21:01:17 +00:00
|
|
|
session->session_is_used = true;
|
2017-06-02 18:48:33 +00:00
|
|
|
|
|
|
|
session->client_info = client_info;
|
|
|
|
|
|
|
|
return session;
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void Context::releaseSession(const String & session_id, std::chrono::steady_clock::duration timeout)
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
session_is_used = false;
|
|
|
|
scheduleCloseSession(getSessionKey(session_id), timeout);
|
2017-06-02 18:48:33 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
std::chrono::steady_clock::duration Context::closeSessions() const
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
|
|
|
|
const auto now = std::chrono::steady_clock::now();
|
|
|
|
|
|
|
|
if (now < shared->close_cycle_time)
|
|
|
|
return shared->close_cycle_time - now;
|
|
|
|
|
|
|
|
const auto current_cycle = shared->close_cycle;
|
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
++shared->close_cycle;
|
2017-06-02 18:48:33 +00:00
|
|
|
shared->close_cycle_time = now + shared->close_interval;
|
|
|
|
|
|
|
|
if (shared->close_times.empty())
|
|
|
|
return shared->close_interval;
|
|
|
|
|
|
|
|
auto & sessions_to_close = shared->close_times.front();
|
|
|
|
|
|
|
|
for (const auto & key : sessions_to_close)
|
|
|
|
{
|
|
|
|
const auto session = shared->sessions.find(key);
|
|
|
|
|
2017-06-02 21:01:17 +00:00
|
|
|
if (session != shared->sessions.end() && session->second->session_close_cycle <= current_cycle)
|
2017-06-02 18:48:33 +00:00
|
|
|
{
|
2017-06-02 21:01:17 +00:00
|
|
|
if (session->second->session_is_used)
|
|
|
|
session->second->scheduleCloseSession(key, std::chrono::seconds(0));
|
2017-06-02 18:48:33 +00:00
|
|
|
else
|
|
|
|
shared->sessions.erase(session);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
shared->close_times.pop_front();
|
|
|
|
|
|
|
|
return shared->close_interval;
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-01-05 16:34:05 +00:00
|
|
|
static String resolveDatabase(const String & database_name, const String & current_database)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
String res = database_name.empty() ? current_database : database_name;
|
|
|
|
if (res.empty())
|
|
|
|
throw Exception("Default database is not selected", ErrorCodes::UNKNOWN_DATABASE);
|
|
|
|
return res;
|
2017-01-05 16:34:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-03-19 01:18:49 +00:00
|
|
|
const DatabasePtr Context::getDatabase(const String & database_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
assertDatabaseExists(db);
|
|
|
|
return shared->databases[db];
|
2016-03-19 01:18:49 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
DatabasePtr Context::getDatabase(const String & database_name)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
assertDatabaseExists(db);
|
|
|
|
return shared->databases[db];
|
2016-03-19 01:18:49 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
const DatabasePtr Context::tryGetDatabase(const String & database_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
auto it = shared->databases.find(db);
|
|
|
|
if (it == shared->databases.end())
|
|
|
|
return {};
|
|
|
|
return it->second;
|
2016-03-19 01:18:49 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
DatabasePtr Context::tryGetDatabase(const String & database_name)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
auto it = shared->databases.find(db);
|
|
|
|
if (it == shared->databases.end())
|
|
|
|
return {};
|
|
|
|
return it->second;
|
2016-03-19 01:18:49 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
String Context::getPath() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->path;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
2015-01-07 17:19:23 +00:00
|
|
|
String Context::getTemporaryPath() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->tmp_path;
|
2015-01-07 17:19:23 +00:00
|
|
|
}
|
|
|
|
|
2016-10-24 14:01:24 +00:00
|
|
|
String Context::getFlagsPath() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
if (!shared->flags_path.empty())
|
|
|
|
return shared->flags_path;
|
2016-12-07 19:25:16 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->flags_path = shared->path + "flags/";
|
|
|
|
Poco::File(shared->flags_path).createDirectories();
|
|
|
|
return shared->flags_path;
|
2016-10-24 14:01:24 +00:00
|
|
|
}
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
|
|
|
|
void Context::setPath(const String & path)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
shared->path = path;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
2015-01-07 17:19:23 +00:00
|
|
|
void Context::setTemporaryPath(const String & path)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
shared->tmp_path = path;
|
2015-01-07 17:19:23 +00:00
|
|
|
}
|
|
|
|
|
2016-10-24 14:01:24 +00:00
|
|
|
void Context::setFlagsPath(const String & path)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
shared->flags_path = path;
|
2016-10-24 14:01:24 +00:00
|
|
|
}
|
|
|
|
|
2017-08-24 14:51:13 +00:00
|
|
|
void Context::setConfig(const ConfigurationPtr & config)
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
shared->config = config;
|
|
|
|
}
|
|
|
|
|
|
|
|
ConfigurationPtr Context::getConfig() const
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
return shared->config;
|
|
|
|
}
|
|
|
|
|
|
|
|
Poco::Util::AbstractConfiguration & Context::getConfigRef() const
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
return shared->config ? *shared->config : Poco::Util::Application::instance().config();
|
|
|
|
}
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2016-10-14 15:06:46 +00:00
|
|
|
void Context::setUsersConfig(const ConfigurationPtr & config)
|
2013-08-10 07:46:45 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
shared->users_config = config;
|
|
|
|
shared->users.loadFromConfig(*shared->users_config);
|
|
|
|
shared->quotas.loadFromConfig(*shared->users_config);
|
2014-02-13 07:17:22 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
ConfigurationPtr Context::getUsersConfig()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->users_config;
|
2013-08-10 07:46:45 +00:00
|
|
|
}
|
|
|
|
|
2017-06-07 12:54:35 +00:00
|
|
|
void Context::calculateUserSettings()
|
2013-08-10 07:46:45 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2013-08-10 07:46:45 +00:00
|
|
|
|
2017-06-07 12:54:35 +00:00
|
|
|
String profile = shared->users.get(client_info.current_user).profile;
|
2017-03-23 14:14:56 +00:00
|
|
|
|
2017-06-07 12:54:35 +00:00
|
|
|
/// 1) Set default settings (hardcoded values)
|
|
|
|
/// NOTE: we ignore global_context settings (from which it is usually copied)
|
|
|
|
/// NOTE: global_context settings are immutable and not auto updated
|
2017-06-05 17:54:40 +00:00
|
|
|
settings = Settings();
|
|
|
|
|
|
|
|
/// 2) Apply settings from default profile
|
2017-04-01 07:20:54 +00:00
|
|
|
auto default_profile_name = getDefaultProfileName();
|
2017-06-07 12:54:35 +00:00
|
|
|
if (profile != default_profile_name)
|
2017-04-01 07:20:54 +00:00
|
|
|
settings.setProfile(default_profile_name, *shared->users_config);
|
2017-06-05 17:54:40 +00:00
|
|
|
|
|
|
|
/// 3) Apply settings from current user
|
2017-06-07 12:54:35 +00:00
|
|
|
settings.setProfile(profile, *shared->users_config);
|
|
|
|
}
|
2017-06-05 17:54:40 +00:00
|
|
|
|
2017-06-07 12:54:35 +00:00
|
|
|
|
|
|
|
void Context::setUser(const String & name, const String & password, const Poco::Net::SocketAddress & address, const String & quota_key)
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
|
|
|
|
const User & user_props = shared->users.get(name, password, address.host());
|
2013-08-10 07:46:45 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
client_info.current_user = name;
|
|
|
|
client_info.current_address = address;
|
2016-11-16 05:09:41 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!quota_key.empty())
|
|
|
|
client_info.quota_key = quota_key;
|
2017-06-07 12:54:35 +00:00
|
|
|
|
|
|
|
calculateUserSettings();
|
|
|
|
|
|
|
|
setQuota(user_props.quota, quota_key, name, address.host());
|
2013-08-10 07:46:45 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2013-11-03 00:24:46 +00:00
|
|
|
void Context::setQuota(const String & name, const String & quota_key, const String & user_name, const Poco::Net::IPAddress & address)
|
2013-08-12 00:36:18 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
quota = shared->quotas.get(name, quota_key, user_name, address);
|
2013-08-12 00:36:18 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
QuotaForIntervals & Context::getQuota()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return *quota;
|
2013-08-12 00:36:18 +00:00
|
|
|
}
|
|
|
|
|
2015-10-01 15:10:41 +00:00
|
|
|
void Context::checkDatabaseAccessRights(const std::string & database_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (client_info.current_user.empty() || (database_name == "system"))
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// An unnamed user, i.e. server, has access to all databases.
|
|
|
|
/// All users have access to the database system.
|
2017-04-01 07:20:54 +00:00
|
|
|
return;
|
|
|
|
}
|
|
|
|
if (!shared->users.isAllowedDatabase(client_info.current_user, database_name))
|
|
|
|
throw Exception("Access denied to database " + database_name, ErrorCodes::DATABASE_ACCESS_DENIED);
|
2015-10-01 15:10:41 +00:00
|
|
|
}
|
|
|
|
|
2013-11-13 14:39:48 +00:00
|
|
|
void Context::addDependency(const DatabaseAndTableName & from, const DatabaseAndTableName & where)
|
2013-11-08 17:43:03 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
checkDatabaseAccessRights(from.first);
|
|
|
|
checkDatabaseAccessRights(where.first);
|
|
|
|
shared->view_dependencies[from].insert(where);
|
2017-10-03 23:43:55 +00:00
|
|
|
|
|
|
|
// Notify table of dependencies change
|
|
|
|
auto table = tryGetTable(from.first, from.second);
|
|
|
|
if (table != nullptr)
|
|
|
|
table->updateDependencies();
|
2013-11-08 17:43:03 +00:00
|
|
|
}
|
|
|
|
|
2013-11-13 14:39:48 +00:00
|
|
|
void Context::removeDependency(const DatabaseAndTableName & from, const DatabaseAndTableName & where)
|
2013-11-08 17:43:03 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
checkDatabaseAccessRights(from.first);
|
|
|
|
checkDatabaseAccessRights(where.first);
|
|
|
|
shared->view_dependencies[from].erase(where);
|
2017-10-03 23:43:55 +00:00
|
|
|
|
|
|
|
// Notify table of dependencies change
|
|
|
|
auto table = tryGetTable(from.first, from.second);
|
|
|
|
if (table != nullptr)
|
|
|
|
table->updateDependencies();
|
2013-11-08 17:43:03 +00:00
|
|
|
}
|
|
|
|
|
2014-12-23 20:32:00 +00:00
|
|
|
Dependencies Context::getDependencies(const String & database_name, const String & table_name) const
|
2013-11-08 17:43:03 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2014-12-23 20:32:00 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
2017-10-13 17:19:48 +00:00
|
|
|
|
2017-10-13 18:46:25 +00:00
|
|
|
if (database_name.empty() && tryGetExternalTable(table_name))
|
2017-10-13 17:19:48 +00:00
|
|
|
{
|
|
|
|
/// Table is temporary. Access granted.
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
checkDatabaseAccessRights(db);
|
|
|
|
}
|
2014-12-23 20:32:00 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
ViewDependencies::const_iterator iter = shared->view_dependencies.find(DatabaseAndTableName(db, table_name));
|
|
|
|
if (iter == shared->view_dependencies.end())
|
|
|
|
return {};
|
2014-12-23 20:32:00 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return Dependencies(iter->second.begin(), iter->second.end());
|
2013-11-08 17:43:03 +00:00
|
|
|
}
|
2013-08-12 00:36:18 +00:00
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
bool Context::isTableExist(const String & database_name, const String & table_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
checkDatabaseAccessRights(db);
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Databases::const_iterator it = shared->databases.find(db);
|
|
|
|
return shared->databases.end() != it
|
2017-09-11 12:39:01 +00:00
|
|
|
&& it->second->isTableExist(*this, table_name);
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
bool Context::isDatabaseExist(const String & database_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
checkDatabaseAccessRights(db);
|
|
|
|
return shared->databases.end() != shared->databases.find(db);
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void Context::assertTableExists(const String & database_name, const String & table_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
checkDatabaseAccessRights(db);
|
2014-06-26 00:58:14 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Databases::const_iterator it = shared->databases.find(db);
|
|
|
|
if (shared->databases.end() == it)
|
|
|
|
throw Exception("Database " + db + " doesn't exist", ErrorCodes::UNKNOWN_DATABASE);
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-09-11 12:39:01 +00:00
|
|
|
if (!it->second->isTableExist(*this, table_name))
|
2017-04-01 07:20:54 +00:00
|
|
|
throw Exception("Table " + db + "." + table_name + " doesn't exist.", ErrorCodes::UNKNOWN_TABLE);
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2015-10-01 15:10:41 +00:00
|
|
|
void Context::assertTableDoesntExist(const String & database_name, const String & table_name, bool check_database_access_rights) const
|
2012-08-02 17:33:31 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
if (check_database_access_rights)
|
|
|
|
checkDatabaseAccessRights(db);
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Databases::const_iterator it = shared->databases.find(db);
|
2017-09-11 12:39:01 +00:00
|
|
|
if (shared->databases.end() != it && it->second->isTableExist(*this, table_name))
|
2017-04-01 07:20:54 +00:00
|
|
|
throw Exception("Table " + db + "." + table_name + " already exists.", ErrorCodes::TABLE_ALREADY_EXISTS);
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2015-10-01 15:10:41 +00:00
|
|
|
void Context::assertDatabaseExists(const String & database_name, bool check_database_access_rights) const
|
2012-08-02 17:33:31 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
if (check_database_access_rights)
|
|
|
|
checkDatabaseAccessRights(db);
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (shared->databases.end() == shared->databases.find(db))
|
|
|
|
throw Exception("Database " + db + " doesn't exist", ErrorCodes::UNKNOWN_DATABASE);
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void Context::assertDatabaseDoesntExist(const String & database_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
checkDatabaseAccessRights(db);
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (shared->databases.end() != shared->databases.find(db))
|
|
|
|
throw Exception("Database " + db + " already exists.", ErrorCodes::DATABASE_ALREADY_EXISTS);
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-03-13 15:00:06 +00:00
|
|
|
Tables Context::getExternalTables() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2014-03-14 15:42:30 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Tables res = external_tables;
|
|
|
|
if (session_context && session_context != this)
|
|
|
|
{
|
|
|
|
Tables buf = session_context->getExternalTables();
|
|
|
|
res.insert(buf.begin(), buf.end());
|
|
|
|
}
|
|
|
|
else if (global_context && global_context != this)
|
|
|
|
{
|
|
|
|
Tables buf = global_context->getExternalTables();
|
|
|
|
res.insert(buf.begin(), buf.end());
|
|
|
|
}
|
|
|
|
return res;
|
2014-03-13 15:00:06 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
StoragePtr Context::tryGetExternalTable(const String & table_name) const
|
2014-03-04 15:31:56 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2014-03-04 15:31:56 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Tables::const_iterator jt = external_tables.find(table_name);
|
|
|
|
if (external_tables.end() == jt)
|
|
|
|
return StoragePtr();
|
2014-03-04 15:31:56 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return jt->second;
|
2014-03-04 15:31:56 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
StoragePtr Context::getTable(const String & database_name, const String & table_name) const
|
2015-08-19 21:15:27 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
Exception exc;
|
|
|
|
auto res = getTableImpl(database_name, table_name, &exc);
|
|
|
|
if (!res)
|
|
|
|
throw exc;
|
|
|
|
return res;
|
2015-08-19 21:15:27 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
StoragePtr Context::tryGetTable(const String & database_name, const String & table_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return getTableImpl(database_name, table_name, nullptr);
|
2015-08-19 21:15:27 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
StoragePtr Context::getTableImpl(const String & database_name, const String & table_name, Exception * exception) const
|
2012-08-02 17:33:31 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (database_name.empty())
|
|
|
|
{
|
|
|
|
StoragePtr res = tryGetExternalTable(table_name);
|
|
|
|
if (res)
|
|
|
|
return res;
|
|
|
|
}
|
2015-08-19 21:15:27 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
checkDatabaseAccessRights(db);
|
2014-06-26 00:58:14 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Databases::const_iterator it = shared->databases.find(db);
|
|
|
|
if (shared->databases.end() == it)
|
|
|
|
{
|
|
|
|
if (exception)
|
|
|
|
*exception = Exception("Database " + db + " doesn't exist", ErrorCodes::UNKNOWN_DATABASE);
|
|
|
|
return {};
|
|
|
|
}
|
2014-06-26 00:58:14 +00:00
|
|
|
|
2017-09-11 12:39:01 +00:00
|
|
|
auto table = it->second->tryGetTable(*this, table_name);
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!table)
|
|
|
|
{
|
|
|
|
if (exception)
|
|
|
|
*exception = Exception("Table " + db + "." + table_name + " doesn't exist.", ErrorCodes::UNKNOWN_TABLE);
|
|
|
|
return {};
|
|
|
|
}
|
2015-08-19 21:15:27 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return table;
|
2013-05-06 10:31:35 +00:00
|
|
|
}
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-09-08 03:47:27 +00:00
|
|
|
void Context::addExternalTable(const String & table_name, const StoragePtr & storage)
|
2014-03-04 15:31:56 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (external_tables.end() != external_tables.find(table_name))
|
|
|
|
throw Exception("Temporary table " + table_name + " already exists.", ErrorCodes::TABLE_ALREADY_EXISTS);
|
2015-08-19 21:15:27 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
external_tables[table_name] = storage;
|
2015-08-19 21:15:27 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (process_list_elem)
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
shared->process_list.addTemporaryTable(*process_list_elem, table_name, storage);
|
|
|
|
}
|
2014-03-04 15:31:56 +00:00
|
|
|
}
|
|
|
|
|
2017-08-15 17:00:18 +00:00
|
|
|
StoragePtr Context::tryRemoveExternalTable(const String & table_name)
|
2017-08-15 12:34:28 +00:00
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
|
2017-08-15 17:00:18 +00:00
|
|
|
Tables::const_iterator it = external_tables.find(table_name);
|
|
|
|
if (external_tables.end() == it)
|
|
|
|
return StoragePtr();
|
2017-08-15 12:34:28 +00:00
|
|
|
|
2017-08-15 17:00:18 +00:00
|
|
|
auto storage = it->second;
|
|
|
|
external_tables.erase(it);
|
|
|
|
return storage;
|
2017-08-15 12:34:28 +00:00
|
|
|
|
|
|
|
return {};
|
|
|
|
}
|
2014-03-04 15:31:56 +00:00
|
|
|
|
2016-03-21 12:57:12 +00:00
|
|
|
DDLGuard::DDLGuard(Map & map_, std::mutex & mutex_, std::unique_lock<std::mutex> && lock, const String & elem, const String & message)
|
2017-04-01 07:20:54 +00:00
|
|
|
: map(map_), mutex(mutex_)
|
2016-03-21 12:57:12 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
bool inserted;
|
|
|
|
std::tie(it, inserted) = map.emplace(elem, message);
|
|
|
|
if (!inserted)
|
|
|
|
throw Exception(it->second, ErrorCodes::DDL_GUARD_IS_ACTIVE);
|
2016-03-21 12:57:12 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
DDLGuard::~DDLGuard()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
std::lock_guard<std::mutex> lock(mutex);
|
|
|
|
map.erase(it);
|
2016-03-21 12:57:12 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
std::unique_ptr<DDLGuard> Context::getDDLGuard(const String & database, const String & table, const String & message) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
std::unique_lock<std::mutex> lock(shared->ddl_guards_mutex);
|
|
|
|
return std::make_unique<DDLGuard>(shared->ddl_guards[database], shared->ddl_guards_mutex, std::move(lock), table, message);
|
2016-03-21 12:57:12 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-04-01 17:41:13 +00:00
|
|
|
std::unique_ptr<DDLGuard> Context::getDDLGuardIfTableDoesntExist(const String & database, const String & table, const String & message) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2016-04-01 17:41:13 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Databases::const_iterator it = shared->databases.find(database);
|
2017-09-11 12:39:01 +00:00
|
|
|
if (shared->databases.end() != it && it->second->isTableExist(*this, table))
|
2017-04-01 07:20:54 +00:00
|
|
|
return {};
|
2016-04-01 17:41:13 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return getDDLGuard(database, table, message);
|
2016-04-01 17:41:13 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-03-19 01:18:49 +00:00
|
|
|
void Context::addDatabase(const String & database_name, const DatabasePtr & database)
|
2012-08-02 17:33:31 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
assertDatabaseDoesntExist(database_name);
|
|
|
|
shared->databases[database_name] = database;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-03-21 12:57:12 +00:00
|
|
|
DatabasePtr Context::detachDatabase(const String & database_name)
|
2012-08-02 17:33:31 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
auto res = getDatabase(database_name);
|
|
|
|
shared->databases.erase(database_name);
|
|
|
|
return res;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-03-19 01:18:49 +00:00
|
|
|
ASTPtr Context::getCreateQuery(const String & database_name, const String & table_name) const
|
2012-08-02 17:33:31 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2012-08-02 17:33:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String db = resolveDatabase(database_name, current_database);
|
|
|
|
assertDatabaseExists(db);
|
2012-08-17 19:53:11 +00:00
|
|
|
|
2017-09-11 12:39:01 +00:00
|
|
|
return shared->databases[db]->getCreateQuery(*this, table_name);
|
2012-08-17 19:53:11 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
Settings Context::getSettings() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return settings;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2013-05-05 20:07:11 +00:00
|
|
|
Limits Context::getLimits() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return settings.limits;
|
2013-05-05 20:07:11 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
void Context::setSettings(const Settings & settings_)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
settings = settings_;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 19:03:32 +00:00
|
|
|
void Context::setSetting(const String & name, const Field & value)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
if (name == "profile")
|
|
|
|
settings.setProfile(value.safeGet<String>(), *shared->users_config);
|
|
|
|
else
|
|
|
|
settings.set(name, value);
|
2014-02-13 07:17:22 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void Context::setSetting(const String & name, const std::string & value)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
if (name == "profile")
|
|
|
|
settings.setProfile(value, *shared->users_config);
|
|
|
|
else
|
|
|
|
settings.set(name, value);
|
2012-08-02 19:03:32 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
String Context::getCurrentDatabase() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return current_database;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-02-12 17:31:02 +00:00
|
|
|
String Context::getCurrentQueryId() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return client_info.current_query_id;
|
2014-02-12 17:31:02 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
void Context::setCurrentDatabase(const String & name)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
assertDatabaseExists(name);
|
|
|
|
current_database = name;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-02-12 17:31:02 +00:00
|
|
|
void Context::setCurrentQueryId(const String & query_id)
|
|
|
|
{
|
2017-05-24 19:31:50 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!client_info.current_query_id.empty())
|
|
|
|
throw Exception("Logical error: attempt to set query_id twice", ErrorCodes::LOGICAL_ERROR);
|
2015-08-19 21:15:27 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String query_id_to_set = query_id;
|
2017-05-24 19:31:50 +00:00
|
|
|
|
2017-04-02 17:37:49 +00:00
|
|
|
if (query_id_to_set.empty()) /// If the user did not submit his query_id, then we generate it ourselves.
|
2017-05-24 19:31:50 +00:00
|
|
|
{
|
|
|
|
/// Generate random UUID, but using lower quality RNG,
|
|
|
|
/// because Poco::UUIDGenerator::generateRandom method is using /dev/random, that is very expensive.
|
|
|
|
/// NOTE: Actually we don't need to use UUIDs for query identifiers.
|
|
|
|
/// We could use any suitable string instead.
|
|
|
|
|
|
|
|
union
|
|
|
|
{
|
|
|
|
char bytes[16];
|
|
|
|
struct
|
|
|
|
{
|
|
|
|
UInt64 a;
|
|
|
|
UInt64 b;
|
|
|
|
};
|
|
|
|
} random;
|
|
|
|
|
|
|
|
random.a = shared->rng();
|
|
|
|
random.b = shared->rng();
|
|
|
|
|
|
|
|
/// Use protected constructor.
|
|
|
|
struct UUID : Poco::UUID
|
|
|
|
{
|
|
|
|
UUID(const char * bytes, Poco::UUID::Version version)
|
|
|
|
: Poco::UUID(bytes, version) {}
|
|
|
|
};
|
|
|
|
|
|
|
|
query_id_to_set = UUID(random.bytes, Poco::UUID::UUID_RANDOM).toString();
|
|
|
|
}
|
2015-07-25 09:49:09 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
client_info.current_query_id = query_id_to_set;
|
2014-02-12 17:31:02 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2013-06-29 18:03:57 +00:00
|
|
|
String Context::getDefaultFormat() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return default_format.empty() ? "TabSeparated" : default_format;
|
2013-06-29 18:03:57 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void Context::setDefaultFormat(const String & name)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
default_format = name;
|
2013-06-29 18:03:57 +00:00
|
|
|
}
|
|
|
|
|
2014-08-11 15:59:01 +00:00
|
|
|
const Macros& Context::getMacros() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return shared->macros;
|
2014-08-11 15:59:01 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void Context::setMacros(Macros && macros)
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// We assume that this assignment occurs once when the server starts. If this is not the case, you need to use a mutex.
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->macros = macros;
|
2014-08-11 15:59:01 +00:00
|
|
|
}
|
|
|
|
|
2016-11-16 11:29:51 +00:00
|
|
|
const Context & Context::getSessionContext() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!session_context)
|
|
|
|
throw Exception("There is no session", ErrorCodes::THERE_IS_NO_SESSION);
|
|
|
|
return *session_context;
|
2016-11-16 11:29:51 +00:00
|
|
|
}
|
2013-06-29 18:03:57 +00:00
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
Context & Context::getSessionContext()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!session_context)
|
|
|
|
throw Exception("There is no session", ErrorCodes::THERE_IS_NO_SESSION);
|
|
|
|
return *session_context;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
2016-11-15 10:57:11 +00:00
|
|
|
const Context & Context::getGlobalContext() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!global_context)
|
|
|
|
throw Exception("Logical error: there is no global context", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
return *global_context;
|
2016-11-15 10:57:11 +00:00
|
|
|
}
|
2012-08-02 17:33:31 +00:00
|
|
|
|
|
|
|
Context & Context::getGlobalContext()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!global_context)
|
|
|
|
throw Exception("Logical error: there is no global context", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
return *global_context;
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|
|
|
|
|
2012-12-19 20:15:15 +00:00
|
|
|
|
2017-01-21 04:24:28 +00:00
|
|
|
const EmbeddedDictionaries & Context::getEmbeddedDictionaries() const
|
2012-12-19 20:15:15 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return getEmbeddedDictionariesImpl(false);
|
2015-02-10 17:40:40 +00:00
|
|
|
}
|
|
|
|
|
2017-08-24 18:19:06 +00:00
|
|
|
EmbeddedDictionaries & Context::getEmbeddedDictionaries()
|
|
|
|
{
|
|
|
|
return getEmbeddedDictionariesImpl(false);
|
|
|
|
}
|
|
|
|
|
2015-02-10 17:40:40 +00:00
|
|
|
|
|
|
|
const ExternalDictionaries & Context::getExternalDictionaries() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return getExternalDictionariesImpl(false);
|
2015-03-27 13:11:22 +00:00
|
|
|
}
|
|
|
|
|
2017-08-24 18:19:06 +00:00
|
|
|
ExternalDictionaries & Context::getExternalDictionaries()
|
|
|
|
{
|
|
|
|
return getExternalDictionariesImpl(false);
|
|
|
|
}
|
|
|
|
|
2015-04-02 16:30:18 +00:00
|
|
|
|
2017-10-17 10:44:46 +00:00
|
|
|
const ExternalModels & Context::getExternalModels() const
|
|
|
|
{
|
|
|
|
return getExternalModelsImpl(false);
|
|
|
|
}
|
|
|
|
|
|
|
|
ExternalModels & Context::getExternalModels()
|
|
|
|
{
|
|
|
|
return getExternalModelsImpl(false);
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-08-24 18:19:06 +00:00
|
|
|
EmbeddedDictionaries & Context::getEmbeddedDictionariesImpl(const bool throw_on_error) const
|
2015-03-27 13:11:22 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
std::lock_guard<std::mutex> lock(shared->embedded_dictionaries_mutex);
|
2015-04-02 16:30:18 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!shared->embedded_dictionaries)
|
2017-08-24 14:51:13 +00:00
|
|
|
shared->embedded_dictionaries = std::make_shared<EmbeddedDictionaries>(*this->global_context, throw_on_error);
|
2015-04-02 16:30:18 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return *shared->embedded_dictionaries;
|
2015-03-27 13:11:22 +00:00
|
|
|
}
|
|
|
|
|
2015-04-02 16:30:18 +00:00
|
|
|
|
2017-08-24 18:19:06 +00:00
|
|
|
ExternalDictionaries & Context::getExternalDictionariesImpl(const bool throw_on_error) const
|
2015-03-27 13:11:22 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
std::lock_guard<std::mutex> lock(shared->external_dictionaries_mutex);
|
2015-04-02 16:30:18 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!shared->external_dictionaries)
|
|
|
|
{
|
|
|
|
if (!this->global_context)
|
|
|
|
throw Exception("Logical error: there is no global context", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
shared->external_dictionaries = std::make_shared<ExternalDictionaries>(*this->global_context, throw_on_error);
|
|
|
|
}
|
2015-04-02 16:30:18 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return *shared->external_dictionaries;
|
2015-04-02 16:30:18 +00:00
|
|
|
}
|
|
|
|
|
2017-10-17 10:44:46 +00:00
|
|
|
ExternalModels & Context::getExternalModelsImpl(bool throw_on_error) const
|
|
|
|
{
|
|
|
|
std::lock_guard<std::mutex> lock(shared->external_models_mutex);
|
|
|
|
|
|
|
|
if (!shared->external_models)
|
|
|
|
{
|
|
|
|
if (!this->global_context)
|
|
|
|
throw Exception("Logical error: there is no global context", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
shared->external_models = std::make_shared<ExternalModels>(*this->global_context, throw_on_error);
|
|
|
|
}
|
|
|
|
|
|
|
|
return *shared->external_models;
|
|
|
|
}
|
2015-04-02 16:30:18 +00:00
|
|
|
|
2017-01-21 04:24:28 +00:00
|
|
|
void Context::tryCreateEmbeddedDictionaries() const
|
2015-04-02 16:30:18 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
static_cast<void>(getEmbeddedDictionariesImpl(true));
|
2015-04-02 16:30:18 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void Context::tryCreateExternalDictionaries() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
static_cast<void>(getExternalDictionariesImpl(true));
|
2012-12-19 20:15:15 +00:00
|
|
|
}
|
|
|
|
|
2013-02-16 14:55:14 +00:00
|
|
|
|
2017-10-17 10:44:46 +00:00
|
|
|
void Context::tryCreateExternalModels() const
|
|
|
|
{
|
|
|
|
static_cast<void>(getExternalModelsImpl(true));
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2013-02-16 14:55:14 +00:00
|
|
|
void Context::setProgressCallback(ProgressCallback callback)
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Callback is set to a session or to a query. In the session, only one query is processed at a time. Therefore, the lock is not needed.
|
2017-04-01 07:20:54 +00:00
|
|
|
progress_callback = callback;
|
2013-02-16 14:55:14 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
ProgressCallback Context::getProgressCallback() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return progress_callback;
|
2013-02-16 14:55:14 +00:00
|
|
|
}
|
|
|
|
|
2013-09-03 20:21:28 +00:00
|
|
|
|
2013-11-03 05:32:42 +00:00
|
|
|
void Context::setProcessListElement(ProcessList::Element * elem)
|
|
|
|
{
|
2017-04-02 17:37:49 +00:00
|
|
|
/// Set to a session or query. In the session, only one query is processed at a time. Therefore, the lock is not needed.
|
2017-04-01 07:20:54 +00:00
|
|
|
process_list_elem = elem;
|
2013-11-03 05:32:42 +00:00
|
|
|
}
|
|
|
|
|
2017-09-04 17:49:39 +00:00
|
|
|
ProcessList::Element * Context::getProcessListElement() const
|
2017-08-29 13:23:04 +00:00
|
|
|
{
|
|
|
|
return process_list_elem;
|
|
|
|
}
|
|
|
|
|
2013-11-03 05:32:42 +00:00
|
|
|
|
2014-03-28 14:36:24 +00:00
|
|
|
void Context::setUncompressedCache(size_t max_size_in_bytes)
|
2013-09-08 05:53:10 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2013-09-08 05:53:10 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (shared->uncompressed_cache)
|
|
|
|
throw Exception("Uncompressed cache has been already created.", ErrorCodes::LOGICAL_ERROR);
|
2013-09-08 05:53:10 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->uncompressed_cache = std::make_shared<UncompressedCache>(max_size_in_bytes);
|
2013-09-08 05:53:10 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
UncompressedCachePtr Context::getUncompressedCache() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->uncompressed_cache;
|
2013-09-08 05:53:10 +00:00
|
|
|
}
|
|
|
|
|
2017-08-07 17:01:04 +00:00
|
|
|
|
|
|
|
void Context::dropUncompressedCache() const
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
if (shared->uncompressed_cache)
|
|
|
|
shared->uncompressed_cache->reset();
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-02-11 13:30:42 +00:00
|
|
|
void Context::setMarkCache(size_t cache_size_in_bytes)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2014-02-11 13:30:42 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (shared->mark_cache)
|
2017-08-30 21:25:44 +00:00
|
|
|
throw Exception("Mark cache has been already created.", ErrorCodes::LOGICAL_ERROR);
|
2014-02-11 13:30:42 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->mark_cache = std::make_shared<MarkCache>(cache_size_in_bytes, std::chrono::seconds(settings.mark_cache_min_lifetime));
|
2014-02-11 13:30:42 +00:00
|
|
|
}
|
|
|
|
|
2017-08-07 17:01:04 +00:00
|
|
|
|
2014-02-11 13:30:42 +00:00
|
|
|
MarkCachePtr Context::getMarkCache() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->mark_cache;
|
2014-02-11 13:30:42 +00:00
|
|
|
}
|
|
|
|
|
2017-08-07 17:01:04 +00:00
|
|
|
|
|
|
|
void Context::dropMarkCache() const
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
if (shared->mark_cache)
|
|
|
|
shared->mark_cache->reset();
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
|
|
void Context::dropCaches() const
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
|
|
|
|
if (shared->uncompressed_cache)
|
|
|
|
shared->uncompressed_cache->reset();
|
|
|
|
|
|
|
|
if (shared->mark_cache)
|
|
|
|
shared->mark_cache->reset();
|
|
|
|
}
|
|
|
|
|
2014-07-02 12:30:38 +00:00
|
|
|
BackgroundProcessingPool & Context::getBackgroundPool()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
if (!shared->background_pool)
|
|
|
|
shared->background_pool = std::make_shared<BackgroundProcessingPool>(settings.background_pool_size);
|
|
|
|
return *shared->background_pool;
|
2014-07-02 12:30:38 +00:00
|
|
|
}
|
|
|
|
|
2016-01-28 01:00:42 +00:00
|
|
|
void Context::setReshardingWorker(std::shared_ptr<ReshardingWorker> resharding_worker)
|
2016-01-28 01:00:27 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
if (shared->resharding_worker)
|
|
|
|
throw Exception("Resharding background thread has already been initialized.", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
shared->resharding_worker = resharding_worker;
|
2016-01-28 01:00:42 +00:00
|
|
|
}
|
2016-01-28 01:00:27 +00:00
|
|
|
|
2017-09-04 17:49:39 +00:00
|
|
|
ReshardingWorker & Context::getReshardingWorker() const
|
2016-01-28 01:00:42 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
if (!shared->resharding_worker)
|
|
|
|
throw Exception("Resharding background thread not initialized: resharding missing in configuration file.",
|
|
|
|
ErrorCodes::LOGICAL_ERROR);
|
|
|
|
return *shared->resharding_worker;
|
2016-01-28 01:00:27 +00:00
|
|
|
}
|
|
|
|
|
2017-04-13 13:42:29 +00:00
|
|
|
void Context::setDDLWorker(std::shared_ptr<DDLWorker> ddl_worker)
|
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
if (shared->ddl_worker)
|
|
|
|
throw Exception("DDL background thread has already been initialized.", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
shared->ddl_worker = ddl_worker;
|
|
|
|
}
|
|
|
|
|
2017-09-04 17:49:39 +00:00
|
|
|
DDLWorker & Context::getDDLWorker() const
|
2017-04-13 13:42:29 +00:00
|
|
|
{
|
|
|
|
auto lock = getLock();
|
|
|
|
if (!shared->ddl_worker)
|
2017-09-04 17:49:39 +00:00
|
|
|
throw Exception("DDL background thread is not initialized.", ErrorCodes::LOGICAL_ERROR);
|
2017-04-13 13:42:29 +00:00
|
|
|
return *shared->ddl_worker;
|
|
|
|
}
|
|
|
|
|
2014-05-13 10:10:26 +00:00
|
|
|
void Context::setZooKeeper(zkutil::ZooKeeperPtr zookeeper)
|
2014-03-21 19:17:59 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
std::lock_guard<std::mutex> lock(shared->zookeeper_mutex);
|
2014-03-21 19:17:59 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (shared->zookeeper)
|
|
|
|
throw Exception("ZooKeeper client has already been set.", ErrorCodes::LOGICAL_ERROR);
|
2014-03-21 19:17:59 +00:00
|
|
|
|
2017-09-08 03:47:27 +00:00
|
|
|
shared->zookeeper = std::move(zookeeper);
|
2014-03-21 19:17:59 +00:00
|
|
|
}
|
|
|
|
|
2014-05-13 10:10:26 +00:00
|
|
|
zkutil::ZooKeeperPtr Context::getZooKeeper() const
|
2014-03-21 19:17:59 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
std::lock_guard<std::mutex> lock(shared->zookeeper_mutex);
|
2014-04-25 13:55:15 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (shared->zookeeper && shared->zookeeper->expired())
|
|
|
|
shared->zookeeper = shared->zookeeper->startNewSession();
|
2014-04-25 13:55:15 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return shared->zookeeper;
|
2014-03-21 19:17:59 +00:00
|
|
|
}
|
|
|
|
|
2017-06-15 20:08:26 +00:00
|
|
|
bool Context::hasZooKeeper() const
|
|
|
|
{
|
|
|
|
std::lock_guard<std::mutex> lock(shared->zookeeper_mutex);
|
|
|
|
return shared->zookeeper != nullptr;
|
|
|
|
}
|
|
|
|
|
2014-03-21 19:17:59 +00:00
|
|
|
|
2014-11-19 20:40:51 +00:00
|
|
|
void Context::setInterserverIOAddress(const String & host, UInt16 port)
|
2014-03-21 19:49:27 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->interserver_io_host = host;
|
|
|
|
shared->interserver_io_port = port;
|
2014-03-21 19:49:27 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-11-19 20:40:51 +00:00
|
|
|
std::pair<String, UInt16> Context::getInterserverIOAddress() const
|
2014-03-21 19:49:27 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (shared->interserver_io_host.empty() || shared->interserver_io_port == 0)
|
|
|
|
throw Exception("Parameter 'interserver_http_port' required for replication is not specified in configuration file.",
|
|
|
|
ErrorCodes::NO_ELEMENTS_IN_CONFIG);
|
2014-11-19 20:40:51 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return { shared->interserver_io_host, shared->interserver_io_port };
|
2014-03-21 19:49:27 +00:00
|
|
|
}
|
|
|
|
|
2015-09-24 04:50:53 +00:00
|
|
|
UInt16 Context::getTCPPort() const
|
|
|
|
{
|
2017-08-24 14:51:13 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
|
|
|
|
auto & config = getConfigRef();
|
|
|
|
return config.getInt("tcp_port");
|
2015-09-24 04:50:53 +00:00
|
|
|
}
|
|
|
|
|
2014-03-21 19:49:27 +00:00
|
|
|
|
2016-10-14 15:06:46 +00:00
|
|
|
std::shared_ptr<Cluster> Context::getCluster(const std::string & cluster_name) const
|
2013-12-07 16:51:29 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto res = getClusters().getCluster(cluster_name);
|
2013-12-10 17:06:57 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!res)
|
|
|
|
throw Exception("Requested cluster '" + cluster_name + "' not found", ErrorCodes::BAD_GET);
|
2016-10-10 08:44:52 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return res;
|
2013-12-07 16:51:29 +00:00
|
|
|
}
|
2015-01-10 02:30:03 +00:00
|
|
|
|
2016-10-10 08:44:52 +00:00
|
|
|
|
2016-10-14 15:06:46 +00:00
|
|
|
std::shared_ptr<Cluster> Context::tryGetCluster(const std::string & cluster_name) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return getClusters().getCluster(cluster_name);
|
2013-12-07 16:51:29 +00:00
|
|
|
}
|
2015-01-10 02:30:03 +00:00
|
|
|
|
2016-10-14 15:06:46 +00:00
|
|
|
|
2016-10-10 08:44:52 +00:00
|
|
|
Clusters & Context::getClusters() const
|
2015-04-30 12:43:16 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
std::lock_guard<std::mutex> lock(shared->clusters_mutex);
|
|
|
|
if (!shared->clusters)
|
|
|
|
{
|
2017-08-24 14:51:13 +00:00
|
|
|
auto & config = shared->clusters_config ? *shared->clusters_config : getConfigRef();
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->clusters = std::make_unique<Clusters>(config, settings);
|
|
|
|
}
|
|
|
|
}
|
2016-03-10 03:08:09 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return *shared->clusters;
|
2015-04-30 12:43:16 +00:00
|
|
|
}
|
2015-01-10 02:30:03 +00:00
|
|
|
|
2016-10-10 08:44:52 +00:00
|
|
|
|
2016-10-14 15:06:46 +00:00
|
|
|
/// On repeating calls updates existing clusters and adds new clusters, doesn't delete old clusters
|
|
|
|
void Context::setClustersConfig(const ConfigurationPtr & config)
|
2016-10-10 08:44:52 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
std::lock_guard<std::mutex> lock(shared->clusters_mutex);
|
2016-10-10 08:44:52 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->clusters_config = config;
|
|
|
|
if (shared->clusters)
|
|
|
|
shared->clusters->updateClusters(*shared->clusters_config, settings);
|
2015-04-30 12:43:16 +00:00
|
|
|
}
|
2015-01-10 02:30:03 +00:00
|
|
|
|
2016-10-10 08:44:52 +00:00
|
|
|
|
2015-01-10 02:30:03 +00:00
|
|
|
Compiler & Context::getCompiler()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2015-01-11 00:35:30 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!shared->compiler)
|
|
|
|
shared->compiler = std::make_unique<Compiler>(shared->path + "build/", 1);
|
2015-01-11 00:35:30 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return *shared->compiler;
|
2015-01-10 02:30:03 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2015-06-26 20:48:10 +00:00
|
|
|
QueryLog & Context::getQueryLog()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2015-06-26 20:48:10 +00:00
|
|
|
|
2017-06-05 13:59:38 +00:00
|
|
|
if (!system_logs)
|
|
|
|
throw Exception("Query log have been already shutdown", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
|
|
|
|
if (!system_logs->query_log)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
if (shared->shutdown_called)
|
2017-06-05 13:59:38 +00:00
|
|
|
throw Exception("Logical error: query log should be destroyed before tables shutdown", ErrorCodes::LOGICAL_ERROR);
|
2016-03-22 19:35:23 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!global_context)
|
|
|
|
throw Exception("Logical error: no global context for query log", ErrorCodes::LOGICAL_ERROR);
|
2016-03-22 19:35:23 +00:00
|
|
|
|
2017-08-24 14:51:13 +00:00
|
|
|
auto & config = getConfigRef();
|
2015-06-26 20:48:10 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String database = config.getString("query_log.database", "system");
|
|
|
|
String table = config.getString("query_log.table", "query_log");
|
2017-04-12 16:37:19 +00:00
|
|
|
size_t flush_interval_milliseconds = config.getUInt64(
|
|
|
|
"query_log.flush_interval_milliseconds", DEFAULT_QUERY_LOG_FLUSH_INTERVAL_MILLISECONDS);
|
2015-06-26 20:48:10 +00:00
|
|
|
|
2017-06-05 13:59:38 +00:00
|
|
|
system_logs->query_log = std::make_unique<QueryLog>(
|
2017-10-26 12:44:47 +00:00
|
|
|
*global_context, database, table, "ENGINE = MergeTree(event_date, event_time, 1024)", flush_interval_milliseconds);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2015-06-26 20:48:10 +00:00
|
|
|
|
2017-06-05 13:59:38 +00:00
|
|
|
return *system_logs->query_log;
|
2015-06-26 20:48:10 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-06-05 13:59:38 +00:00
|
|
|
PartLog * Context::getPartLog(const String & database, const String & table)
|
2017-03-07 17:13:54 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2017-03-07 17:13:54 +00:00
|
|
|
|
2017-08-24 14:51:13 +00:00
|
|
|
auto & config = getConfigRef();
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!config.has("part_log"))
|
|
|
|
return nullptr;
|
2017-03-07 17:13:54 +00:00
|
|
|
|
2017-06-05 13:59:38 +00:00
|
|
|
/// System logs are shutdown
|
|
|
|
if (!system_logs)
|
|
|
|
return nullptr;
|
|
|
|
|
|
|
|
String part_log_database = config.getString("part_log.database", "system");
|
|
|
|
String part_log_table = config.getString("part_log.table", "part_log");
|
|
|
|
|
|
|
|
/// Will not log system.part_log itself.
|
|
|
|
/// It doesn't make sense and not allow to destruct PartLog correctly due to infinite logging and flushing
|
|
|
|
if (database == part_log_database && table == part_log_table)
|
|
|
|
return nullptr;
|
|
|
|
|
|
|
|
if (!system_logs->part_log)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
if (shared->shutdown_called)
|
2017-06-05 13:59:38 +00:00
|
|
|
throw Exception("Logical error: part log should be destroyed before tables shutdown", ErrorCodes::LOGICAL_ERROR);
|
2017-03-07 17:13:54 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!global_context)
|
|
|
|
throw Exception("Logical error: no global context for part log", ErrorCodes::LOGICAL_ERROR);
|
2017-03-07 17:13:54 +00:00
|
|
|
|
2017-04-12 16:37:19 +00:00
|
|
|
size_t flush_interval_milliseconds = config.getUInt64(
|
|
|
|
"part_log.flush_interval_milliseconds", DEFAULT_QUERY_LOG_FLUSH_INTERVAL_MILLISECONDS);
|
2017-10-26 12:44:47 +00:00
|
|
|
system_logs->part_log = std::make_unique<PartLog>(
|
|
|
|
*global_context, part_log_database, part_log_table,
|
|
|
|
"ENGINE = MergeTree(event_date, event_time, 1024)", flush_interval_milliseconds);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2017-03-07 17:13:54 +00:00
|
|
|
|
2017-06-05 13:59:38 +00:00
|
|
|
return system_logs->part_log.get();
|
2017-03-07 17:13:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-10-13 01:02:16 +00:00
|
|
|
CompressionSettings Context::chooseCompressionSettings(size_t part_size, double part_size_ratio) const
|
2015-04-17 05:35:53 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2015-04-17 05:35:53 +00:00
|
|
|
|
2017-10-13 01:02:16 +00:00
|
|
|
if (!shared->compression_settings_selector)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
constexpr auto config_name = "compression";
|
2017-08-24 14:51:13 +00:00
|
|
|
auto & config = getConfigRef();
|
2015-04-17 05:35:53 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (config.has(config_name))
|
2017-10-13 01:02:16 +00:00
|
|
|
shared->compression_settings_selector = std::make_unique<CompressionSettingsSelector>(config, "compression");
|
2017-04-01 07:20:54 +00:00
|
|
|
else
|
2017-10-13 01:02:16 +00:00
|
|
|
shared->compression_settings_selector = std::make_unique<CompressionSettingsSelector>();
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
2015-04-17 05:35:53 +00:00
|
|
|
|
2017-10-13 01:02:16 +00:00
|
|
|
return shared->compression_settings_selector->choose(part_size, part_size_ratio);
|
2015-04-17 05:35:53 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2015-07-16 21:32:51 +00:00
|
|
|
const MergeTreeSettings & Context::getMergeTreeSettings()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
2015-07-16 21:32:51 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!shared->merge_tree_settings)
|
|
|
|
{
|
2017-08-24 14:51:13 +00:00
|
|
|
auto & config = getConfigRef();
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->merge_tree_settings = std::make_unique<MergeTreeSettings>();
|
|
|
|
shared->merge_tree_settings->loadFromConfig("merge_tree", config);
|
|
|
|
}
|
2015-07-16 21:32:51 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
return *shared->merge_tree_settings;
|
2015-07-16 21:32:51 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-01-19 19:11:12 +00:00
|
|
|
void Context::setMaxTableSizeToDrop(size_t max_size)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
// Is initialized at server startup
|
|
|
|
shared->max_table_size_to_drop = max_size;
|
2017-01-19 19:11:12 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void Context::checkTableCanBeDropped(const String & database, const String & table, size_t table_size)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t max_table_size_to_drop = shared->max_table_size_to_drop;
|
2017-01-19 19:11:12 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (!max_table_size_to_drop || table_size <= max_table_size_to_drop)
|
|
|
|
return;
|
2017-01-19 19:11:12 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Poco::File force_file(getFlagsPath() + "force_drop_table");
|
|
|
|
bool force_file_exists = force_file.exists();
|
2017-01-19 19:11:12 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
if (force_file_exists)
|
|
|
|
{
|
|
|
|
try
|
|
|
|
{
|
|
|
|
force_file.remove();
|
|
|
|
return;
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
/// User should recreate force file on each drop, it shouldn't be protected
|
|
|
|
tryLogCurrentException("Drop table check", "Can't remove force file to enable table drop");
|
|
|
|
}
|
|
|
|
}
|
2017-01-19 19:11:12 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String table_size_str = formatReadableSizeWithDecimalSuffix(table_size);
|
|
|
|
String max_table_size_to_drop_str = formatReadableSizeWithDecimalSuffix(max_table_size_to_drop);
|
|
|
|
std::stringstream ostr;
|
2017-01-19 19:11:12 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
ostr << "Table " << backQuoteIfNeed(database) << "." << backQuoteIfNeed(table) << " was not dropped.\n"
|
|
|
|
<< "Reason:\n"
|
|
|
|
<< "1. Table size (" << table_size_str << ") is greater than max_table_size_to_drop (" << max_table_size_to_drop_str << ")\n"
|
2017-08-16 10:24:06 +00:00
|
|
|
<< "2. File '" << force_file.path() << "' intended to force DROP "
|
2017-04-01 07:20:54 +00:00
|
|
|
<< (force_file_exists ? "exists but not writeable (could not be removed)" : "doesn't exist") << "\n";
|
2017-01-23 19:18:25 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
ostr << "How to fix this:\n"
|
|
|
|
<< "1. Either increase (or set to zero) max_table_size_to_drop in server config and restart ClickHouse\n"
|
|
|
|
<< "2. Either create forcing file " << force_file.path() << " and make sure that ClickHouse has write permission for it.\n"
|
|
|
|
<< "Example:\nsudo touch '" << force_file.path() << "' && sudo chmod 666 '" << force_file.path() << "'";
|
2017-01-23 19:18:25 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
throw Exception(ostr.str(), ErrorCodes::TABLE_SIZE_EXCEEDS_MAX_DROP_SIZE_LIMIT);
|
2017-01-19 19:11:12 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-02-13 06:37:19 +00:00
|
|
|
BlockInputStreamPtr Context::getInputFormat(const String & name, ReadBuffer & buf, const Block & sample, size_t max_block_size) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return shared->format_factory.getInput(name, buf, sample, *this, max_block_size);
|
2016-02-13 06:37:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
BlockOutputStreamPtr Context::getOutputFormat(const String & name, WriteBuffer & buf, const Block & sample) const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return shared->format_factory.getOutput(name, buf, sample, *this);
|
2016-02-13 06:37:19 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-04-09 07:47:08 +00:00
|
|
|
time_t Context::getUptimeSeconds() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
auto lock = getLock();
|
|
|
|
return shared->uptime_watch.elapsedSeconds();
|
2016-04-09 07:47:08 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2015-04-16 06:12:35 +00:00
|
|
|
void Context::shutdown()
|
|
|
|
{
|
2017-06-05 13:59:38 +00:00
|
|
|
system_logs.reset();
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->shutdown();
|
2015-04-16 06:12:35 +00:00
|
|
|
}
|
|
|
|
|
2016-11-11 17:01:02 +00:00
|
|
|
|
|
|
|
Context::ApplicationType Context::getApplicationType() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return shared->application_type;
|
2016-11-11 17:01:02 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void Context::setApplicationType(ApplicationType type)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Lock isn't required, you should set it at start
|
|
|
|
shared->application_type = type;
|
2016-11-11 17:01:02 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-03-23 14:14:56 +00:00
|
|
|
String Context::getDefaultProfileName() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return shared->default_profile_name;
|
2017-03-23 14:14:56 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void Context::setDefaultProfileName(const String & name)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
shared->default_profile_name = name;
|
2017-03-23 14:14:56 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2017-06-02 18:48:33 +00:00
|
|
|
SessionCleaner::~SessionCleaner()
|
|
|
|
{
|
|
|
|
try
|
|
|
|
{
|
|
|
|
{
|
|
|
|
std::lock_guard<std::mutex> lock{mutex};
|
|
|
|
quit = true;
|
|
|
|
}
|
|
|
|
|
|
|
|
cond.notify_one();
|
|
|
|
|
|
|
|
thread.join();
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
DB::tryLogCurrentException(__PRETTY_FUNCTION__);
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
void SessionCleaner::run()
|
|
|
|
{
|
|
|
|
setThreadName("HTTPSessionCleaner");
|
|
|
|
|
|
|
|
std::unique_lock<std::mutex> lock{mutex};
|
|
|
|
|
|
|
|
while (true)
|
|
|
|
{
|
|
|
|
auto interval = context.closeSessions();
|
|
|
|
|
|
|
|
if (cond.wait_for(lock, interval, [this]() -> bool { return quit; }))
|
|
|
|
break;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2012-08-02 17:33:31 +00:00
|
|
|
}
|