ClickHouse/src/DataStreams/RemoteQueryExecutorReadContext.h

210 lines
5.5 KiB
C++
Raw Normal View History

2020-12-04 11:00:32 +00:00
#pragma once
#include <sys/epoll.h>
2020-12-10 09:30:43 +00:00
#include <Common/Fiber.h>
#include <Common/FiberStack.h>
#include <Common/TimerDescriptor.h>
2020-12-04 11:00:32 +00:00
namespace DB
{
namespace ErrorCodes
{
extern const int CANNOT_READ_FROM_SOCKET;
extern const int CANNOT_OPEN_FILE;
extern const int SOCKET_TIMEOUT;
}
class RemoteQueryExecutorReadContext
{
public:
using Self = RemoteQueryExecutorReadContext;
bool is_read_in_progress = false;
Packet packet;
std::exception_ptr exception;
FiberStack<> stack;
2020-12-04 13:35:24 +00:00
std::mutex fiber_mutex;
2020-12-04 11:00:32 +00:00
boost::context::fiber fiber;
Poco::Timespan receive_timeout;
MultiplexedConnections & connections;
TimerDescriptor timer{CLOCK_MONOTONIC, 0};
2020-12-04 13:35:24 +00:00
int socket_fd = -1;
2020-12-04 11:00:32 +00:00
int epoll_fd;
explicit RemoteQueryExecutorReadContext(MultiplexedConnections & connections_) : connections(connections_)
{
2020-12-04 13:35:24 +00:00
epoll_fd = epoll_create(2);
if (-1 == epoll_fd)
2020-12-04 11:00:32 +00:00
throwFromErrno("Cannot create epoll descriptor", ErrorCodes::CANNOT_OPEN_FILE);
{
epoll_event timer_event;
timer_event.events = EPOLLIN | EPOLLPRI;
timer_event.data.fd = timer.getDescriptor();
if (-1 == epoll_ctl(epoll_fd, EPOLL_CTL_ADD, timer_event.data.fd, &timer_event))
throwFromErrno("Cannot add timer descriptor to epoll", ErrorCodes::CANNOT_OPEN_FILE);
}
fiber = boost::context::fiber(std::allocator_arg_t(), stack, Routine{connections, *this});
}
2020-12-04 13:35:24 +00:00
void setSocket(Poco::Net::Socket & socket)
2020-12-04 11:00:32 +00:00
{
2020-12-04 13:35:24 +00:00
int fd = socket.impl()->sockfd();
if (fd == socket_fd)
return;
epoll_event socket_event;
socket_event.events = EPOLLIN | EPOLLPRI;
socket_event.data.fd = fd;
if (socket_fd != -1)
2020-12-04 11:00:32 +00:00
{
2020-12-04 13:35:24 +00:00
if (-1 == epoll_ctl(epoll_fd, EPOLL_CTL_DEL, socket_fd, &socket_event))
throwFromErrno("Cannot remove socket descriptor to epoll", ErrorCodes::CANNOT_OPEN_FILE);
2020-12-04 11:00:32 +00:00
}
2020-12-04 13:35:24 +00:00
socket_fd = fd;
if (-1 == epoll_ctl(epoll_fd, EPOLL_CTL_ADD, socket_fd, &socket_event))
throwFromErrno("Cannot add socket descriptor to epoll", ErrorCodes::CANNOT_OPEN_FILE);
receive_timeout = socket.impl()->getReceiveTimeout();
2020-12-04 11:00:32 +00:00
}
void checkTimeout() const
{
try
{
checkTimeoutImpl();
}
catch (DB::Exception & e)
{
e.addMessage(" while reading from socket ({})", connections.getSocket().peerAddress().toString());
2020-12-04 13:35:24 +00:00
throw;
2020-12-04 11:00:32 +00:00
}
}
void checkTimeoutImpl() const
{
epoll_event events[2];
/// Wait for epoll_fd will not block if it was polled externally.
int num_events = epoll_wait(epoll_fd, events, 2, 0);
if (num_events == -1)
throwFromErrno("Failed to epoll_wait", ErrorCodes::CANNOT_READ_FROM_SOCKET);
bool is_socket_ready = false;
bool has_timer_alarm = false;
for (int i = 0; i < num_events; ++i)
{
if (events[i].data.fd == socket_fd)
is_socket_ready = true;
if (events[i].data.fd == timer.getDescriptor())
has_timer_alarm = true;
}
if (has_timer_alarm && !is_socket_ready)
{
/// Socket receive timeout. Drain it in case or error, or it may be hide by timeout exception.
timer.drain();
throw NetException("Timeout exceeded", ErrorCodes::SOCKET_TIMEOUT);
}
}
void setTimer() const
{
/// Did not get packet yet. Init timeout for the next async reading.
timer.reset();
if (receive_timeout.totalMicroseconds())
timer.setRelative(receive_timeout);
}
2020-12-04 13:35:24 +00:00
bool resumeRoutine()
2020-12-04 11:00:32 +00:00
{
2020-12-04 13:35:24 +00:00
if (is_read_in_progress)
checkTimeout();
{
std::lock_guard guard(fiber_mutex);
if (!fiber)
return false;
fiber = std::move(fiber).resume();
}
2020-12-04 11:00:32 +00:00
if (exception)
std::rethrow_exception(std::move(exception));
2020-12-04 13:35:24 +00:00
if (is_read_in_progress)
{
auto & socket = connections.getSocket();
try
{
setSocket(socket);
}
catch (DB::Exception & e)
{
e.addMessage(" while reading from socket ({})", socket.peerAddress().toString());
throw;
}
}
return true;
}
void cancel()
{
std::lock_guard guard(fiber_mutex);
boost::context::fiber to_destroy = std::move(fiber);
2020-12-04 11:00:32 +00:00
}
~RemoteQueryExecutorReadContext()
{
/// socket_fd is closed by Poco::Net::Socket
/// timer_fd is closed by TimerDescriptor
close(epoll_fd);
}
struct Routine
{
MultiplexedConnections & connections;
Self & read_context;
2020-12-10 09:30:43 +00:00
Fiber operator()(Fiber && sink) const
2020-12-04 11:00:32 +00:00
{
try
{
while (true)
{
connections.setFiber(&sink);
2020-12-04 13:35:24 +00:00
read_context.is_read_in_progress = true;
2020-12-04 11:00:32 +00:00
read_context.packet = connections.receivePacket();
read_context.is_read_in_progress = false;
sink = std::move(sink).resume();
}
}
2020-12-04 13:35:24 +00:00
catch (const boost::context::detail::forced_unwind &)
{
throw;
}
2020-12-04 11:00:32 +00:00
catch (...)
{
read_context.exception = std::current_exception();
}
return std::move(sink);
}
};
};
}