ClickHouse/src/Client/ConnectionEstablisher.cpp

Ignoring revisions in .git-blame-ignore-revs. Click here to bypass and see the normal blame view.

236 lines
7.4 KiB
C++
Raw Normal View History

2021-02-17 17:34:52 +00:00
#include <Client/ConnectionEstablisher.h>
#include <Common/quoteString.h>
#include <Common/ProfileEvents.h>
namespace ProfileEvents
{
2023-05-22 18:23:39 +00:00
extern const Event DistributedConnectionTries;
extern const Event DistributedConnectionUsable;
2021-02-17 17:34:52 +00:00
extern const Event DistributedConnectionMissingTable;
extern const Event DistributedConnectionStaleReplica;
}
namespace DB
{
namespace ErrorCodes
{
extern const int ATTEMPT_TO_READ_AFTER_EOF;
extern const int DNS_ERROR;
2021-02-17 17:34:52 +00:00
extern const int NETWORK_ERROR;
extern const int SOCKET_TIMEOUT;
}
ConnectionEstablisher::ConnectionEstablisher(
IConnectionPool * pool_,
const ConnectionTimeouts * timeouts_,
const Settings * settings_,
2021-02-21 14:03:24 +00:00
Poco::Logger * log_,
2021-02-21 21:59:07 +00:00
const QualifiedTableName * table_to_check_)
: pool(pool_), timeouts(timeouts_), settings(settings_), log(log_), table_to_check(table_to_check_), is_finished(false)
2021-02-21 14:03:24 +00:00
{
}
void ConnectionEstablisher::run(ConnectionEstablisher::TryResult & result, std::string & fail_message)
{
is_finished = false;
SCOPE_EXIT(is_finished = true);
try
{
2023-05-22 18:23:39 +00:00
ProfileEvents::increment(ProfileEvents::DistributedConnectionTries);
2021-02-21 14:03:24 +00:00
result.entry = pool->get(*timeouts, settings, /* force_connected = */ false);
AsyncCallbackSetter async_setter(&*result.entry, std::move(async_callback));
UInt64 server_revision = 0;
if (table_to_check)
server_revision = result.entry->getServerRevision(*timeouts);
if (!table_to_check || server_revision < DBMS_MIN_REVISION_WITH_TABLES_STATUS)
{
result.entry->forceConnected(*timeouts);
2023-05-22 18:23:39 +00:00
ProfileEvents::increment(ProfileEvents::DistributedConnectionUsable);
2021-02-21 14:03:24 +00:00
result.is_usable = true;
result.is_up_to_date = true;
return;
}
/// Only status of the remote table corresponding to the Distributed table is taken into account.
/// TODO: request status for joined tables also.
TablesStatusRequest status_request;
status_request.tables.emplace(*table_to_check);
TablesStatusResponse status_response = result.entry->getTablesStatus(*timeouts, status_request);
auto table_status_it = status_response.table_states_by_id.find(*table_to_check);
if (table_status_it == status_response.table_states_by_id.end())
{
2023-01-16 23:11:59 +00:00
LOG_WARNING(LogToStr(fail_message, log), "There is no table {}.{} on server: {}",
backQuote(table_to_check->database), backQuote(table_to_check->table), result.entry->getDescription());
2021-02-21 14:03:24 +00:00
ProfileEvents::increment(ProfileEvents::DistributedConnectionMissingTable);
return;
}
2023-05-22 18:23:39 +00:00
ProfileEvents::increment(ProfileEvents::DistributedConnectionUsable);
2021-02-21 14:03:24 +00:00
result.is_usable = true;
UInt64 max_allowed_delay = settings ? UInt64(settings->max_replica_delay_for_distributed_queries) : 0;
if (!max_allowed_delay)
{
result.is_up_to_date = true;
return;
}
UInt32 delay = table_status_it->second.absolute_delay;
if (delay < max_allowed_delay)
result.is_up_to_date = true;
else
{
result.is_up_to_date = false;
result.staleness = delay;
LOG_TRACE(log, "Server {} has unacceptable replica delay for table {}.{}: {}", result.entry->getDescription(), table_to_check->database, table_to_check->table, delay);
ProfileEvents::increment(ProfileEvents::DistributedConnectionStaleReplica);
}
}
catch (const Exception & e)
{
if (e.code() != ErrorCodes::NETWORK_ERROR && e.code() != ErrorCodes::SOCKET_TIMEOUT
&& e.code() != ErrorCodes::ATTEMPT_TO_READ_AFTER_EOF && e.code() != ErrorCodes::DNS_ERROR)
2021-02-21 14:03:24 +00:00
throw;
fail_message = getCurrentExceptionMessage(/* with_stacktrace = */ false);
if (!result.entry.isNull())
{
result.entry->disconnect();
result.reset();
}
}
}
2021-02-22 07:52:19 +00:00
#if defined(OS_LINUX)
2021-02-21 14:03:24 +00:00
ConnectionEstablisherAsync::ConnectionEstablisherAsync(
IConnectionPool * pool_,
const ConnectionTimeouts * timeouts_,
const Settings * settings_,
Poco::Logger * log_,
2021-02-17 17:34:52 +00:00
const QualifiedTableName * table_to_check_)
2023-03-03 19:30:43 +00:00
: AsyncTaskExecutor(std::make_unique<Task>(*this)), connection_establisher(pool_, timeouts_, settings_, log_, table_to_check_)
2021-02-17 17:34:52 +00:00
{
2023-03-03 19:30:43 +00:00
epoll.add(timeout_descriptor.getDescriptor());
2021-02-17 17:34:52 +00:00
}
2023-05-22 18:22:05 +00:00
void ConnectionEstablisherAsync::Task::run(AsyncCallback async_callback, SuspendCallback)
2021-02-17 17:34:52 +00:00
{
2023-03-03 19:30:43 +00:00
connection_establisher_async.reset();
connection_establisher_async.connection_establisher.setAsyncCallback(async_callback);
connection_establisher_async.connection_establisher.run(connection_establisher_async.result, connection_establisher_async.fail_message);
connection_establisher_async.is_finished = true;
2021-02-17 17:34:52 +00:00
}
2023-03-03 19:30:43 +00:00
void ConnectionEstablisherAsync::processAsyncEvent(int fd, Poco::Timespan socket_timeout, AsyncEventTimeoutType type, const std::string & description, uint32_t events)
2021-02-17 17:34:52 +00:00
{
2023-03-03 19:30:43 +00:00
socket_fd = fd;
socket_description = description;
epoll.add(fd, events);
timeout_descriptor.setRelative(socket_timeout);
timeout = socket_timeout;
timeout_type = type;
2021-02-17 17:34:52 +00:00
}
2023-03-03 19:30:43 +00:00
void ConnectionEstablisherAsync::clearAsyncEvent()
2021-02-17 17:34:52 +00:00
{
2023-03-03 19:30:43 +00:00
timeout_descriptor.reset();
epoll.remove(socket_fd);
}
2021-02-21 14:03:24 +00:00
2023-03-03 19:30:43 +00:00
bool ConnectionEstablisherAsync::checkBeforeTaskResume()
{
/// If we just restarted the task, no need to check timeout.
if (restarted)
2021-02-21 14:03:24 +00:00
{
2023-03-03 19:30:43 +00:00
restarted = false;
return true;
2021-02-17 17:34:52 +00:00
}
2023-03-03 19:30:43 +00:00
return checkTimeout();
2021-02-21 14:03:24 +00:00
}
2023-03-28 16:00:00 +00:00
void ConnectionEstablisherAsync::cancelAfter()
{
if (!is_finished)
reset();
}
2023-03-03 19:30:43 +00:00
bool ConnectionEstablisherAsync::checkTimeout()
2021-02-21 14:03:24 +00:00
{
2021-02-17 17:34:52 +00:00
bool is_socket_ready = false;
2023-03-03 19:30:43 +00:00
bool is_timeout_alarmed = false;
2021-02-17 17:34:52 +00:00
epoll_event events[2];
2021-02-18 11:21:48 +00:00
events[0].data.fd = events[1].data.fd = -1;
2021-02-21 14:03:24 +00:00
size_t ready_count = epoll.getManyReady(2, events, false);
2021-02-17 17:34:52 +00:00
for (size_t i = 0; i != ready_count; ++i)
{
if (events[i].data.fd == socket_fd)
is_socket_ready = true;
2023-03-03 19:30:43 +00:00
if (events[i].data.fd == timeout_descriptor.getDescriptor())
is_timeout_alarmed = true;
2021-02-17 17:34:52 +00:00
}
if (is_timeout_alarmed && !is_socket_ready && !haveMoreAddressesToConnect())
2021-02-21 14:03:24 +00:00
{
2023-03-03 19:30:43 +00:00
/// In not async case timeout exception would be thrown and caught in ConnectionEstablisher::run,
2021-02-21 14:03:24 +00:00
/// but in async case we process timeout outside and cannot throw exception. So, we just save fail message.
2023-03-03 19:30:43 +00:00
fail_message = getSocketTimeoutExceededMessageByTimeoutType(timeout_type, timeout, socket_description);
2021-02-21 14:03:24 +00:00
epoll.remove(socket_fd);
2023-03-03 19:30:43 +00:00
/// Restart task, so the connection process will start from the beginning in the next resume().
restart();
/// The result should be Null in case of timeout.
2021-02-21 14:03:24 +00:00
resetResult();
2023-03-03 19:30:43 +00:00
restarted = true;
/// Mark that current connection process is finished.
is_finished = true;
2021-02-21 14:03:24 +00:00
return false;
}
2021-02-17 17:34:52 +00:00
2021-02-21 14:03:24 +00:00
return true;
2021-02-17 17:34:52 +00:00
}
2023-03-03 19:30:43 +00:00
void ConnectionEstablisherAsync::afterTaskResume()
2021-02-17 17:34:52 +00:00
{
2023-03-03 19:30:43 +00:00
if (is_finished)
{
restart();
restarted = true;
}
2021-02-17 17:34:52 +00:00
}
2021-02-21 14:03:24 +00:00
void ConnectionEstablisherAsync::reset()
2021-02-17 17:34:52 +00:00
{
resetResult();
2021-02-21 14:03:24 +00:00
fail_message.clear();
socket_fd = -1;
2023-03-03 19:30:43 +00:00
is_finished = false;
2021-02-17 17:34:52 +00:00
}
2021-02-21 14:03:24 +00:00
void ConnectionEstablisherAsync::resetResult()
2021-02-17 17:34:52 +00:00
{
if (!result.entry.isNull())
{
result.entry->disconnect();
result.reset();
}
}
bool ConnectionEstablisherAsync::haveMoreAddressesToConnect()
{
return !result.entry.isNull() && result.entry->haveMoreAddressesToConnect();
}
2021-02-22 07:52:19 +00:00
#endif
2021-02-17 17:34:52 +00:00
}