2021-02-06 00:54:27 +00:00
|
|
|
#pragma once
|
|
|
|
|
|
|
|
#if defined(OS_LINUX)
|
|
|
|
|
|
|
|
#include <Common/TimerDescriptor.h>
|
2021-02-17 17:34:52 +00:00
|
|
|
#include <Common/Epoll.h>
|
|
|
|
#include <Common/FiberStack.h>
|
|
|
|
#include <Common/Fiber.h>
|
|
|
|
#include <Client/ConnectionEstablisher.h>
|
2021-02-06 00:54:27 +00:00
|
|
|
#include <Client/ConnectionPoolWithFailover.h>
|
|
|
|
#include <Core/Settings.h>
|
|
|
|
#include <unordered_map>
|
|
|
|
#include <memory>
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
/** Class for establishing hedged connections with replicas.
|
|
|
|
* The process of establishing connection is divided on stages, on each stage if
|
|
|
|
* replica doesn't respond for a long time, we start establishing connection with
|
|
|
|
* the next replica, without cancelling working with previous one.
|
|
|
|
* It works with multiple replicas simultaneously without blocking by using epoll.
|
|
|
|
*/
|
|
|
|
class HedgedConnectionsFactory
|
|
|
|
{
|
|
|
|
public:
|
|
|
|
using ShuffledPool = ConnectionPoolWithFailover::Base::ShuffledPool;
|
2021-02-21 14:03:24 +00:00
|
|
|
using TryResult = PoolWithFailoverBase<IConnectionPool>::TryResult;
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-17 17:34:52 +00:00
|
|
|
enum class State
|
|
|
|
{
|
|
|
|
READY,
|
|
|
|
NOT_READY,
|
|
|
|
CANNOT_CHOOSE,
|
|
|
|
};
|
|
|
|
|
2021-02-15 13:21:36 +00:00
|
|
|
struct ReplicaStatus
|
|
|
|
{
|
2021-02-21 14:03:24 +00:00
|
|
|
explicit ReplicaStatus(ConnectionEstablisherAsync connection_stablisher_) : connection_establisher(std::move(connection_stablisher_))
|
2021-02-15 13:21:36 +00:00
|
|
|
{
|
|
|
|
}
|
|
|
|
|
2021-02-21 14:03:24 +00:00
|
|
|
ConnectionEstablisherAsync connection_establisher;
|
2021-02-15 13:21:36 +00:00
|
|
|
TimerDescriptor change_replica_timeout;
|
|
|
|
bool is_ready = false;
|
2021-02-06 00:54:27 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
HedgedConnectionsFactory(const ConnectionPoolWithFailoverPtr & pool_,
|
|
|
|
const Settings * settings_,
|
|
|
|
const ConnectionTimeouts & timeouts_,
|
|
|
|
std::shared_ptr<QualifiedTableName> table_to_check_ = nullptr);
|
|
|
|
|
|
|
|
/// Create and return active connections according to pool_mode.
|
|
|
|
std::vector<Connection *> getManyConnections(PoolMode pool_mode);
|
|
|
|
|
2021-02-26 15:53:40 +00:00
|
|
|
/// Try to get connection to the new replica without blocking. Process all current events in epoll (connections, timeouts),
|
|
|
|
/// Returned state might be READY (connection established successfully),
|
|
|
|
/// NOT_READY (there are no ready events now) and CANNOT_CHOOSE (cannot produce new connection anymore).
|
2021-02-06 00:54:27 +00:00
|
|
|
/// If state is READY, replica connection will be written in connection_out.
|
2021-02-26 15:53:40 +00:00
|
|
|
State waitForReadyConnections(Connection *& connection_out);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-21 14:03:24 +00:00
|
|
|
State startNewConnection(Connection *& connection_out);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
|
|
|
/// Stop working with all replicas that are not READY.
|
|
|
|
void stopChoosingReplicas();
|
|
|
|
|
2021-02-17 17:34:52 +00:00
|
|
|
bool hasEventsInProcess() const { return !epoll.empty(); }
|
2021-02-06 00:54:27 +00:00
|
|
|
|
|
|
|
int getFileDescriptor() const { return epoll.getFileDescriptor(); }
|
|
|
|
|
|
|
|
const ConnectionTimeouts & getConnectionTimeouts() const { return timeouts; }
|
|
|
|
|
2022-10-07 10:46:45 +00:00
|
|
|
size_t numberOfProcessingReplicas() const;
|
2021-02-21 14:03:24 +00:00
|
|
|
|
2021-02-26 15:53:40 +00:00
|
|
|
/// Tell Factory to not return connections with two level aggregation incompatibility.
|
|
|
|
void skipReplicasWithTwoLevelAggregationIncompatibility() { skip_replicas_with_two_level_aggregation_incompatibility = true; }
|
2021-02-21 14:03:24 +00:00
|
|
|
|
2021-02-06 00:54:27 +00:00
|
|
|
~HedgedConnectionsFactory();
|
|
|
|
|
|
|
|
private:
|
2021-02-26 15:53:40 +00:00
|
|
|
State waitForReadyConnectionsImpl(bool blocking, Connection *& connection_out);
|
|
|
|
|
2021-02-15 13:21:36 +00:00
|
|
|
/// Try to start establishing connection to the new replica. Return
|
|
|
|
/// the index of the new replica or -1 if cannot start new connection.
|
2021-02-21 14:03:24 +00:00
|
|
|
State startNewConnectionImpl(Connection *& connection_out);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
|
|
|
/// Find an index of the next free replica to start connection.
|
|
|
|
/// Return -1 if there is no free replica.
|
|
|
|
int getNextIndex();
|
|
|
|
|
|
|
|
int getReadyFileDescriptor(bool blocking);
|
|
|
|
|
2021-02-21 14:03:24 +00:00
|
|
|
void processFailedConnection(int index, const std::string & fail_message);
|
|
|
|
|
|
|
|
State resumeConnectionEstablisher(int index, Connection *& connection_out);
|
|
|
|
|
|
|
|
State processFinishedConnection(int index, TryResult result, Connection *& connection_out);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-21 14:03:24 +00:00
|
|
|
void removeReplicaFromEpoll(int index, int fd);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-21 14:03:24 +00:00
|
|
|
void addNewReplicaToEpoll(int index, int fd);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-09 02:13:47 +00:00
|
|
|
/// Return NOT_READY state if there is no ready events, READY if replica is ready
|
2021-02-15 13:21:36 +00:00
|
|
|
/// and CANNOT_CHOOSE if there is no more events in epoll.
|
2021-02-09 02:01:09 +00:00
|
|
|
State processEpollEvents(bool blocking, Connection *& connection_out);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-09 02:01:09 +00:00
|
|
|
State setBestUsableReplica(Connection *& connection_out);
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-26 15:53:40 +00:00
|
|
|
bool isTwoLevelAggregationIncompatible(Connection * connection);
|
|
|
|
|
2021-02-06 00:54:27 +00:00
|
|
|
const ConnectionPoolWithFailoverPtr pool;
|
|
|
|
const Settings * settings;
|
|
|
|
const ConnectionTimeouts timeouts;
|
|
|
|
|
|
|
|
std::vector<ShuffledPool> shuffled_pools;
|
2021-02-17 17:34:52 +00:00
|
|
|
std::vector<ReplicaStatus> replicas;
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-09 02:01:09 +00:00
|
|
|
/// Map socket file descriptor to replica index.
|
|
|
|
std::unordered_map<int, int> fd_to_replica_index;
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-17 17:34:52 +00:00
|
|
|
/// Map timeout for changing replica to replica index.
|
|
|
|
std::unordered_map<int, int> timeout_fd_to_replica_index;
|
|
|
|
|
2021-02-26 15:53:40 +00:00
|
|
|
/// If this flag is true, don't return connections with
|
|
|
|
/// two level aggregation incompatibility
|
|
|
|
bool skip_replicas_with_two_level_aggregation_incompatibility = false;
|
2021-02-06 00:54:27 +00:00
|
|
|
|
2021-02-17 17:34:52 +00:00
|
|
|
std::shared_ptr<QualifiedTableName> table_to_check;
|
2021-02-06 00:54:27 +00:00
|
|
|
int last_used_index = -1;
|
|
|
|
bool fallback_to_stale_replicas;
|
|
|
|
Epoll epoll;
|
|
|
|
Poco::Logger * log;
|
|
|
|
std::string fail_messages;
|
2021-02-26 15:53:40 +00:00
|
|
|
|
|
|
|
/// The maximum number of attempts to connect to replicas.
|
2021-02-06 00:54:27 +00:00
|
|
|
size_t max_tries;
|
2021-02-26 15:53:40 +00:00
|
|
|
/// Total number of established connections.
|
2021-02-21 14:03:24 +00:00
|
|
|
size_t entries_count = 0;
|
2021-02-26 15:53:40 +00:00
|
|
|
/// The number of established connections that are usable.
|
2021-02-21 14:03:24 +00:00
|
|
|
size_t usable_count = 0;
|
2021-02-26 15:53:40 +00:00
|
|
|
/// The number of established connections that are up to date.
|
2021-02-21 14:03:24 +00:00
|
|
|
size_t up_to_date_count = 0;
|
2021-02-26 15:53:40 +00:00
|
|
|
/// The number of failed connections (replica is considered failed after max_tries attempts to connect).
|
2021-02-21 14:03:24 +00:00
|
|
|
size_t failed_pools_count= 0;
|
2021-02-26 15:53:40 +00:00
|
|
|
|
|
|
|
/// The number of replicas that are in process of connection.
|
2021-02-21 14:03:24 +00:00
|
|
|
size_t replicas_in_process_count = 0;
|
2021-02-26 15:53:40 +00:00
|
|
|
/// The number of ready replicas (replica is considered ready when it's
|
|
|
|
/// connection returns outside).
|
2021-02-21 14:03:24 +00:00
|
|
|
size_t ready_replicas_count = 0;
|
2021-02-26 15:53:40 +00:00
|
|
|
|
|
|
|
/// The number of requested in startNewConnection replicas (it's needed for
|
|
|
|
/// checking the number of requested replicas that are still in process).
|
|
|
|
size_t requested_connections_count = 0;
|
2021-02-06 00:54:27 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|
|
|
|
#endif
|