2014-08-15 09:50:05 +00:00
|
|
|
#pragma once
|
|
|
|
|
2020-04-14 18:12:08 +00:00
|
|
|
#include <Core/BackgroundSchedulePool.h>
|
2020-06-03 23:50:47 +00:00
|
|
|
#include <Client/ConnectionPool.h>
|
2015-04-16 07:22:29 +00:00
|
|
|
|
2017-09-09 23:04:22 +00:00
|
|
|
#include <atomic>
|
2014-08-19 08:04:13 +00:00
|
|
|
#include <mutex>
|
2015-04-16 07:22:29 +00:00
|
|
|
#include <condition_variable>
|
2019-04-08 10:04:26 +00:00
|
|
|
#include <IO/ReadBufferFromFile.h>
|
2015-04-16 07:22:29 +00:00
|
|
|
|
2014-08-15 09:50:05 +00:00
|
|
|
|
2020-05-27 10:07:38 +00:00
|
|
|
namespace CurrentMetrics { class Increment; }
|
|
|
|
|
2014-08-15 09:50:05 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2021-01-09 12:26:37 +00:00
|
|
|
class IDisk;
|
|
|
|
using DiskPtr = std::shared_ptr<IDisk>;
|
|
|
|
|
2020-06-03 23:50:47 +00:00
|
|
|
class StorageDistributed;
|
|
|
|
class ActionBlocker;
|
|
|
|
class BackgroundSchedulePool;
|
|
|
|
|
2016-12-12 03:33:34 +00:00
|
|
|
/** Details of StorageDistributed.
|
|
|
|
* This type is not designed for standalone use.
|
|
|
|
*/
|
|
|
|
class StorageDistributedDirectoryMonitor
|
2014-08-15 09:50:05 +00:00
|
|
|
{
|
|
|
|
public:
|
2019-04-08 05:13:16 +00:00
|
|
|
StorageDistributedDirectoryMonitor(
|
2021-01-09 12:26:37 +00:00
|
|
|
StorageDistributed & storage_,
|
|
|
|
const DiskPtr & disk_,
|
|
|
|
const std::string & relative_path_,
|
|
|
|
ConnectionPoolPtr pool_,
|
|
|
|
ActionBlocker & monitor_blocker_,
|
|
|
|
BackgroundSchedulePool & bg_pool);
|
2019-04-08 05:13:16 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
~StorageDistributedDirectoryMonitor();
|
2014-08-19 08:04:13 +00:00
|
|
|
|
2017-07-27 15:24:39 +00:00
|
|
|
static ConnectionPoolPtr createPool(const std::string & name, const StorageDistributed & storage);
|
2017-07-25 19:42:36 +00:00
|
|
|
|
2021-01-09 12:26:37 +00:00
|
|
|
void updatePath(const std::string & new_relative_path);
|
2019-12-19 19:39:49 +00:00
|
|
|
|
2019-05-10 04:19:02 +00:00
|
|
|
void flushAllData();
|
2019-04-08 05:13:16 +00:00
|
|
|
|
2018-04-21 00:35:20 +00:00
|
|
|
void shutdownAndDropAllData();
|
2020-01-04 14:45:11 +00:00
|
|
|
|
|
|
|
static BlockInputStreamPtr createStreamFromFile(const String & file_name);
|
2020-04-14 18:12:08 +00:00
|
|
|
|
|
|
|
/// For scheduling via DistributedBlockOutputStream
|
2021-01-26 18:45:37 +00:00
|
|
|
bool addAndSchedule(size_t file_size, size_t ms);
|
2020-06-02 23:47:32 +00:00
|
|
|
|
|
|
|
/// system.distribution_queue interface
|
2020-06-03 23:50:47 +00:00
|
|
|
struct Status
|
|
|
|
{
|
|
|
|
std::string path;
|
|
|
|
std::exception_ptr last_exception;
|
|
|
|
size_t error_count;
|
|
|
|
size_t files_count;
|
|
|
|
size_t bytes_count;
|
|
|
|
bool is_blocked;
|
|
|
|
};
|
|
|
|
Status getStatus() const;
|
2020-06-02 23:47:32 +00:00
|
|
|
|
2014-08-19 08:04:13 +00:00
|
|
|
private:
|
2017-04-01 07:20:54 +00:00
|
|
|
void run();
|
2020-06-02 23:47:32 +00:00
|
|
|
|
2021-01-26 18:45:37 +00:00
|
|
|
std::map<UInt64, std::string> getFiles() const;
|
2020-08-26 21:43:00 +00:00
|
|
|
bool processFiles(const std::map<UInt64, std::string> & files);
|
|
|
|
void processFile(const std::string & file_path);
|
|
|
|
void processFilesWithBatching(const std::map<UInt64, std::string> & files);
|
2018-03-27 18:59:53 +00:00
|
|
|
|
|
|
|
void markAsBroken(const std::string & file_path) const;
|
2021-01-26 18:45:37 +00:00
|
|
|
void markAsSend(const std::string & file_path) const;
|
2018-03-27 18:59:53 +00:00
|
|
|
bool maybeMarkAsBroken(const std::string & file_path, const Exception & e) const;
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
std::string getLoggerName() const;
|
|
|
|
|
|
|
|
StorageDistributed & storage;
|
2019-12-24 18:25:00 +00:00
|
|
|
const ConnectionPoolPtr pool;
|
2021-01-09 12:26:37 +00:00
|
|
|
|
|
|
|
DiskPtr disk;
|
|
|
|
std::string relative_path;
|
2017-04-01 07:20:54 +00:00
|
|
|
std::string path;
|
2017-07-10 15:28:04 +00:00
|
|
|
|
2019-12-24 18:25:00 +00:00
|
|
|
const bool should_batch_inserts = false;
|
2021-01-09 12:42:21 +00:00
|
|
|
const bool dir_fsync = false;
|
2019-12-24 18:25:00 +00:00
|
|
|
const size_t min_batched_block_size_rows = 0;
|
|
|
|
const size_t min_batched_block_size_bytes = 0;
|
2017-07-10 15:28:04 +00:00
|
|
|
String current_batch_file_path;
|
|
|
|
|
2017-07-21 12:03:37 +00:00
|
|
|
struct BatchHeader;
|
2017-07-10 15:28:04 +00:00
|
|
|
struct Batch;
|
|
|
|
|
2020-06-03 08:22:48 +00:00
|
|
|
mutable std::mutex metrics_mutex;
|
2020-06-02 23:47:32 +00:00
|
|
|
size_t error_count = 0;
|
2021-01-26 18:45:37 +00:00
|
|
|
mutable size_t files_count = 0;
|
|
|
|
mutable size_t bytes_count = 0;
|
2020-06-03 07:59:48 +00:00
|
|
|
std::exception_ptr last_exception;
|
2020-06-02 23:47:32 +00:00
|
|
|
|
2019-12-24 18:25:00 +00:00
|
|
|
const std::chrono::milliseconds default_sleep_time;
|
2017-04-01 07:20:54 +00:00
|
|
|
std::chrono::milliseconds sleep_time;
|
2019-12-24 18:25:00 +00:00
|
|
|
const std::chrono::milliseconds max_sleep_time;
|
2017-04-01 07:20:54 +00:00
|
|
|
std::chrono::time_point<std::chrono::system_clock> last_decrease_time {std::chrono::system_clock::now()};
|
2017-09-09 23:04:22 +00:00
|
|
|
std::atomic<bool> quit {false};
|
2017-04-01 07:20:54 +00:00
|
|
|
std::mutex mutex;
|
2020-05-30 21:57:37 +00:00
|
|
|
Poco::Logger * log;
|
2019-04-08 05:13:16 +00:00
|
|
|
ActionBlocker & monitor_blocker;
|
2020-04-14 18:12:08 +00:00
|
|
|
|
|
|
|
BackgroundSchedulePoolTaskHolder task_handle;
|
2019-04-08 10:04:26 +00:00
|
|
|
|
2021-01-26 18:45:37 +00:00
|
|
|
mutable CurrentMetrics::Increment metric_pending_files;
|
2020-08-26 21:43:00 +00:00
|
|
|
|
2020-01-04 18:33:16 +00:00
|
|
|
friend class DirectoryMonitorBlockInputStream;
|
2014-08-15 09:50:05 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|