2015-01-14 10:06:30 +00:00
|
|
|
#pragma once
|
|
|
|
|
2019-03-01 23:04:33 +00:00
|
|
|
#include <mutex>
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Common/Throttler.h>
|
|
|
|
#include <Client/Connection.h>
|
2017-04-19 17:40:55 +00:00
|
|
|
#include <Client/ConnectionPoolWithFailover.h>
|
2019-03-01 23:04:33 +00:00
|
|
|
#include <IO/ConnectionTimeouts.h>
|
2021-01-19 19:21:06 +00:00
|
|
|
#include <Client/IConnections.h>
|
2015-01-14 10:06:30 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
2015-02-06 22:32:54 +00:00
|
|
|
|
2015-01-14 10:06:30 +00:00
|
|
|
|
2017-08-02 13:05:01 +00:00
|
|
|
/** To retrieve data directly from multiple replicas (connections) from one shard
|
2017-03-09 00:56:38 +00:00
|
|
|
* within a single thread. As a degenerate case, it can also work with one connection.
|
|
|
|
* It is assumed that all functions except sendCancel are always executed in one thread.
|
2015-02-10 20:48:17 +00:00
|
|
|
*
|
2017-03-09 00:56:38 +00:00
|
|
|
* The interface is almost the same as Connection.
|
2015-02-10 20:48:17 +00:00
|
|
|
*/
|
2021-01-19 19:21:06 +00:00
|
|
|
class MultiplexedConnections final : public IConnections
|
2015-02-10 20:48:17 +00:00
|
|
|
{
|
|
|
|
public:
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Accepts ready connection.
|
2017-07-28 19:34:25 +00:00
|
|
|
MultiplexedConnections(Connection & connection, const Settings & settings_, const ThrottlerPtr & throttler_);
|
|
|
|
|
2018-11-28 14:33:40 +00:00
|
|
|
/// Accepts a vector of connections to replicas of one shard already taken from pool.
|
2017-07-28 19:34:25 +00:00
|
|
|
MultiplexedConnections(
|
2018-11-28 14:33:40 +00:00
|
|
|
std::vector<IConnectionPool::Entry> && connections,
|
|
|
|
const Settings & settings_, const ThrottlerPtr & throttler_);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-01-19 19:21:06 +00:00
|
|
|
void sendScalarsData(Scalars & data) override;
|
|
|
|
void sendExternalTablesData(std::vector<ExternalTablesData> & data) override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
void sendQuery(
|
2019-03-01 23:04:33 +00:00
|
|
|
const ConnectionTimeouts & timeouts,
|
2017-04-01 07:20:54 +00:00
|
|
|
const String & query,
|
2020-05-17 05:45:20 +00:00
|
|
|
const String & query_id,
|
|
|
|
UInt64 stage,
|
|
|
|
const ClientInfo & client_info,
|
2021-01-19 19:21:06 +00:00
|
|
|
bool with_pending_data) override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-04-13 10:59:02 +00:00
|
|
|
void sendReadTaskResponse(const String &) override;
|
2021-04-06 11:05:47 +00:00
|
|
|
|
2021-01-19 19:21:06 +00:00
|
|
|
Packet receivePacket() override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-01-19 19:21:06 +00:00
|
|
|
void disconnect() override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-01-19 19:21:06 +00:00
|
|
|
void sendCancel() override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2020-11-20 17:23:53 +00:00
|
|
|
/// Send parts' uuids to replicas to exclude them from query processing
|
2021-02-06 15:23:41 +00:00
|
|
|
void sendIgnoredPartUUIDs(const std::vector<UUID> & uuids) override;
|
2020-11-20 17:23:53 +00:00
|
|
|
|
2021-01-19 19:21:06 +00:00
|
|
|
Packet drain() override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2021-01-19 19:21:06 +00:00
|
|
|
std::string dumpAddresses() const override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Without locking, because sendCancel() does not change this number.
|
2021-01-19 19:21:06 +00:00
|
|
|
size_t size() const override { return replica_states.size(); }
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Without locking, because sendCancel() does not change the state of the replicas.
|
2021-01-19 19:21:06 +00:00
|
|
|
bool hasActiveConnections() const override { return active_connection_count > 0; }
|
2015-01-15 12:09:26 +00:00
|
|
|
|
2015-02-10 20:48:17 +00:00
|
|
|
private:
|
2021-07-14 13:17:30 +00:00
|
|
|
Packet receivePacketUnlocked(AsyncCallback async_callback, bool is_draining) override;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
2017-08-02 13:05:01 +00:00
|
|
|
/// Internal version of `dumpAddresses` function without locking.
|
|
|
|
std::string dumpAddressesUnlocked() const;
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Description of a single replica.
|
|
|
|
struct ReplicaState
|
|
|
|
{
|
2017-08-02 13:05:01 +00:00
|
|
|
Connection * connection = nullptr;
|
|
|
|
ConnectionPool::Entry pool_entry;
|
2017-04-01 07:20:54 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
/// Get a replica where you can read the data.
|
2021-07-14 13:17:30 +00:00
|
|
|
ReplicaState & getReplicaForReading(bool is_draining);
|
2015-01-14 10:06:30 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Mark the replica as invalid.
|
2017-08-02 13:05:01 +00:00
|
|
|
void invalidateReplica(ReplicaState & replica_state);
|
2015-02-05 22:31:03 +00:00
|
|
|
|
2015-02-25 17:15:31 +00:00
|
|
|
private:
|
2017-07-31 15:03:22 +00:00
|
|
|
const Settings & settings;
|
2015-11-06 17:44:01 +00:00
|
|
|
|
2021-07-14 13:17:30 +00:00
|
|
|
/// The following two fields are from settings but can be referenced outside the lifetime of
|
|
|
|
/// settings when connection is drained asynchronously.
|
|
|
|
Poco::Timespan drain_timeout;
|
|
|
|
Poco::Timespan receive_timeout;
|
|
|
|
|
2017-08-02 13:05:01 +00:00
|
|
|
/// The current number of valid connections to the replicas of this shard.
|
|
|
|
size_t active_connection_count = 0;
|
2015-02-06 10:41:03 +00:00
|
|
|
|
2017-08-02 13:05:01 +00:00
|
|
|
std::vector<ReplicaState> replica_states;
|
|
|
|
std::unordered_map<int, size_t> fd_to_replica_state_idx;
|
2015-02-06 14:46:04 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Connection that received last block.
|
2017-08-02 13:05:01 +00:00
|
|
|
Connection * current_connection = nullptr;
|
2015-10-12 14:53:16 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
bool sent_query = false;
|
|
|
|
bool cancelled = false;
|
2015-02-25 17:15:31 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
/// A mutex for the sendCancel function to execute safely
|
|
|
|
/// in separate thread.
|
|
|
|
mutable std::mutex cancel_mutex;
|
2020-12-04 13:35:24 +00:00
|
|
|
|
2021-01-15 20:07:25 +00:00
|
|
|
friend struct RemoteQueryExecutorRoutine;
|
2015-02-10 20:48:17 +00:00
|
|
|
};
|
2015-02-06 14:46:04 +00:00
|
|
|
|
2015-01-14 10:06:30 +00:00
|
|
|
}
|