2015-01-14 10:06:30 +00:00
|
|
|
#pragma once
|
|
|
|
|
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>
|
2016-05-28 10:15:36 +00:00
|
|
|
#include <mutex>
|
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
|
|
|
*/
|
2015-11-06 17:44:01 +00:00
|
|
|
class MultiplexedConnections final : private boost::noncopyable
|
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_);
|
|
|
|
|
|
|
|
/** Accepts a vector of connections to replicas of one shard already taken from pool.
|
|
|
|
* If the append_extra_info flag is set, additional information appended to each received block.
|
|
|
|
*/
|
|
|
|
MultiplexedConnections(
|
|
|
|
std::vector<IConnectionPool::Entry> && connections,
|
|
|
|
const Settings & settings_, const ThrottlerPtr & throttler_, bool append_extra_info);
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Send all content of external tables to replicas.
|
|
|
|
void sendExternalTablesData(std::vector<ExternalTablesData> & data);
|
|
|
|
|
|
|
|
/// Send request to replicas.
|
|
|
|
void sendQuery(
|
|
|
|
const String & query,
|
|
|
|
const String & query_id = "",
|
|
|
|
UInt64 stage = QueryProcessingStage::Complete,
|
|
|
|
const ClientInfo * client_info = nullptr,
|
|
|
|
bool with_pending_data = false);
|
|
|
|
|
2017-04-17 16:16:04 +00:00
|
|
|
/// Get packet from any replica.
|
2017-04-01 07:20:54 +00:00
|
|
|
Connection::Packet receivePacket();
|
|
|
|
|
2017-09-10 02:06:16 +00:00
|
|
|
/// Get information about the last received packet.
|
2017-04-01 07:20:54 +00:00
|
|
|
BlockExtraInfo getBlockExtraInfo() const;
|
|
|
|
|
|
|
|
/// Break all active connections.
|
|
|
|
void disconnect();
|
|
|
|
|
|
|
|
/// Send a request to the replica to cancel the request
|
|
|
|
void sendCancel();
|
|
|
|
|
|
|
|
/** On each replica, read and skip all packets to EndOfStream or Exception.
|
|
|
|
* Returns EndOfStream if no exception has been received. Otherwise
|
|
|
|
* returns the last received packet of type Exception.
|
|
|
|
*/
|
|
|
|
Connection::Packet drain();
|
|
|
|
|
|
|
|
/// Get the replica addresses as a string.
|
|
|
|
std::string dumpAddresses() const;
|
|
|
|
|
|
|
|
/// Returns the number of replicas.
|
|
|
|
/// Without locking, because sendCancel() does not change this number.
|
2017-08-02 13:05:01 +00:00
|
|
|
size_t size() const { return replica_states.size(); }
|
2017-04-01 07:20:54 +00:00
|
|
|
|
|
|
|
/// Check if there are any valid replicas.
|
|
|
|
/// Without locking, because sendCancel() does not change the state of the replicas.
|
2017-08-02 13:05:01 +00:00
|
|
|
bool hasActiveConnections() const { return active_connection_count > 0; }
|
2015-01-15 12:09:26 +00:00
|
|
|
|
2015-02-10 20:48:17 +00:00
|
|
|
private:
|
2017-08-02 13:05:01 +00:00
|
|
|
/// Internal version of `receivePacket` function without locking.
|
|
|
|
Connection::Packet receivePacketUnlocked();
|
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.
|
2017-08-02 13:05:01 +00:00
|
|
|
ReplicaState & getReplicaForReading();
|
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
|
|
|
|
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;
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Information about the last received block, if supported.
|
|
|
|
std::unique_ptr<BlockExtraInfo> block_extra_info;
|
2015-10-12 14:53:16 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
bool sent_query = false;
|
2017-04-17 16:16:04 +00:00
|
|
|
|
2017-04-01 07:20:54 +00:00
|
|
|
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;
|
2015-02-10 20:48:17 +00:00
|
|
|
};
|
2015-02-06 14:46:04 +00:00
|
|
|
|
2015-01-14 10:06:30 +00:00
|
|
|
}
|