2014-08-15 09:50:05 +00:00
|
|
|
#pragma once
|
|
|
|
|
|
|
|
#include <DB/DataStreams/RemoteBlockOutputStream.h>
|
|
|
|
#include <DB/Common/escapeForFileName.h>
|
2015-09-24 18:54:21 +00:00
|
|
|
#include <DB/Common/setThreadName.h>
|
2016-01-21 01:47:28 +00:00
|
|
|
#include <DB/Common/CurrentMetrics.h>
|
2014-08-20 02:20:07 +00:00
|
|
|
#include <DB/Storages/StorageDistributed.h>
|
2015-04-16 06:12:35 +00:00
|
|
|
#include <DB/IO/ReadBufferFromFile.h>
|
2015-04-16 07:22:29 +00:00
|
|
|
|
2014-08-15 09:50:05 +00:00
|
|
|
#include <boost/algorithm/string/find_iterator.hpp>
|
|
|
|
#include <boost/algorithm/string/finder.hpp>
|
2015-04-16 07:22:29 +00:00
|
|
|
|
2015-04-16 06:12:35 +00:00
|
|
|
#include <Poco/DirectoryIterator.h>
|
2015-04-16 07:22:29 +00:00
|
|
|
|
2014-08-19 08:04:13 +00:00
|
|
|
#include <thread>
|
|
|
|
#include <mutex>
|
2015-04-16 07:22:29 +00:00
|
|
|
#include <condition_variable>
|
|
|
|
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2016-01-11 21:46:36 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
|
|
|
extern const int INCORRECT_FILE_NAME;
|
|
|
|
extern const int CHECKSUM_DOESNT_MATCH;
|
|
|
|
extern const int TOO_LARGE_SIZE_COMPRESSED;
|
|
|
|
}
|
|
|
|
|
|
|
|
|
2014-08-15 09:50:05 +00:00
|
|
|
namespace
|
|
|
|
{
|
2014-12-25 20:53:18 +00:00
|
|
|
static constexpr const std::chrono::seconds max_sleep_time{30};
|
|
|
|
static constexpr const std::chrono::minutes decrease_error_count_period{5};
|
|
|
|
|
2014-08-20 02:20:07 +00:00
|
|
|
template <typename PoolFactory>
|
|
|
|
ConnectionPools createPoolsForAddresses(const std::string & name, PoolFactory && factory)
|
2014-08-15 09:50:05 +00:00
|
|
|
{
|
|
|
|
ConnectionPools pools;
|
|
|
|
|
|
|
|
for (auto it = boost::make_split_iterator(name, boost::first_finder(",")); it != decltype(it){}; ++it)
|
|
|
|
{
|
2014-08-22 14:05:34 +00:00
|
|
|
const auto address = boost::copy_range<std::string>(*it);
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
const auto user_pw_end = strchr(address.data(), '@');
|
|
|
|
const auto colon = strchr(address.data(), ':');
|
|
|
|
if (!user_pw_end || !colon)
|
2014-08-21 12:07:29 +00:00
|
|
|
throw Exception{
|
|
|
|
"Shard address '" + address + "' does not match to 'user[:password]@host:port' pattern",
|
|
|
|
ErrorCodes::INCORRECT_FILE_NAME
|
|
|
|
};
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
const auto has_pw = colon < user_pw_end;
|
|
|
|
const auto host_end = has_pw ? strchr(user_pw_end + 1, ':') : colon;
|
|
|
|
if (!host_end)
|
2014-08-21 12:07:29 +00:00
|
|
|
throw Exception{
|
|
|
|
"Shard address '" + address + "' does not contain port",
|
|
|
|
ErrorCodes::INCORRECT_FILE_NAME
|
|
|
|
};
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
const auto user = unescapeForFileName({address.data(), has_pw ? colon : user_pw_end});
|
|
|
|
const auto password = has_pw ? unescapeForFileName({colon + 1, user_pw_end}) : std::string{};
|
|
|
|
const auto host = unescapeForFileName({user_pw_end + 1, host_end});
|
2014-08-21 12:07:29 +00:00
|
|
|
const auto port = parse<UInt16>(host_end + 1);
|
2014-08-15 09:50:05 +00:00
|
|
|
|
2014-08-15 12:07:50 +00:00
|
|
|
pools.emplace_back(factory(host, port, user, password));
|
2014-08-15 09:50:05 +00:00
|
|
|
}
|
|
|
|
|
2014-08-21 12:07:29 +00:00
|
|
|
return pools;
|
2014-08-15 09:50:05 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2014-08-21 12:07:29 +00:00
|
|
|
/** Implementation for StorageDistributed::DirectoryMonitor nested class.
|
|
|
|
* This type is not designed for standalone use. */
|
2014-08-19 08:04:13 +00:00
|
|
|
class StorageDistributed::DirectoryMonitor
|
2014-08-15 09:50:05 +00:00
|
|
|
{
|
|
|
|
public:
|
|
|
|
DirectoryMonitor(StorageDistributed & storage, const std::string & name)
|
2014-08-21 12:07:29 +00:00
|
|
|
: storage(storage), pool{createPool(name)}, path{storage.path + name + '/'}
|
2014-12-25 20:53:18 +00:00
|
|
|
, default_sleep_time{storage.context.getSettingsRef().distributed_directory_monitor_sleep_time_ms.totalMilliseconds()}
|
|
|
|
, sleep_time{default_sleep_time}
|
2014-08-21 12:07:29 +00:00
|
|
|
, log{&Logger::get(getLoggerName())}
|
2014-08-15 09:50:05 +00:00
|
|
|
{
|
|
|
|
}
|
|
|
|
|
2014-08-19 08:04:13 +00:00
|
|
|
~DirectoryMonitor()
|
|
|
|
{
|
|
|
|
{
|
|
|
|
quit = true;
|
2014-08-22 14:05:34 +00:00
|
|
|
std::lock_guard<std::mutex> lock{mutex};
|
2014-08-19 08:04:13 +00:00
|
|
|
}
|
|
|
|
cond.notify_one();
|
|
|
|
thread.join();
|
|
|
|
}
|
|
|
|
|
|
|
|
private:
|
2014-08-15 09:50:05 +00:00
|
|
|
void run()
|
|
|
|
{
|
2015-09-24 18:54:21 +00:00
|
|
|
setThreadName("DistrDirMonitor");
|
|
|
|
|
2014-08-19 08:04:13 +00:00
|
|
|
std::unique_lock<std::mutex> lock{mutex};
|
|
|
|
|
|
|
|
const auto quit_requested = [this] { return quit; };
|
|
|
|
|
|
|
|
while (!quit_requested())
|
2014-08-15 09:50:05 +00:00
|
|
|
{
|
2014-08-19 08:04:13 +00:00
|
|
|
auto do_sleep = true;
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
try
|
|
|
|
{
|
2014-08-19 08:04:13 +00:00
|
|
|
do_sleep = !findFiles();
|
2014-08-15 09:50:05 +00:00
|
|
|
}
|
|
|
|
catch (...)
|
|
|
|
{
|
2014-08-19 08:04:13 +00:00
|
|
|
do_sleep = true;
|
2014-12-25 20:53:18 +00:00
|
|
|
++error_count;
|
|
|
|
sleep_time = std::min(
|
|
|
|
std::chrono::milliseconds{std::int64_t(default_sleep_time.count() * std::exp2(error_count))},
|
|
|
|
std::chrono::milliseconds{max_sleep_time});
|
2014-08-15 09:50:05 +00:00
|
|
|
tryLogCurrentException(getLoggerName().data());
|
2014-12-25 20:53:18 +00:00
|
|
|
};
|
2014-08-15 09:50:05 +00:00
|
|
|
|
2014-08-19 08:04:13 +00:00
|
|
|
if (do_sleep)
|
|
|
|
cond.wait_for(lock, sleep_time, quit_requested);
|
2014-12-25 20:53:18 +00:00
|
|
|
|
|
|
|
const auto now = std::chrono::system_clock::now();
|
|
|
|
if (now - last_decrease_time > decrease_error_count_period)
|
|
|
|
{
|
|
|
|
error_count /= 2;
|
|
|
|
last_decrease_time = now;
|
|
|
|
}
|
2014-08-15 09:50:05 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
ConnectionPoolPtr createPool(const std::string & name)
|
|
|
|
{
|
2014-08-21 12:07:29 +00:00
|
|
|
const auto pool_factory = [this, &name] (const std::string & host, const UInt16 port,
|
|
|
|
const std::string & user, const std::string & password) {
|
2014-08-15 09:50:05 +00:00
|
|
|
return new ConnectionPool{
|
|
|
|
1, host, port, "",
|
2015-05-28 03:49:28 +00:00
|
|
|
user, password,
|
2014-08-20 02:20:07 +00:00
|
|
|
storage.getName() + '_' + name};
|
2014-08-15 09:50:05 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
auto pools = createPoolsForAddresses(name, pool_factory);
|
|
|
|
|
2014-08-21 12:07:29 +00:00
|
|
|
return pools.size() == 1 ? pools.front() : new ConnectionPoolWithFailover(pools, LoadBalancing::RANDOM);
|
2014-08-15 09:50:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
bool findFiles()
|
|
|
|
{
|
|
|
|
std::map<UInt64, std::string> files;
|
|
|
|
|
|
|
|
Poco::DirectoryIterator end;
|
|
|
|
for (Poco::DirectoryIterator it{path}; it != end; ++it)
|
|
|
|
{
|
|
|
|
const auto & file_path_str = it->path();
|
|
|
|
Poco::Path file_path{file_path_str};
|
|
|
|
|
|
|
|
if (!it->isDirectory() && 0 == strncmp(file_path.getExtension().data(), "bin", strlen("bin")))
|
2014-08-21 12:07:29 +00:00
|
|
|
files[parse<UInt64>(file_path.getBaseName())] = file_path_str;
|
2014-08-15 09:50:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if (files.empty())
|
|
|
|
return false;
|
|
|
|
|
|
|
|
for (const auto & file : files)
|
|
|
|
{
|
2014-08-19 08:04:13 +00:00
|
|
|
if (quit)
|
2014-08-15 09:50:05 +00:00
|
|
|
return true;
|
|
|
|
|
|
|
|
processFile(file.second);
|
|
|
|
}
|
|
|
|
|
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
|
|
|
void processFile(const std::string & file_path)
|
|
|
|
{
|
|
|
|
LOG_TRACE(log, "Started processing `" << file_path << '`');
|
|
|
|
auto connection = pool->get();
|
|
|
|
|
|
|
|
try
|
|
|
|
{
|
2016-01-21 01:47:28 +00:00
|
|
|
CurrentMetrics::Increment metric_increment{CurrentMetrics::DistributedSend};
|
|
|
|
|
2014-08-21 12:07:29 +00:00
|
|
|
ReadBufferFromFile in{file_path};
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
std::string insert_query;
|
2014-08-21 12:07:29 +00:00
|
|
|
readStringBinary(insert_query, in);
|
2014-08-15 09:50:05 +00:00
|
|
|
|
2014-08-21 12:07:29 +00:00
|
|
|
RemoteBlockOutputStream remote{*connection, insert_query};
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
remote.writePrefix();
|
|
|
|
remote.writePrepared(in);
|
|
|
|
remote.writeSuffix();
|
|
|
|
}
|
|
|
|
catch (const Exception & e)
|
|
|
|
{
|
|
|
|
const auto code = e.code();
|
|
|
|
|
|
|
|
/// mark file as broken if necessary
|
|
|
|
if (code == ErrorCodes::CHECKSUM_DOESNT_MATCH ||
|
|
|
|
code == ErrorCodes::TOO_LARGE_SIZE_COMPRESSED ||
|
|
|
|
code == ErrorCodes::CANNOT_READ_ALL_DATA)
|
|
|
|
{
|
|
|
|
const auto last_path_separator_pos = file_path.rfind('/');
|
|
|
|
const auto & path = file_path.substr(0, last_path_separator_pos + 1);
|
|
|
|
const auto & file_name = file_path.substr(last_path_separator_pos + 1);
|
|
|
|
const auto & broken_path = path + "broken/";
|
2014-08-21 12:07:29 +00:00
|
|
|
const auto & broken_file_path = broken_path + file_name;
|
2014-08-15 09:50:05 +00:00
|
|
|
|
|
|
|
Poco::File{broken_path}.createDirectory();
|
2014-08-21 12:07:29 +00:00
|
|
|
Poco::File{file_path}.renameTo(broken_file_path);
|
2014-08-15 09:50:05 +00:00
|
|
|
|
2014-08-21 12:07:29 +00:00
|
|
|
LOG_ERROR(log, "Renamed `" << file_path << "` to `" << broken_file_path << '`');
|
2014-08-15 09:50:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
throw;
|
|
|
|
}
|
|
|
|
|
|
|
|
Poco::File{file_path}.remove();
|
|
|
|
|
|
|
|
LOG_TRACE(log, "Finished processing `" << file_path << '`');
|
|
|
|
}
|
|
|
|
|
2014-08-21 12:07:29 +00:00
|
|
|
std::string getLoggerName() const
|
|
|
|
{
|
2014-08-15 09:50:05 +00:00
|
|
|
return storage.name + '.' + storage.getName() + ".DirectoryMonitor";
|
|
|
|
}
|
|
|
|
|
|
|
|
StorageDistributed & storage;
|
|
|
|
ConnectionPoolPtr pool;
|
|
|
|
std::string path;
|
2014-12-25 20:53:18 +00:00
|
|
|
std::size_t error_count{};
|
|
|
|
std::chrono::milliseconds default_sleep_time;
|
2014-08-15 09:50:05 +00:00
|
|
|
std::chrono::milliseconds sleep_time;
|
2014-12-25 20:53:18 +00:00
|
|
|
std::chrono::time_point<std::chrono::system_clock> last_decrease_time{
|
|
|
|
std::chrono::system_clock::now()
|
|
|
|
};
|
2014-08-19 08:04:13 +00:00
|
|
|
bool quit{false};
|
|
|
|
std::mutex mutex;
|
|
|
|
std::condition_variable cond;
|
2014-08-15 09:50:05 +00:00
|
|
|
Logger * log;
|
2014-08-19 08:04:13 +00:00
|
|
|
std::thread thread{&DirectoryMonitor::run, this};
|
2014-08-15 09:50:05 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|