2017-03-03 00:24:56 +00:00
|
|
|
#include "Server.h"
|
2014-04-15 16:39:56 +00:00
|
|
|
|
2017-03-03 18:59:42 +00:00
|
|
|
#include <memory>
|
2017-03-03 00:24:56 +00:00
|
|
|
#include <sys/resource.h>
|
2018-05-09 04:22:30 +00:00
|
|
|
#include <errno.h>
|
2018-07-10 17:40:17 +00:00
|
|
|
#include <Poco/Version.h>
|
2017-03-03 18:59:42 +00:00
|
|
|
#include <Poco/DirectoryIterator.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include <Poco/Net/HTTPServer.h>
|
2016-06-25 03:06:36 +00:00
|
|
|
#include <Poco/Net/NetException.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include <ext/scope_guard.h>
|
2018-02-08 19:12:37 +00:00
|
|
|
#include <common/logger_useful.h>
|
2015-09-29 19:19:54 +00:00
|
|
|
#include <common/ErrorHandlers.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include <common/getMemoryAmount.h>
|
|
|
|
#include <Common/ClickHouseRevision.h>
|
2018-04-19 13:56:14 +00:00
|
|
|
#include <Common/DNSResolver.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include <Common/CurrentMetrics.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/Macros.h>
|
2018-01-15 19:07:47 +00:00
|
|
|
#include <Common/StringUtils/StringUtils.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include <Common/ZooKeeper/ZooKeeper.h>
|
|
|
|
#include <Common/ZooKeeper/ZooKeeperNodeCache.h>
|
|
|
|
#include <Common/config.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/getFQDNOrHostName.h>
|
|
|
|
#include <Common/getMultipleKeysFromConfig.h>
|
2017-06-22 18:08:14 +00:00
|
|
|
#include <Common/getNumberOfPhysicalCPUCores.h>
|
2018-06-14 14:29:42 +00:00
|
|
|
#include <Common/TaskStatsInfoGetter.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <IO/HTTPCommon.h>
|
|
|
|
#include <Interpreters/AsynchronousMetrics.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include <Interpreters/DDLWorker.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Interpreters/ProcessList.h>
|
|
|
|
#include <Interpreters/loadMetadata.h>
|
2018-05-14 18:36:01 +00:00
|
|
|
#include <Interpreters/DNSCacheUpdater.h>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Storages/StorageReplicatedMergeTree.h>
|
|
|
|
#include <Storages/System/attachSystemTables.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include <AggregateFunctions/registerAggregateFunctions.h>
|
|
|
|
#include <Functions/registerFunctions.h>
|
|
|
|
#include <TableFunctions/registerTableFunctions.h>
|
2017-12-30 00:36:06 +00:00
|
|
|
#include <Storages/registerStorages.h>
|
2018-02-28 20:34:25 +00:00
|
|
|
#include <Common/Config/ConfigReloader.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include "HTTPHandlerFactory.h"
|
2016-01-17 13:34:36 +00:00
|
|
|
#include "MetricsTransmitter.h"
|
2018-06-05 20:09:51 +00:00
|
|
|
#include <Common/StatusFile.h>
|
2017-08-09 14:33:07 +00:00
|
|
|
#include "TCPHandlerFactory.h"
|
2012-03-09 03:06:09 +00:00
|
|
|
|
2018-05-14 18:36:01 +00:00
|
|
|
#if USE_POCO_NETSSL
|
2017-03-28 20:30:57 +00:00
|
|
|
#include <Poco/Net/Context.h>
|
|
|
|
#include <Poco/Net/SecureServerSocket.h>
|
|
|
|
#endif
|
2017-03-21 19:08:09 +00:00
|
|
|
|
2017-08-01 14:34:06 +00:00
|
|
|
namespace CurrentMetrics
|
|
|
|
{
|
|
|
|
extern const Metric Revision;
|
|
|
|
}
|
|
|
|
|
2012-03-09 03:06:09 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
2017-08-01 14:34:06 +00:00
|
|
|
|
2017-08-10 23:25:51 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
|
|
|
extern const int NO_ELEMENTS_IN_CONFIG;
|
|
|
|
extern const int SUPPORT_IS_DISABLED;
|
2017-12-20 20:25:22 +00:00
|
|
|
extern const int ARGUMENT_OUT_OF_BOUND;
|
2018-07-30 18:32:21 +00:00
|
|
|
extern const int EXCESSIVE_ELEMENT_IN_CONFIG;
|
2017-08-10 23:25:51 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-12-13 18:51:19 +00:00
|
|
|
static std::string getCanonicalPath(std::string && path)
|
2012-03-09 03:06:09 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
Poco::trimInPlace(path);
|
|
|
|
if (path.empty())
|
|
|
|
throw Exception("path configuration parameter is empty");
|
|
|
|
if (path.back() != '/')
|
|
|
|
path += '/';
|
2018-05-06 06:29:57 +00:00
|
|
|
return std::move(path);
|
2016-12-13 18:51:19 +00:00
|
|
|
}
|
|
|
|
|
2018-02-08 19:12:37 +00:00
|
|
|
void Server::uninitialize()
|
|
|
|
{
|
|
|
|
logger().information("shutting down");
|
|
|
|
BaseDaemon::uninitialize();
|
|
|
|
}
|
|
|
|
|
|
|
|
void Server::initialize(Poco::Util::Application & self)
|
|
|
|
{
|
|
|
|
BaseDaemon::initialize(self);
|
|
|
|
logger().information("starting up");
|
|
|
|
}
|
|
|
|
|
2017-01-09 13:42:29 +00:00
|
|
|
std::string Server::getDefaultCorePath() const
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
return getCanonicalPath(config().getString("path")) + "cores";
|
2017-01-09 13:42:29 +00:00
|
|
|
}
|
2016-12-13 18:51:19 +00:00
|
|
|
|
2017-12-02 02:47:12 +00:00
|
|
|
int Server::main(const std::vector<std::string> & /*args*/)
|
2016-12-13 18:51:19 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
Logger * log = &logger();
|
|
|
|
|
2017-04-21 17:47:27 +00:00
|
|
|
registerFunctions();
|
2017-05-05 20:39:25 +00:00
|
|
|
registerAggregateFunctions();
|
2017-06-10 09:04:31 +00:00
|
|
|
registerTableFunctions();
|
2017-12-30 00:36:06 +00:00
|
|
|
registerStorages();
|
2017-04-21 17:47:27 +00:00
|
|
|
|
2017-08-01 14:34:06 +00:00
|
|
|
CurrentMetrics::set(CurrentMetrics::Revision, ClickHouseRevision::get());
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/** Context contains all that query execution is dependent:
|
|
|
|
* settings, available functions, data types, aggregate functions, databases...
|
|
|
|
*/
|
2017-06-19 20:31:23 +00:00
|
|
|
global_context = std::make_unique<Context>(Context::createGlobal());
|
2017-04-01 07:20:54 +00:00
|
|
|
global_context->setGlobalContext(*global_context);
|
|
|
|
global_context->setApplicationType(Context::ApplicationType::SERVER);
|
|
|
|
|
2018-04-03 19:43:33 +00:00
|
|
|
bool has_zookeeper = config().has("zookeeper");
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
zkutil::ZooKeeperNodeCache main_config_zk_node_cache([&] { return global_context->getZooKeeper(); });
|
|
|
|
if (loaded_config.has_zk_includes)
|
|
|
|
{
|
|
|
|
auto old_configuration = loaded_config.configuration;
|
2017-11-21 16:54:25 +00:00
|
|
|
ConfigProcessor config_processor(config_path);
|
|
|
|
loaded_config = config_processor.loadConfigWithZooKeeperIncludes(
|
|
|
|
main_config_zk_node_cache, /* fallback_to_preprocessed = */ true);
|
|
|
|
config_processor.savePreprocessedConfig(loaded_config);
|
2017-04-01 07:20:54 +00:00
|
|
|
config().removeConfiguration(old_configuration.get());
|
|
|
|
config().add(loaded_config.configuration.duplicate(), PRIO_DEFAULT, false);
|
|
|
|
}
|
|
|
|
|
|
|
|
std::string path = getCanonicalPath(config().getString("path"));
|
|
|
|
std::string default_database = config().getString("default_database", "default");
|
|
|
|
|
|
|
|
global_context->setPath(path);
|
|
|
|
|
|
|
|
/// Create directories for 'path' and for default database, if not exist.
|
|
|
|
Poco::File(path + "data/" + default_database).createDirectories();
|
|
|
|
Poco::File(path + "metadata/" + default_database).createDirectories();
|
|
|
|
|
|
|
|
StatusFile status{path + "status"};
|
|
|
|
|
2017-08-30 14:47:35 +00:00
|
|
|
SCOPE_EXIT({
|
|
|
|
/** Explicitly destroy Context. It is more convenient than in destructor of Server, because logger is still available.
|
|
|
|
* At this moment, no one could own shared part of Context.
|
|
|
|
*/
|
|
|
|
global_context.reset();
|
|
|
|
|
|
|
|
LOG_DEBUG(log, "Destroyed global context.");
|
|
|
|
});
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Try to increase limit on number of open files.
|
|
|
|
{
|
|
|
|
rlimit rlim;
|
|
|
|
if (getrlimit(RLIMIT_NOFILE, &rlim))
|
|
|
|
throw Poco::Exception("Cannot getrlimit");
|
|
|
|
|
|
|
|
if (rlim.rlim_cur == rlim.rlim_max)
|
|
|
|
{
|
|
|
|
LOG_DEBUG(log, "rlimit on number of file descriptors is " << rlim.rlim_cur);
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
rlim_t old = rlim.rlim_cur;
|
|
|
|
rlim.rlim_cur = config().getUInt("max_open_files", rlim.rlim_max);
|
|
|
|
int rc = setrlimit(RLIMIT_NOFILE, &rlim);
|
|
|
|
if (rc != 0)
|
|
|
|
LOG_WARNING(log,
|
2017-12-20 20:25:22 +00:00
|
|
|
"Cannot set max number of file descriptors to " << rlim.rlim_cur
|
|
|
|
<< ". Try to specify max_open_files according to your system limits. error: "
|
|
|
|
<< strerror(errno));
|
2017-04-01 07:20:54 +00:00
|
|
|
else
|
|
|
|
LOG_DEBUG(log, "Set max number of file descriptors to " << rlim.rlim_cur << " (was " << old << ").");
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
static ServerErrorHandler error_handler;
|
|
|
|
Poco::ErrorHandler::set(&error_handler);
|
|
|
|
|
|
|
|
/// Initialize DateLUT early, to not interfere with running time of first query.
|
|
|
|
LOG_DEBUG(log, "Initializing DateLUT.");
|
|
|
|
DateLUT::instance();
|
|
|
|
LOG_TRACE(log, "Initialized DateLUT with time zone `" << DateLUT::instance().getTimeZone() << "'.");
|
|
|
|
|
2018-04-19 05:32:09 +00:00
|
|
|
/// Directory with temporary data for processing of heavy queries.
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
std::string tmp_path = config().getString("tmp_path", path + "tmp/");
|
|
|
|
global_context->setTemporaryPath(tmp_path);
|
|
|
|
Poco::File(tmp_path).createDirectories();
|
|
|
|
|
|
|
|
/// Clearing old temporary files.
|
|
|
|
Poco::DirectoryIterator dir_end;
|
|
|
|
for (Poco::DirectoryIterator it(tmp_path); it != dir_end; ++it)
|
|
|
|
{
|
|
|
|
if (it->isFile() && startsWith(it.name(), "tmp"))
|
|
|
|
{
|
|
|
|
LOG_DEBUG(log, "Removing old temporary file " << it->path());
|
|
|
|
it->remove();
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
/** Directory with 'flags': files indicating temporary settings for the server set by system administrator.
|
|
|
|
* Flags may be cleared automatically after being applied by the server.
|
|
|
|
* Examples: do repair of local data; clone all replicated tables from replica.
|
|
|
|
*/
|
2018-04-19 05:32:09 +00:00
|
|
|
{
|
|
|
|
Poco::File(path + "flags/").createDirectories();
|
|
|
|
global_context->setFlagsPath(path + "flags/");
|
|
|
|
}
|
|
|
|
|
|
|
|
/** Directory with user provided files that are usable by 'file' table function.
|
|
|
|
*/
|
|
|
|
{
|
|
|
|
|
|
|
|
std::string user_files_path = config().getString("user_files_path", path + "user_files/");
|
|
|
|
global_context->setUserFilesPath(user_files_path);
|
|
|
|
Poco::File(user_files_path).createDirectories();
|
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-30 18:32:21 +00:00
|
|
|
if (config().has("interserver_http_port") && config().has("interserver_https_port"))
|
|
|
|
throw Exception("Both http and https interserver ports are specified", ErrorCodes::EXCESSIVE_ELEMENT_IN_CONFIG);
|
|
|
|
|
|
|
|
static const auto interserver_tags =
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-07-30 18:32:21 +00:00
|
|
|
std::make_tuple("interserver_http_host", "interserver_http_port", "http"),
|
|
|
|
std::make_tuple("interserver_https_host", "interserver_https_port", "https")
|
|
|
|
};
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-30 18:32:21 +00:00
|
|
|
for (auto [host_tag, port_tag, scheme] : interserver_tags)
|
|
|
|
{
|
|
|
|
if (config().has(port_tag))
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-07-30 18:32:21 +00:00
|
|
|
String this_host = config().getString(host_tag, "");
|
|
|
|
|
|
|
|
if (this_host.empty())
|
|
|
|
{
|
|
|
|
this_host = getFQDNOrHostName();
|
|
|
|
LOG_DEBUG(log,
|
|
|
|
"Configuration parameter '" + String(host_tag) + "' doesn't exist or exists and empty. Will use '" + this_host
|
|
|
|
+ "' as replica host.");
|
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-30 18:32:21 +00:00
|
|
|
String port_str = config().getString(port_tag);
|
|
|
|
int port = parse<int>(port_str);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-30 18:32:21 +00:00
|
|
|
if (port < 0 || port > 0xFFFF)
|
|
|
|
throw Exception("Out of range '" + String(port_tag) + "': " + toString(port), ErrorCodes::ARGUMENT_OUT_OF_BOUND);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-30 18:32:21 +00:00
|
|
|
global_context->setInterserverIOAddress(this_host, port);
|
|
|
|
global_context->setInterserverScheme(scheme);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-07-26 16:01:43 +00:00
|
|
|
if (config().has("interserver_http_credentials"))
|
2018-07-26 15:10:57 +00:00
|
|
|
{
|
|
|
|
String user = config().getString("interserver_http_credentials.user", "");
|
|
|
|
String password = config().getString("interserver_http_credentials.password", "");
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-26 15:10:57 +00:00
|
|
|
if (user.empty())
|
2018-07-26 16:10:21 +00:00
|
|
|
throw Exception("Configuration parameter interserver_http_credentials user can't be empty", ErrorCodes::NO_ELEMENTS_IN_CONFIG);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-07-30 18:32:21 +00:00
|
|
|
global_context->setInterserverCredentials(user, password);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if (config().has("macros"))
|
2018-03-13 23:44:23 +00:00
|
|
|
global_context->setMacros(std::make_unique<Macros>(config(), "macros"));
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Initialize main config reloader.
|
|
|
|
std::string include_from_path = config().getString("include_from", "/etc/metrika.xml");
|
|
|
|
auto main_config_reloader = std::make_unique<ConfigReloader>(config_path,
|
|
|
|
include_from_path,
|
|
|
|
std::move(main_config_zk_node_cache),
|
2018-02-28 20:34:25 +00:00
|
|
|
[&](ConfigurationPtr config)
|
|
|
|
{
|
2018-03-28 12:20:45 +00:00
|
|
|
buildLoggers(*config);
|
2018-02-28 20:34:25 +00:00
|
|
|
global_context->setClustersConfig(config);
|
2018-03-13 23:44:23 +00:00
|
|
|
global_context->setMacros(std::make_unique<Macros>(*config, "macros"));
|
2018-02-28 20:34:25 +00:00
|
|
|
},
|
2017-04-01 07:20:54 +00:00
|
|
|
/* already_loaded = */ true);
|
|
|
|
|
|
|
|
/// Initialize users config reloader.
|
|
|
|
std::string users_config_path = config().getString("users_config", config_path);
|
|
|
|
/// If path to users' config isn't absolute, try guess its root (current) dir.
|
|
|
|
/// At first, try to find it in dir of main config, after will use current dir.
|
|
|
|
if (users_config_path.empty() || users_config_path[0] != '/')
|
|
|
|
{
|
|
|
|
std::string config_dir = Poco::Path(config_path).parent().toString();
|
|
|
|
if (Poco::File(config_dir + users_config_path).exists())
|
|
|
|
users_config_path = config_dir + users_config_path;
|
|
|
|
}
|
|
|
|
auto users_config_reloader = std::make_unique<ConfigReloader>(users_config_path,
|
|
|
|
include_from_path,
|
|
|
|
zkutil::ZooKeeperNodeCache([&] { return global_context->getZooKeeper(); }),
|
|
|
|
[&](ConfigurationPtr config) { global_context->setUsersConfig(config); },
|
|
|
|
/* already_loaded = */ false);
|
|
|
|
|
2018-03-13 10:41:47 +00:00
|
|
|
/// Reload config in SYSTEM RELOAD CONFIG query.
|
2018-05-07 02:01:11 +00:00
|
|
|
global_context->setConfigReloadCallback([&]()
|
|
|
|
{
|
2018-03-13 10:41:47 +00:00
|
|
|
main_config_reloader->reload();
|
|
|
|
users_config_reloader->reload();
|
|
|
|
});
|
|
|
|
|
2017-08-09 15:34:09 +00:00
|
|
|
/// Limit on total number of concurrently executed queries.
|
2017-04-01 07:20:54 +00:00
|
|
|
global_context->getProcessList().setMaxSize(config().getInt("max_concurrent_queries", 0));
|
|
|
|
|
|
|
|
/// Setup protection to avoid accidental DROP for big tables (that are greater than 50 GB by default)
|
|
|
|
if (config().has("max_table_size_to_drop"))
|
|
|
|
global_context->setMaxTableSizeToDrop(config().getUInt64("max_table_size_to_drop"));
|
|
|
|
|
2018-08-01 17:41:18 +00:00
|
|
|
if (config().has("max_partition_size_to_drop"))
|
2018-08-03 08:33:57 +00:00
|
|
|
global_context->setMaxPartitionSizeToDrop(config().getUInt64("max_partition_size_to_drop"));
|
2018-08-01 17:41:18 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Size of cache for uncompressed blocks. Zero means disabled.
|
2017-04-12 16:37:19 +00:00
|
|
|
size_t uncompressed_cache_size = config().getUInt64("uncompressed_cache_size", 0);
|
2017-04-01 07:20:54 +00:00
|
|
|
if (uncompressed_cache_size)
|
|
|
|
global_context->setUncompressedCache(uncompressed_cache_size);
|
|
|
|
|
2018-02-01 13:52:29 +00:00
|
|
|
/// Load global settings from default_profile and system_profile.
|
|
|
|
global_context->setDefaultProfiles(config());
|
2017-04-01 07:20:54 +00:00
|
|
|
Settings & settings = global_context->getSettingsRef();
|
|
|
|
|
2017-12-13 20:32:26 +00:00
|
|
|
/// Size of cache for marks (index of MergeTree family of tables). It is necessary.
|
|
|
|
size_t mark_cache_size = config().getUInt64("mark_cache_size");
|
|
|
|
if (mark_cache_size)
|
|
|
|
global_context->setMarkCache(mark_cache_size);
|
|
|
|
|
2017-11-10 06:48:28 +00:00
|
|
|
/// Set path for format schema files
|
|
|
|
auto format_schema_path = Poco::File(config().getString("format_schema_path", path + "format_schemas/"));
|
|
|
|
global_context->setFormatSchemaPath(format_schema_path.path() + "/");
|
|
|
|
format_schema_path.createDirectories();
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
LOG_INFO(log, "Loading metadata.");
|
2017-06-18 05:43:29 +00:00
|
|
|
loadMetadataSystem(*global_context);
|
2018-03-10 19:57:13 +00:00
|
|
|
/// After attaching system databases we can initialize system log.
|
|
|
|
global_context->initializeSystemLogs();
|
2017-06-18 05:43:29 +00:00
|
|
|
/// After the system database is created, attach virtual system tables (in addition to query_log and part_log)
|
|
|
|
attachSystemTablesServer(*global_context->getDatabase("system"), has_zookeeper);
|
|
|
|
/// Then, load remaining databases
|
2017-04-01 07:20:54 +00:00
|
|
|
loadMetadata(*global_context);
|
|
|
|
LOG_DEBUG(log, "Loaded metadata.");
|
|
|
|
|
|
|
|
global_context->setCurrentDatabase(default_database);
|
|
|
|
|
2017-08-30 14:47:35 +00:00
|
|
|
SCOPE_EXIT({
|
|
|
|
/** Ask to cancel background jobs all table engines,
|
|
|
|
* and also query_log.
|
|
|
|
* It is important to do early, not in destructor of Context, because
|
|
|
|
* table engines could use Context on destroy.
|
|
|
|
*/
|
|
|
|
LOG_INFO(log, "Shutting down storages.");
|
|
|
|
global_context->shutdown();
|
|
|
|
LOG_DEBUG(log, "Shutted down storages.");
|
|
|
|
});
|
|
|
|
|
2017-04-13 13:42:29 +00:00
|
|
|
if (has_zookeeper && config().has("distributed_ddl"))
|
|
|
|
{
|
2017-04-18 15:44:31 +00:00
|
|
|
/// DDL worker should be started after all tables were loaded
|
2017-04-13 16:12:56 +00:00
|
|
|
String ddl_zookeeper_path = config().getString("distributed_ddl.path", "/clickhouse/task_queue/ddl/");
|
2017-08-14 05:44:04 +00:00
|
|
|
global_context->setDDLWorker(std::make_shared<DDLWorker>(ddl_zookeeper_path, *global_context, &config(), "distributed_ddl"));
|
2017-04-13 13:42:29 +00:00
|
|
|
}
|
|
|
|
|
2018-03-29 20:21:01 +00:00
|
|
|
std::unique_ptr<DNSCacheUpdater> dns_cache_updater;
|
|
|
|
if (config().has("disable_internal_dns_cache") && config().getInt("disable_internal_dns_cache"))
|
|
|
|
{
|
|
|
|
/// Disable DNS caching at all
|
2018-04-19 13:56:14 +00:00
|
|
|
DNSResolver::instance().setDisableCacheFlag();
|
2018-03-29 20:21:01 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
/// Initialize a watcher updating DNS cache in case of network errors
|
|
|
|
dns_cache_updater = std::make_unique<DNSCacheUpdater>(*global_context);
|
|
|
|
}
|
2018-03-26 14:12:07 +00:00
|
|
|
|
2018-06-14 14:29:42 +00:00
|
|
|
if (!TaskStatsInfoGetter::checkProcessHasRequiredPermissions())
|
|
|
|
{
|
|
|
|
LOG_INFO(log, "It looks like the process has not CAP_NET_ADMIN capability, some performance statistics will be disabled."
|
|
|
|
" It could happen due to incorrect clickhouse package installation."
|
|
|
|
" You could resolve the problem manually calling 'sudo setcap cap_net_admin=+ep /usr/bin/clickhouse'");
|
|
|
|
}
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2017-09-08 16:41:35 +00:00
|
|
|
Poco::Timespan keep_alive_timeout(config().getUInt("keep_alive_timeout", 10), 0);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-09-08 16:41:35 +00:00
|
|
|
Poco::ThreadPool server_pool(3, config().getUInt("max_connections", 1024));
|
2017-04-01 07:20:54 +00:00
|
|
|
Poco::Net::HTTPServerParams::Ptr http_params = new Poco::Net::HTTPServerParams;
|
2018-07-10 18:39:32 +00:00
|
|
|
http_params->setTimeout(settings.http_receive_timeout);
|
2017-04-01 07:20:54 +00:00
|
|
|
http_params->setKeepAliveTimeout(keep_alive_timeout);
|
|
|
|
|
|
|
|
std::vector<std::unique_ptr<Poco::Net::TCPServer>> servers;
|
|
|
|
|
2017-06-22 18:56:40 +00:00
|
|
|
std::vector<std::string> listen_hosts = DB::getMultipleValuesFromConfig(config(), "", "listen_host");
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-03-29 17:24:46 +00:00
|
|
|
bool listen_try = config().getBool("listen_try", false);
|
2017-04-01 07:20:54 +00:00
|
|
|
if (listen_hosts.empty())
|
|
|
|
{
|
|
|
|
listen_hosts.emplace_back("::1");
|
|
|
|
listen_hosts.emplace_back("127.0.0.1");
|
2018-02-12 19:36:18 +00:00
|
|
|
listen_try = true;
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
2017-09-06 02:42:44 +00:00
|
|
|
auto make_socket_address = [&](const std::string & host, UInt16 port)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
Poco::Net::SocketAddress socket_address;
|
|
|
|
try
|
|
|
|
{
|
|
|
|
socket_address = Poco::Net::SocketAddress(host, port);
|
|
|
|
}
|
|
|
|
catch (const Poco::Net::DNSException & e)
|
|
|
|
{
|
2018-03-12 15:48:55 +00:00
|
|
|
const auto code = e.code();
|
|
|
|
if (code == EAI_FAMILY
|
2017-03-07 19:01:37 +00:00
|
|
|
#if defined(EAI_ADDRFAMILY)
|
2018-03-12 15:48:55 +00:00
|
|
|
|| code == EAI_ADDRFAMILY
|
2017-03-07 19:01:37 +00:00
|
|
|
#endif
|
2017-04-01 07:20:54 +00:00
|
|
|
)
|
|
|
|
{
|
|
|
|
LOG_ERROR(log,
|
2018-03-12 15:48:55 +00:00
|
|
|
"Cannot resolve listen_host (" << host << "), error " << e.code() << ": " << e.message() << ". "
|
2017-05-04 04:01:19 +00:00
|
|
|
"If it is an IPv6 address and your host has disabled IPv6, then consider to "
|
|
|
|
"specify IPv4 address to listen in <listen_host> element of configuration "
|
|
|
|
"file. Example: <listen_host>0.0.0.0</listen_host>");
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
throw;
|
|
|
|
}
|
|
|
|
return socket_address;
|
|
|
|
};
|
|
|
|
|
2018-03-29 17:24:46 +00:00
|
|
|
auto socket_bind_listen = [&](auto & socket, const std::string & host, UInt16 port, bool secure = 0)
|
|
|
|
{
|
|
|
|
auto address = make_socket_address(host, port);
|
|
|
|
#if !POCO_CLICKHOUSE_PATCH || POCO_VERSION <= 0x02000000 // TODO: fill correct version
|
|
|
|
if (secure)
|
|
|
|
/// Bug in old poco, listen() after bind() with reusePort param will fail because have no implementation in SecureServerSocketImpl
|
|
|
|
/// https://github.com/pocoproject/poco/pull/2257
|
2018-03-30 12:42:06 +00:00
|
|
|
socket.bind(address, /* reuseAddress = */ true);
|
2018-03-29 17:24:46 +00:00
|
|
|
else
|
|
|
|
#endif
|
2018-03-30 12:42:06 +00:00
|
|
|
#if POCO_VERSION < 0x01080000
|
|
|
|
socket.bind(address, /* reuseAddress = */ true);
|
|
|
|
#else
|
2018-03-29 17:24:46 +00:00
|
|
|
socket.bind(address, /* reuseAddress = */ true, /* reusePort = */ config().getBool("listen_reuse_port", false));
|
2018-03-30 12:42:06 +00:00
|
|
|
#endif
|
2018-03-29 17:24:46 +00:00
|
|
|
|
|
|
|
socket.listen(/* backlog = */ config().getUInt("listen_backlog", 64));
|
|
|
|
|
|
|
|
return address;
|
|
|
|
};
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
for (const auto & listen_host : listen_hosts)
|
|
|
|
{
|
|
|
|
/// For testing purposes, user may omit tcp_port or http_port or https_port in configuration file.
|
2017-05-05 21:25:53 +00:00
|
|
|
try
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2017-04-27 21:51:09 +00:00
|
|
|
/// HTTP
|
|
|
|
if (config().has("http_port"))
|
|
|
|
{
|
2018-03-29 17:24:46 +00:00
|
|
|
Poco::Net::ServerSocket socket;
|
|
|
|
auto address = socket_bind_listen(socket, listen_host, config().getInt("http_port"));
|
|
|
|
socket.setReceiveTimeout(settings.http_receive_timeout);
|
|
|
|
socket.setSendTimeout(settings.http_send_timeout);
|
2017-04-27 21:51:09 +00:00
|
|
|
servers.emplace_back(new Poco::Net::HTTPServer(
|
2017-08-09 14:33:07 +00:00
|
|
|
new HTTPHandlerFactory(*this, "HTTPHandler-factory"),
|
|
|
|
server_pool,
|
2018-03-29 17:24:46 +00:00
|
|
|
socket,
|
2017-08-09 14:33:07 +00:00
|
|
|
http_params));
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2018-03-29 17:24:46 +00:00
|
|
|
LOG_INFO(log, "Listening http://" + address.toString());
|
2017-04-27 21:51:09 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-27 21:51:09 +00:00
|
|
|
/// HTTPS
|
|
|
|
if (config().has("https_port"))
|
|
|
|
{
|
2018-05-14 18:36:01 +00:00
|
|
|
#if USE_POCO_NETSSL
|
2018-06-16 05:54:06 +00:00
|
|
|
initSSL();
|
2018-03-29 17:24:46 +00:00
|
|
|
Poco::Net::SecureServerSocket socket;
|
|
|
|
auto address = socket_bind_listen(socket, listen_host, config().getInt("https_port"), /* secure = */ true);
|
|
|
|
socket.setReceiveTimeout(settings.http_receive_timeout);
|
|
|
|
socket.setSendTimeout(settings.http_send_timeout);
|
2017-04-27 21:51:09 +00:00
|
|
|
servers.emplace_back(new Poco::Net::HTTPServer(
|
2018-03-13 19:49:21 +00:00
|
|
|
new HTTPHandlerFactory(*this, "HTTPSHandler-factory"),
|
2017-08-09 14:33:07 +00:00
|
|
|
server_pool,
|
2018-03-29 17:24:46 +00:00
|
|
|
socket,
|
2017-08-09 14:33:07 +00:00
|
|
|
http_params));
|
2017-04-27 21:51:09 +00:00
|
|
|
|
2018-03-29 17:24:46 +00:00
|
|
|
LOG_INFO(log, "Listening https://" + address.toString());
|
2017-08-09 14:33:07 +00:00
|
|
|
#else
|
2017-12-28 04:33:35 +00:00
|
|
|
throw Exception{"HTTPS protocol is disabled because Poco library was built without NetSSL support.",
|
2017-04-27 21:51:09 +00:00
|
|
|
ErrorCodes::SUPPORT_IS_DISABLED};
|
2017-08-09 14:33:07 +00:00
|
|
|
#endif
|
2017-04-27 21:51:09 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-27 21:51:09 +00:00
|
|
|
/// TCP
|
|
|
|
if (config().has("tcp_port"))
|
|
|
|
{
|
2018-03-29 17:24:46 +00:00
|
|
|
Poco::Net::ServerSocket socket;
|
|
|
|
auto address = socket_bind_listen(socket, listen_host, config().getInt("tcp_port"));
|
|
|
|
socket.setReceiveTimeout(settings.receive_timeout);
|
|
|
|
socket.setSendTimeout(settings.send_timeout);
|
2017-08-09 14:33:07 +00:00
|
|
|
servers.emplace_back(new Poco::Net::TCPServer(
|
|
|
|
new TCPHandlerFactory(*this),
|
|
|
|
server_pool,
|
2018-03-29 17:24:46 +00:00
|
|
|
socket,
|
2017-08-09 14:33:07 +00:00
|
|
|
new Poco::Net::TCPServerParams));
|
2017-04-27 21:51:09 +00:00
|
|
|
|
2018-03-29 17:24:46 +00:00
|
|
|
LOG_INFO(log, "Listening tcp: " + address.toString());
|
2017-04-27 21:51:09 +00:00
|
|
|
}
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-10-03 19:11:38 +00:00
|
|
|
/// TCP with SSL
|
2018-03-29 01:41:06 +00:00
|
|
|
if (config().has("tcp_port_secure"))
|
2017-09-28 19:43:31 +00:00
|
|
|
{
|
2018-05-14 18:36:01 +00:00
|
|
|
#if USE_POCO_NETSSL
|
2018-06-16 05:54:06 +00:00
|
|
|
initSSL();
|
2018-03-29 17:24:46 +00:00
|
|
|
Poco::Net::SecureServerSocket socket;
|
|
|
|
auto address = socket_bind_listen(socket, listen_host, config().getInt("tcp_port_secure"), /* secure = */ true);
|
|
|
|
socket.setReceiveTimeout(settings.receive_timeout);
|
|
|
|
socket.setSendTimeout(settings.send_timeout);
|
2017-09-28 19:43:31 +00:00
|
|
|
servers.emplace_back(new Poco::Net::TCPServer(
|
2018-03-13 19:49:21 +00:00
|
|
|
new TCPHandlerFactory(*this, /* secure= */ true ),
|
2017-09-28 19:43:31 +00:00
|
|
|
server_pool,
|
2018-03-29 17:24:46 +00:00
|
|
|
socket,
|
2017-09-28 19:43:31 +00:00
|
|
|
new Poco::Net::TCPServerParams));
|
2018-03-29 17:24:46 +00:00
|
|
|
LOG_INFO(log, "Listening tcp_secure: " + address.toString());
|
2017-09-28 19:43:31 +00:00
|
|
|
#else
|
2017-12-28 04:33:35 +00:00
|
|
|
throw Exception{"SSL support for TCP protocol is disabled because Poco library was built without NetSSL support.",
|
2017-09-28 19:43:31 +00:00
|
|
|
ErrorCodes::SUPPORT_IS_DISABLED};
|
|
|
|
#endif
|
|
|
|
}
|
|
|
|
|
2017-04-27 21:51:09 +00:00
|
|
|
/// At least one of TCP and HTTP servers must be created.
|
|
|
|
if (servers.empty())
|
|
|
|
throw Exception("No 'tcp_port' and 'http_port' is specified in configuration file.", ErrorCodes::NO_ELEMENTS_IN_CONFIG);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-04-27 21:51:09 +00:00
|
|
|
/// Interserver IO HTTP
|
|
|
|
if (config().has("interserver_http_port"))
|
|
|
|
{
|
2018-03-29 17:24:46 +00:00
|
|
|
Poco::Net::ServerSocket socket;
|
|
|
|
auto address = socket_bind_listen(socket, listen_host, config().getInt("interserver_http_port"));
|
|
|
|
socket.setReceiveTimeout(settings.http_receive_timeout);
|
|
|
|
socket.setSendTimeout(settings.http_send_timeout);
|
2017-04-27 21:51:09 +00:00
|
|
|
servers.emplace_back(new Poco::Net::HTTPServer(
|
2017-08-09 14:33:07 +00:00
|
|
|
new InterserverIOHTTPHandlerFactory(*this, "InterserverIOHTTPHandler-factory"),
|
2017-04-27 21:51:09 +00:00
|
|
|
server_pool,
|
2018-03-29 17:24:46 +00:00
|
|
|
socket,
|
2017-04-27 21:51:09 +00:00
|
|
|
http_params));
|
|
|
|
|
2018-03-29 17:24:46 +00:00
|
|
|
LOG_INFO(log, "Listening interserver http: " + address.toString());
|
2017-04-27 21:51:09 +00:00
|
|
|
}
|
2018-07-30 18:32:21 +00:00
|
|
|
|
|
|
|
if (config().has("interserver_https_port"))
|
|
|
|
{
|
|
|
|
#if USE_POCO_NETSSL
|
|
|
|
initSSL();
|
|
|
|
Poco::Net::SecureServerSocket socket;
|
|
|
|
auto address = socket_bind_listen(socket, listen_host, config().getInt("interserver_https_port"), /* secure = */ true);
|
|
|
|
socket.setReceiveTimeout(settings.http_receive_timeout);
|
|
|
|
socket.setSendTimeout(settings.http_send_timeout);
|
|
|
|
servers.emplace_back(new Poco::Net::HTTPServer(
|
|
|
|
new InterserverIOHTTPHandlerFactory(*this, "InterserverIOHTTPHandler-factory"),
|
|
|
|
server_pool,
|
|
|
|
socket,
|
|
|
|
http_params));
|
|
|
|
|
|
|
|
LOG_INFO(log, "Listening interserver https: " + address.toString());
|
|
|
|
#else
|
|
|
|
throw Exception{"SSL support for TCP protocol is disabled because Poco library was built without NetSSL support.",
|
|
|
|
ErrorCodes::SUPPORT_IS_DISABLED};
|
|
|
|
#endif
|
|
|
|
}
|
2017-04-27 21:51:09 +00:00
|
|
|
}
|
|
|
|
catch (const Poco::Net::NetException & e)
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
2018-03-18 09:02:29 +00:00
|
|
|
if (listen_try)
|
2018-03-12 15:48:55 +00:00
|
|
|
LOG_ERROR(log, "Listen [" << listen_host << "]: " << e.code() << ": " << e.what() << ": " << e.message()
|
2017-05-04 04:02:28 +00:00
|
|
|
<< " If it is an IPv6 or IPv4 address and your host has disabled IPv6 or IPv4, then consider to "
|
|
|
|
"specify not disabled IPv4 or IPv6 address to listen in <listen_host> element of configuration "
|
|
|
|
"file. Example for disabled IPv6: <listen_host>0.0.0.0</listen_host> ."
|
|
|
|
" Example for disabled IPv4: <listen_host>::</listen_host>");
|
2017-04-27 21:51:09 +00:00
|
|
|
else
|
|
|
|
throw;
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2017-04-27 21:51:09 +00:00
|
|
|
if (servers.empty())
|
|
|
|
throw Exception("No servers started (add valid listen_host and 'tcp_port' or 'http_port' to configuration file.)", ErrorCodes::NO_ELEMENTS_IN_CONFIG);
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
for (auto & server : servers)
|
|
|
|
server->start();
|
|
|
|
|
2018-03-03 14:39:16 +00:00
|
|
|
main_config_reloader->start();
|
|
|
|
users_config_reloader->start();
|
|
|
|
|
2017-06-22 18:08:14 +00:00
|
|
|
{
|
|
|
|
std::stringstream message;
|
2017-06-22 18:16:28 +00:00
|
|
|
message << "Available RAM = " << formatReadableSizeWithBinarySuffix(getMemoryAmount()) << ";"
|
2017-06-22 18:08:14 +00:00
|
|
|
<< " physical cores = " << getNumberOfPhysicalCPUCores() << ";"
|
|
|
|
// on ARM processors it can show only enabled at current moment cores
|
2017-06-22 18:16:28 +00:00
|
|
|
<< " threads = " << std::thread::hardware_concurrency() << ".";
|
2017-06-22 18:08:14 +00:00
|
|
|
LOG_INFO(log, message.str());
|
|
|
|
}
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
LOG_INFO(log, "Ready for connections.");
|
|
|
|
|
|
|
|
SCOPE_EXIT({
|
|
|
|
LOG_DEBUG(log, "Received termination signal.");
|
|
|
|
LOG_DEBUG(log, "Waiting for current connections to close.");
|
|
|
|
|
|
|
|
is_cancelled = true;
|
|
|
|
|
|
|
|
int current_connections = 0;
|
|
|
|
for (auto & server : servers)
|
|
|
|
{
|
|
|
|
server->stop();
|
|
|
|
current_connections += server->currentConnections();
|
|
|
|
}
|
|
|
|
|
|
|
|
LOG_DEBUG(log,
|
|
|
|
"Closed all listening sockets."
|
2017-12-20 20:25:22 +00:00
|
|
|
<< (current_connections ? " Waiting for " + toString(current_connections) + " outstanding connections." : ""));
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
if (current_connections)
|
|
|
|
{
|
|
|
|
const int sleep_max_ms = 1000 * config().getInt("shutdown_wait_unfinished", 5);
|
|
|
|
const int sleep_one_ms = 100;
|
|
|
|
int sleep_current_ms = 0;
|
|
|
|
while (sleep_current_ms < sleep_max_ms)
|
|
|
|
{
|
|
|
|
current_connections = 0;
|
|
|
|
for (auto & server : servers)
|
|
|
|
current_connections += server->currentConnections();
|
|
|
|
if (!current_connections)
|
|
|
|
break;
|
|
|
|
sleep_current_ms += sleep_one_ms;
|
|
|
|
std::this_thread::sleep_for(std::chrono::milliseconds(sleep_one_ms));
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
LOG_DEBUG(
|
2017-12-20 20:25:22 +00:00
|
|
|
log, "Closed connections." << (current_connections ? " But " + toString(current_connections) + " remains."
|
2017-05-05 21:25:53 +00:00
|
|
|
" Tip: To increase wait time add to config: <shutdown_wait_unfinished>60</shutdown_wait_unfinished>" : ""));
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
main_config_reloader.reset();
|
|
|
|
users_config_reloader.reset();
|
|
|
|
});
|
|
|
|
|
|
|
|
/// try to load dictionaries immediately, throw on error and die
|
|
|
|
try
|
|
|
|
{
|
|
|
|
if (!config().getBool("dictionaries_lazy_load", true))
|
|
|
|
{
|
|
|
|
global_context->tryCreateEmbeddedDictionaries();
|
|
|
|
global_context->tryCreateExternalDictionaries();
|
|
|
|
}
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
LOG_ERROR(log, "Caught exception while loading dictionaries.");
|
|
|
|
throw;
|
|
|
|
}
|
|
|
|
|
|
|
|
/// This object will periodically calculate some metrics.
|
|
|
|
AsynchronousMetrics async_metrics(*global_context);
|
2017-06-18 05:43:29 +00:00
|
|
|
attachSystemTablesAsync(*global_context->getDatabase("system"), async_metrics);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
std::vector<std::unique_ptr<MetricsTransmitter>> metrics_transmitters;
|
|
|
|
for (const auto & graphite_key : DB::getMultipleKeysFromConfig(config(), "", "graphite"))
|
|
|
|
{
|
2017-08-24 14:51:13 +00:00
|
|
|
metrics_transmitters.emplace_back(std::make_unique<MetricsTransmitter>(
|
|
|
|
*global_context, async_metrics, graphite_key));
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
|
2017-06-02 18:48:33 +00:00
|
|
|
SessionCleaner session_cleaner(*global_context);
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
waitForTerminationRequest();
|
|
|
|
}
|
|
|
|
|
|
|
|
return Application::EXIT_OK;
|
2012-03-09 03:06:09 +00:00
|
|
|
}
|
|
|
|
}
|
2016-10-31 19:54:49 +00:00
|
|
|
|
2017-12-02 02:47:12 +00:00
|
|
|
int mainEntryClickHouseServer(int argc, char ** argv)
|
|
|
|
{
|
|
|
|
DB::Server app;
|
|
|
|
try
|
|
|
|
{
|
|
|
|
return app.run(argc, argv);
|
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
|
|
|
std::cerr << DB::getCurrentExceptionMessage(true) << "\n";
|
|
|
|
auto code = DB::getCurrentExceptionCode();
|
|
|
|
return code ? code : 1;
|
|
|
|
}
|
|
|
|
}
|