2014-02-22 18:53:42 +00:00
|
|
|
#pragma once
|
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Client/ConnectionPool.h>
|
|
|
|
#include <Client/ConnectionPoolWithFailover.h>
|
2021-04-10 23:33:54 +00:00
|
|
|
|
2013-12-07 16:51:29 +00:00
|
|
|
#include <Poco/Net/SocketAddress.h>
|
|
|
|
|
2021-04-10 23:33:54 +00:00
|
|
|
#include <map>
|
|
|
|
|
2020-12-10 22:05:02 +00:00
|
|
|
namespace Poco
|
|
|
|
{
|
|
|
|
namespace Util
|
|
|
|
{
|
|
|
|
class AbstractConfiguration;
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2013-12-07 16:51:29 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
2020-12-10 22:05:02 +00:00
|
|
|
|
|
|
|
struct Settings;
|
|
|
|
|
2020-02-25 18:10:48 +00:00
|
|
|
namespace ErrorCodes
|
|
|
|
{
|
|
|
|
extern const int LOGICAL_ERROR;
|
|
|
|
}
|
2015-05-28 21:41:28 +00:00
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// Cluster contains connection pools to each node
|
|
|
|
/// With the local nodes, the connection is not established, but the request is executed directly.
|
|
|
|
/// Therefore we store only the number of local nodes
|
|
|
|
/// In the config, the cluster includes nodes <node> or <shard>
|
2015-10-20 14:59:29 +00:00
|
|
|
class Cluster
|
2013-12-07 16:51:29 +00:00
|
|
|
{
|
|
|
|
public:
|
2020-09-14 21:55:43 +00:00
|
|
|
Cluster(const Poco::Util::AbstractConfiguration & config,
|
|
|
|
const Settings & settings,
|
|
|
|
const String & config_prefix_,
|
|
|
|
const String & cluster_name);
|
2013-12-07 16:51:29 +00:00
|
|
|
|
2017-12-01 17:13:14 +00:00
|
|
|
/// Construct a cluster by the names of shards and replicas.
|
2018-03-05 12:47:09 +00:00
|
|
|
/// Local are treated as well as remote ones if treat_local_as_remote is true.
|
2021-07-23 11:16:35 +00:00
|
|
|
/// Local are also treated as remote if treat_local_port_as_remote is set and the local address includes a port
|
2017-09-07 17:55:02 +00:00
|
|
|
/// 'clickhouse_port' - port that this server instance listen for queries.
|
|
|
|
/// This parameter is needed only to check that some address is local (points to ourself).
|
2020-09-14 21:55:43 +00:00
|
|
|
///
|
|
|
|
/// Used for remote() function.
|
2021-07-23 11:16:35 +00:00
|
|
|
Cluster(
|
|
|
|
const Settings & settings,
|
|
|
|
const std::vector<std::vector<String>> & names,
|
|
|
|
const String & username,
|
|
|
|
const String & password,
|
|
|
|
UInt16 clickhouse_port,
|
|
|
|
bool treat_local_as_remote,
|
|
|
|
bool treat_local_port_as_remote,
|
|
|
|
bool secure = false,
|
|
|
|
Int64 priority = 1);
|
2014-02-07 15:11:57 +00:00
|
|
|
|
2020-01-07 10:26:16 +00:00
|
|
|
Cluster(const Cluster &)= delete;
|
2017-04-01 07:20:54 +00:00
|
|
|
Cluster & operator=(const Cluster &) = delete;
|
2013-12-07 16:51:29 +00:00
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// is used to set a limit on the size of the timeout
|
2021-04-29 16:11:20 +00:00
|
|
|
static Poco::Timespan saturate(Poco::Timespan v, Poco::Timespan limit);
|
2015-04-30 12:43:16 +00:00
|
|
|
|
|
|
|
public:
|
2021-03-07 19:07:25 +00:00
|
|
|
using SlotToShard = std::vector<UInt64>;
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
struct Address
|
|
|
|
{
|
|
|
|
/** In configuration file,
|
|
|
|
* addresses are located either in <node> elements:
|
|
|
|
* <node>
|
|
|
|
* <host>example01-01-1</host>
|
|
|
|
* <port>9000</port>
|
2020-06-27 06:52:10 +00:00
|
|
|
* <!-- <user>, <password>, <default_database>, <compression>, <priority>. <secure> if needed -->
|
2017-04-01 07:20:54 +00:00
|
|
|
* </node>
|
|
|
|
* ...
|
|
|
|
* or in <shard> and inside in <replica> elements:
|
|
|
|
* <shard>
|
|
|
|
* <replica>
|
|
|
|
* <host>example01-01-1</host>
|
|
|
|
* <port>9000</port>
|
2020-06-27 06:52:10 +00:00
|
|
|
* <!-- <user>, <password>, <default_database>, <compression>, <priority>. <secure> if needed -->
|
2017-04-01 07:20:54 +00:00
|
|
|
* </replica>
|
|
|
|
* </shard>
|
|
|
|
*/
|
2018-03-29 20:21:01 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String host_name;
|
|
|
|
UInt16 port;
|
|
|
|
String user;
|
|
|
|
String password;
|
2020-09-14 21:55:43 +00:00
|
|
|
|
|
|
|
/// For inter-server authorization
|
|
|
|
String cluster;
|
|
|
|
String cluster_secret;
|
|
|
|
|
2020-02-16 04:13:54 +00:00
|
|
|
UInt32 shard_index{}; /// shard serial number in configuration file, starting from 1.
|
|
|
|
UInt32 replica_index{}; /// replica serial number in this shard, starting from 1; zero means no replicas.
|
2020-01-31 09:22:30 +00:00
|
|
|
|
2018-03-29 20:21:01 +00:00
|
|
|
/// This database is selected when no database is specified for Distributed table
|
|
|
|
String default_database;
|
|
|
|
/// The locality is determined at the initialization, and is not changed even if DNS is changed
|
2019-07-08 01:43:41 +00:00
|
|
|
bool is_local = false;
|
2018-12-28 17:11:52 +00:00
|
|
|
bool user_specified = false;
|
|
|
|
|
2018-03-29 01:41:06 +00:00
|
|
|
Protocol::Compression compression = Protocol::Compression::Enable;
|
|
|
|
Protocol::Secure secure = Protocol::Secure::Disable;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-06-27 06:52:10 +00:00
|
|
|
Int64 priority = 1;
|
|
|
|
|
2017-07-26 19:31:32 +00:00
|
|
|
Address() = default;
|
2021-03-29 00:39:10 +00:00
|
|
|
|
2020-03-13 18:49:46 +00:00
|
|
|
Address(
|
|
|
|
const Poco::Util::AbstractConfiguration & config,
|
|
|
|
const String & config_prefix,
|
2020-09-14 21:55:43 +00:00
|
|
|
const String & cluster_,
|
|
|
|
const String & cluster_secret_,
|
2020-03-13 18:49:46 +00:00
|
|
|
UInt32 shard_index_ = 0,
|
|
|
|
UInt32 replica_index_ = 0);
|
2021-03-29 00:39:10 +00:00
|
|
|
|
2020-03-13 18:49:46 +00:00
|
|
|
Address(
|
|
|
|
const String & host_port_,
|
|
|
|
const String & user_,
|
|
|
|
const String & password_,
|
|
|
|
UInt16 clickhouse_port,
|
2021-07-23 11:16:35 +00:00
|
|
|
bool treat_local_port_as_remote,
|
2020-06-27 06:52:10 +00:00
|
|
|
bool secure_ = false,
|
2021-03-09 17:05:24 +00:00
|
|
|
Int64 priority_ = 1,
|
|
|
|
UInt32 shard_index_ = 0,
|
|
|
|
UInt32 replica_index_ = 0);
|
2020-02-12 14:32:18 +00:00
|
|
|
|
2017-07-28 16:14:49 +00:00
|
|
|
/// Returns 'escaped_host_name:port'
|
2017-04-13 16:12:56 +00:00
|
|
|
String toString() const;
|
2017-05-30 11:49:17 +00:00
|
|
|
|
2017-07-28 16:14:49 +00:00
|
|
|
/// Returns 'host_name:port'
|
|
|
|
String readableString() const;
|
|
|
|
|
2017-05-30 11:49:17 +00:00
|
|
|
static String toString(const String & host_name, UInt16 port);
|
|
|
|
|
2019-01-21 19:45:26 +00:00
|
|
|
static std::pair<String, UInt16> fromString(const String & host_port_string);
|
2017-07-26 19:31:32 +00:00
|
|
|
|
2020-03-13 18:49:46 +00:00
|
|
|
/// Returns escaped shard{shard_index}_replica{replica_index} or escaped
|
|
|
|
/// user:password@resolved_host_address:resolved_host_port#default_database
|
|
|
|
/// depending on use_compact_format flag
|
|
|
|
String toFullString(bool use_compact_format) const;
|
2020-01-31 10:49:10 +00:00
|
|
|
|
|
|
|
/// Returns address with only shard index and replica index or full address without shard index and replica index
|
2019-01-21 19:45:26 +00:00
|
|
|
static Address fromFullString(const String & address_full_string);
|
2018-12-02 02:17:08 +00:00
|
|
|
|
2019-07-08 01:43:41 +00:00
|
|
|
/// Returns resolved address if it does resolve.
|
|
|
|
std::optional<Poco::Net::SocketAddress> getResolvedAddress() const;
|
2018-03-29 20:21:01 +00:00
|
|
|
|
2019-01-21 19:45:26 +00:00
|
|
|
auto tuple() const { return std::tie(host_name, port, secure, user, password, default_database); }
|
|
|
|
bool operator==(const Address & other) const { return tuple() == other.tuple(); }
|
2018-12-02 02:17:08 +00:00
|
|
|
|
2018-03-29 20:21:01 +00:00
|
|
|
private:
|
2019-07-08 01:43:41 +00:00
|
|
|
bool isLocal(UInt16 clickhouse_port) const;
|
2017-04-01 07:20:54 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
using Addresses = std::vector<Address>;
|
|
|
|
using AddressesWithFailover = std::vector<Addresses>;
|
|
|
|
|
2020-11-07 23:44:35 +00:00
|
|
|
/// Name of directory for asynchronous write to StorageDistributed if has_internal_replication
|
|
|
|
///
|
|
|
|
/// Contains different path for permutations of:
|
|
|
|
/// - prefer_localhost_replica
|
|
|
|
/// Notes with prefer_localhost_replica==0 will contains local nodes.
|
|
|
|
/// - use_compact_format_in_distributed_parts_names
|
|
|
|
/// See toFullString()
|
|
|
|
///
|
|
|
|
/// This is cached to avoid looping by replicas in insertPathForInternalReplication().
|
|
|
|
struct ShardInfoInsertPathForInternalReplication
|
|
|
|
{
|
|
|
|
/// prefer_localhost_replica == 1 && use_compact_format_in_distributed_parts_names=0
|
|
|
|
std::string prefer_localhost_replica;
|
|
|
|
/// prefer_localhost_replica == 0 && use_compact_format_in_distributed_parts_names=0
|
|
|
|
std::string no_prefer_localhost_replica;
|
Drop replicas from dirname for internal_replication=true
Under use_compact_format_in_distributed_parts_names=1 and
internal_replication=true the server encodes all replicas for the
directory name for async INSERT into Distributed, and the directory name
looks like:
shard1_replica1,shard1_replica2,shard3_replica3
This is required for creating connections (to specific replicas only),
but in case of internal_replication=true, this can be avoided, since
this path will always includes all replicas.
This patch replaces all replicas with "_all_replicas" marker.
Note, that initial problem was that this path may overflow the NAME_MAX
if you will have more then 15 replicas, and the server will fail to
create the directory.
Also note, that changed directory name should not be a problem, since:
- empty directories will be removed since #16729
- and replicas encoded in the directory name is also supported anyway.
2021-06-20 13:50:01 +00:00
|
|
|
/// use_compact_format_in_distributed_parts_names=1
|
|
|
|
std::string compact;
|
2020-11-07 23:44:35 +00:00
|
|
|
};
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
struct ShardInfo
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
bool isLocal() const { return !local_addresses.empty(); }
|
2018-08-10 01:27:54 +00:00
|
|
|
bool hasRemoteConnections() const { return local_addresses.size() != per_replica_pools.size(); }
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t getLocalNodeCount() const { return local_addresses.size(); }
|
2017-05-30 11:49:17 +00:00
|
|
|
bool hasInternalReplication() const { return has_internal_replication; }
|
2020-05-14 00:02:28 +00:00
|
|
|
/// Name of directory for asynchronous write to StorageDistributed if has_internal_replication
|
2020-11-07 23:44:35 +00:00
|
|
|
const std::string & insertPathForInternalReplication(bool prefer_localhost_replica, bool use_compact_format) const;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
public:
|
2020-11-07 23:44:35 +00:00
|
|
|
ShardInfoInsertPathForInternalReplication insert_path_for_internal_replication;
|
2017-04-25 15:21:03 +00:00
|
|
|
/// Number of the shard, the indexation begins with 1
|
2019-04-22 16:07:09 +00:00
|
|
|
UInt32 shard_num = 0;
|
|
|
|
UInt32 weight = 1;
|
2017-04-01 07:20:54 +00:00
|
|
|
Addresses local_addresses;
|
2017-10-13 19:13:41 +00:00
|
|
|
/// nullptr if there are no remote addresses
|
2017-04-19 17:40:55 +00:00
|
|
|
ConnectionPoolWithFailoverPtr pool;
|
2018-02-14 15:11:39 +00:00
|
|
|
/// Connection pool for each replica, contains nullptr for local replicas
|
|
|
|
ConnectionPoolPtrs per_replica_pools;
|
2018-09-01 20:28:46 +00:00
|
|
|
bool has_internal_replication = false;
|
2017-04-01 07:20:54 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
using ShardsInfo = std::vector<ShardInfo>;
|
|
|
|
|
|
|
|
String getHashOfAddresses() const { return hash_of_addresses; }
|
|
|
|
const ShardsInfo & getShardsInfo() const { return shards_info; }
|
2017-08-11 15:02:07 +00:00
|
|
|
const AddressesWithFailover & getShardsAddresses() const { return addresses_with_failover; }
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
const ShardInfo & getAnyShardInfo() const
|
|
|
|
{
|
|
|
|
if (shards_info.empty())
|
|
|
|
throw Exception("Cluster is empty", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
return shards_info.front();
|
|
|
|
}
|
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// The number of remote shards.
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t getRemoteShardCount() const { return remote_shard_count; }
|
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// The number of clickhouse nodes located locally
|
|
|
|
/// we access the local nodes directly.
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t getLocalShardCount() const { return local_shard_count; }
|
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// The number of all shards.
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t getShardCount() const { return shards_info.size(); }
|
|
|
|
|
2020-09-14 21:55:43 +00:00
|
|
|
const String & getSecret() const { return secret; }
|
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// Get a subcluster consisting of one shard - index by count (from 0) of the shard of this cluster.
|
2017-04-01 07:20:54 +00:00
|
|
|
std::unique_ptr<Cluster> getClusterWithSingleShard(size_t index) const;
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2018-11-16 09:55:16 +00:00
|
|
|
/// Get a subcluster consisting of one or multiple shards - indexes by count (from 0) of the shard of this cluster.
|
2018-11-21 04:04:24 +00:00
|
|
|
std::unique_ptr<Cluster> getClusterWithMultipleShards(const std::vector<size_t> & indices) const;
|
2018-11-16 09:55:16 +00:00
|
|
|
|
2020-01-10 17:44:34 +00:00
|
|
|
/// Get a new Cluster that contains all servers (all shards with all replicas) from existing cluster as independent shards.
|
2020-01-07 10:26:16 +00:00
|
|
|
std::unique_ptr<Cluster> getClusterWithReplicasAsShards(const Settings & settings) const;
|
|
|
|
|
2020-09-23 18:28:59 +00:00
|
|
|
/// Returns false if cluster configuration doesn't allow to use it for cross-replication.
|
|
|
|
/// NOTE: true does not mean, that it's actually a cross-replication cluster.
|
|
|
|
bool maybeCrossReplication() const;
|
|
|
|
|
2016-05-13 03:22:16 +00:00
|
|
|
private:
|
2017-04-01 07:20:54 +00:00
|
|
|
SlotToShard slot_to_shard;
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2015-10-20 14:59:29 +00:00
|
|
|
public:
|
2017-04-01 07:20:54 +00:00
|
|
|
const SlotToShard & getSlotToShard() const { return slot_to_shard; }
|
2015-04-30 12:43:16 +00:00
|
|
|
|
|
|
|
private:
|
2017-04-01 07:20:54 +00:00
|
|
|
void initMisc();
|
2015-10-20 14:59:29 +00:00
|
|
|
|
2018-11-16 09:55:16 +00:00
|
|
|
/// For getClusterWithMultipleShards implementation.
|
2020-01-10 17:44:34 +00:00
|
|
|
struct SubclusterTag {};
|
|
|
|
Cluster(SubclusterTag, const Cluster & from, const std::vector<size_t> & indices);
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2020-01-07 18:19:39 +00:00
|
|
|
/// For getClusterWithReplicasAsShards implementation
|
2020-01-10 17:44:34 +00:00
|
|
|
struct ReplicasAsShardsTag {};
|
|
|
|
Cluster(ReplicasAsShardsTag, const Cluster & from, const Settings & settings);
|
2020-01-07 18:19:39 +00:00
|
|
|
|
2020-09-14 21:55:43 +00:00
|
|
|
/// Inter-server secret
|
|
|
|
String secret;
|
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
String hash_of_addresses;
|
2017-06-02 21:37:28 +00:00
|
|
|
/// Description of the cluster shards.
|
2017-04-01 07:20:54 +00:00
|
|
|
ShardsInfo shards_info;
|
2017-06-02 21:37:28 +00:00
|
|
|
/// Any remote shard.
|
2017-04-01 07:20:54 +00:00
|
|
|
ShardInfo * any_remote_shard_info = nullptr;
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// Non-empty is either addresses or addresses_with_failover.
|
|
|
|
/// The size and order of the elements in the corresponding array corresponds to shards_info.
|
2016-05-13 03:22:16 +00:00
|
|
|
|
2017-06-02 21:37:28 +00:00
|
|
|
/// An array of shards. For each shard, an array of replica addresses (servers that are considered identical).
|
2017-04-01 07:20:54 +00:00
|
|
|
AddressesWithFailover addresses_with_failover;
|
2013-12-07 16:51:29 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
size_t remote_shard_count = 0;
|
|
|
|
size_t local_shard_count = 0;
|
2021-07-01 10:18:29 +00:00
|
|
|
|
|
|
|
String name;
|
2013-12-07 16:51:29 +00:00
|
|
|
};
|
|
|
|
|
2016-10-10 08:44:52 +00:00
|
|
|
using ClusterPtr = std::shared_ptr<Cluster>;
|
|
|
|
|
2016-03-04 02:40:48 +00:00
|
|
|
|
2016-03-01 17:47:53 +00:00
|
|
|
class Clusters
|
2013-12-07 16:51:29 +00:00
|
|
|
{
|
2016-03-01 17:47:53 +00:00
|
|
|
public:
|
2020-09-14 21:55:43 +00:00
|
|
|
Clusters(const Poco::Util::AbstractConfiguration & config, const Settings & settings, const String & config_prefix = "remote_servers");
|
2016-03-01 17:47:53 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Clusters(const Clusters &) = delete;
|
|
|
|
Clusters & operator=(const Clusters &) = delete;
|
2016-03-01 17:47:53 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
ClusterPtr getCluster(const std::string & cluster_name) const;
|
2017-11-03 19:53:10 +00:00
|
|
|
void setCluster(const String & cluster_name, const ClusterPtr & cluster);
|
2016-10-10 08:44:52 +00:00
|
|
|
|
2021-03-12 21:17:19 +00:00
|
|
|
void updateClusters(const Poco::Util::AbstractConfiguration & new_config, const Settings & settings, const String & config_prefix, Poco::Util::AbstractConfiguration * old_config = nullptr);
|
2016-03-01 17:47:53 +00:00
|
|
|
|
|
|
|
public:
|
2017-04-01 07:20:54 +00:00
|
|
|
using Impl = std::map<String, ClusterPtr>;
|
2016-10-10 08:44:52 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
Impl getContainer() const;
|
2016-10-10 08:44:52 +00:00
|
|
|
|
|
|
|
protected:
|
2017-04-01 07:20:54 +00:00
|
|
|
Impl impl;
|
|
|
|
mutable std::mutex mutex;
|
2013-12-07 16:51:29 +00:00
|
|
|
};
|
2014-02-22 21:50:27 +00:00
|
|
|
|
2013-12-07 16:51:29 +00:00
|
|
|
}
|