mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-21 15:12:02 +00:00
Fixed tests
This commit is contained in:
parent
d55561422f
commit
04047f76c7
@ -100,7 +100,7 @@ void OwnSplitChannel::logSplit(const Poco::Message & msg)
|
||||
columns[i++]->insert(msg.getSource());
|
||||
columns[i++]->insert(msg.getText());
|
||||
|
||||
(void)(logs_queue->emplace(std::move(columns)));
|
||||
[[maybe_unused]] bool push_result = logs_queue->emplace(std::move(columns));
|
||||
}
|
||||
|
||||
/// Also log to system.text_log table, if message is not too noisy
|
||||
|
@ -34,6 +34,7 @@ ReplicatedAccessStorage::ReplicatedAccessStorage(
|
||||
: IAccessStorage(storage_name_)
|
||||
, zookeeper_path(zookeeper_path_)
|
||||
, get_zookeeper(get_zookeeper_)
|
||||
, refresh_queue(std::numeric_limits<size_t>::max())
|
||||
{
|
||||
if (zookeeper_path.empty())
|
||||
throw Exception("ZooKeeper path must be non-empty", ErrorCodes::BAD_ARGUMENTS);
|
||||
@ -366,7 +367,7 @@ void ReplicatedAccessStorage::refreshEntities(const zkutil::ZooKeeperPtr & zooke
|
||||
const String zookeeper_uuids_path = zookeeper_path + "/uuid";
|
||||
auto watch_entities_list = [this](const Coordination::WatchResponse &)
|
||||
{
|
||||
refresh_queue.push(UUIDHelpers::Nil);
|
||||
[[maybe_unused]] bool push_result = refresh_queue.push(UUIDHelpers::Nil);
|
||||
};
|
||||
Coordination::Stat stat;
|
||||
const auto entity_uuid_strs = zookeeper->getChildrenWatch(zookeeper_uuids_path, &stat, watch_entities_list);
|
||||
@ -418,7 +419,7 @@ void ReplicatedAccessStorage::refreshEntityNoLock(const zkutil::ZooKeeperPtr & z
|
||||
const auto watch_entity = [this, id](const Coordination::WatchResponse & response)
|
||||
{
|
||||
if (response.type == Coordination::Event::CHANGED)
|
||||
refresh_queue.push(id);
|
||||
[[maybe_unused]] bool push_result = refresh_queue.push(id);
|
||||
};
|
||||
Coordination::Stat entity_stat;
|
||||
const String entity_path = zookeeper_path + "/uuid/" + toString(id);
|
||||
|
@ -1,17 +1,20 @@
|
||||
#pragma once
|
||||
|
||||
#include <Access/IAccessStorage.h>
|
||||
#include <Common/ThreadPool.h>
|
||||
#include <Common/ZooKeeper/Common.h>
|
||||
#include <Common/ZooKeeper/ZooKeeper.h>
|
||||
#include <base/scope_guard.h>
|
||||
#include <Coordination/ThreadSafeQueue.h>
|
||||
#include <atomic>
|
||||
#include <list>
|
||||
#include <memory>
|
||||
#include <mutex>
|
||||
#include <unordered_map>
|
||||
|
||||
#include <base/scope_guard.h>
|
||||
|
||||
#include <Common/ThreadPool.h>
|
||||
#include <Common/ZooKeeper/Common.h>
|
||||
#include <Common/ZooKeeper/ZooKeeper.h>
|
||||
#include <Common/ConcurrentBoundedQueue.h>
|
||||
|
||||
#include <Access/IAccessStorage.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
{
|
||||
@ -36,7 +39,7 @@ private:
|
||||
std::atomic<bool> initialized = false;
|
||||
std::atomic<bool> stop_flag = false;
|
||||
ThreadFromGlobalPool worker_thread;
|
||||
ThreadSafeQueue<UUID> refresh_queue;
|
||||
ConcurrentBoundedQueue<UUID> refresh_queue;
|
||||
|
||||
UUID insertImpl(const AccessEntityPtr & entity, bool replace_if_exists) override;
|
||||
void removeImpl(const UUID & id) override;
|
||||
|
@ -179,6 +179,13 @@ public:
|
||||
return is_finished;
|
||||
}
|
||||
|
||||
/// Returns if queue is finished and empty
|
||||
bool isFinishedAndEmpty() const
|
||||
{
|
||||
std::lock_guard<std::mutex> lock(queue_mutex);
|
||||
return is_finished && queue.empty();
|
||||
}
|
||||
|
||||
/// Clear queue
|
||||
void clear()
|
||||
{
|
||||
|
@ -15,6 +15,7 @@ namespace ErrorCodes
|
||||
|
||||
KeeperDispatcher::KeeperDispatcher()
|
||||
: coordination_settings(std::make_shared<CoordinationSettings>())
|
||||
, responses_queue(std::numeric_limits<size_t>::max())
|
||||
, log(&Poco::Logger::get("KeeperDispatcher"))
|
||||
{
|
||||
}
|
||||
@ -414,7 +415,12 @@ void KeeperDispatcher::addErrorResponses(const KeeperStorage::RequestsForSession
|
||||
response->xid = request->xid;
|
||||
response->zxid = 0;
|
||||
response->error = error;
|
||||
responses_queue.push(DB::KeeperStorage::ResponseForSession{session_id, response});
|
||||
if (!responses_queue.push(DB::KeeperStorage::ResponseForSession{session_id, response}))
|
||||
throw Exception(ErrorCodes::LOGICAL_ERROR,
|
||||
"Could not push error response xid {} zxid {} error message {} to responses queue",
|
||||
response->xid,
|
||||
response->zxid,
|
||||
errorMessage(error));
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -120,7 +120,8 @@ nuraft::ptr<nuraft::buffer> KeeperStateMachine::commit(const uint64_t log_idx, n
|
||||
session_id = storage->getSessionID(session_id_request.session_timeout_ms);
|
||||
LOG_DEBUG(log, "Session ID response {} with timeout {}", session_id, session_id_request.session_timeout_ms);
|
||||
response->session_id = session_id;
|
||||
responses_queue.push(response_for_session);
|
||||
if (!responses_queue.push(response_for_session))
|
||||
LOG_WARNING(log, "Could not push response {} into responses queue", session_id);
|
||||
}
|
||||
}
|
||||
else
|
||||
@ -128,7 +129,8 @@ nuraft::ptr<nuraft::buffer> KeeperStateMachine::commit(const uint64_t log_idx, n
|
||||
std::lock_guard lock(storage_and_responses_lock);
|
||||
KeeperStorage::ResponsesForSessions responses_for_sessions = storage->processRequest(request_for_session.request, request_for_session.session_id, log_idx);
|
||||
for (auto & response_for_session : responses_for_sessions)
|
||||
responses_queue.push(response_for_session);
|
||||
if (!responses_queue.push(response_for_session))
|
||||
LOG_WARNING(log, "Could not push response {} into responses queue", response_for_session.session_id);
|
||||
}
|
||||
|
||||
last_committed_idx = log_idx;
|
||||
@ -305,7 +307,8 @@ void KeeperStateMachine::processReadRequest(const KeeperStorage::RequestForSessi
|
||||
std::lock_guard lock(storage_and_responses_lock);
|
||||
auto responses = storage->processRequest(request_for_session.request, request_for_session.session_id, std::nullopt);
|
||||
for (const auto & response : responses)
|
||||
responses_queue.push(response);
|
||||
if (!responses_queue.push(response))
|
||||
LOG_WARNING(log, "Could not push response {} into responses queue", response.session_id);
|
||||
}
|
||||
|
||||
std::vector<int64_t> KeeperStateMachine::getDeadSessions()
|
||||
|
@ -1,16 +1,17 @@
|
||||
#pragma once
|
||||
|
||||
#include <Common/ConcurrentBoundedQueue.h>
|
||||
#include <Coordination/KeeperStorage.h>
|
||||
#include <libnuraft/nuraft.hxx> // Y_IGNORE
|
||||
#include <base/logger_useful.h>
|
||||
#include <Coordination/ThreadSafeQueue.h>
|
||||
#include <Coordination/CoordinationSettings.h>
|
||||
#include <Coordination/KeeperSnapshotManager.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
using ResponsesQueue = ThreadSafeQueue<KeeperStorage::ResponseForSession>;
|
||||
using ResponsesQueue = ConcurrentBoundedQueue<KeeperStorage::ResponseForSession>;
|
||||
using SnapshotsQueue = ConcurrentBoundedQueue<CreateSnapshotTask>;
|
||||
|
||||
/// ClickHouse Keeper state machine. Wrapper for KeeperStorage.
|
||||
|
@ -1,78 +0,0 @@
|
||||
#pragma once
|
||||
|
||||
#include <queue>
|
||||
#include <mutex>
|
||||
|
||||
#include <base/MoveOrCopyIfThrow.h>
|
||||
|
||||
|
||||
namespace DB
|
||||
{
|
||||
|
||||
/// Queue with mutex and condvar. As simple as possible.
|
||||
template <typename T>
|
||||
class ThreadSafeQueue
|
||||
{
|
||||
private:
|
||||
mutable std::mutex queue_mutex;
|
||||
std::condition_variable cv;
|
||||
std::queue<T> queue;
|
||||
bool is_finished;
|
||||
public:
|
||||
|
||||
bool push(const T & response)
|
||||
{
|
||||
{
|
||||
std::lock_guard lock(queue_mutex);
|
||||
|
||||
if (is_finished)
|
||||
return false;
|
||||
|
||||
queue.push(response);
|
||||
}
|
||||
|
||||
cv.notify_one();
|
||||
return true;
|
||||
}
|
||||
|
||||
[[nodiscard]] bool tryPop(T & response, int64_t timeout_ms = 0)
|
||||
{
|
||||
std::unique_lock lock(queue_mutex);
|
||||
if (!cv.wait_for(lock,
|
||||
std::chrono::milliseconds(timeout_ms), [this] { return is_finished || !queue.empty(); }))
|
||||
return false;
|
||||
|
||||
if (is_finished && queue.empty())
|
||||
return false;
|
||||
|
||||
::detail::moveOrCopyIfThrow(std::move(queue.front()), response);
|
||||
queue.pop();
|
||||
|
||||
return true;
|
||||
}
|
||||
|
||||
size_t size() const
|
||||
{
|
||||
std::lock_guard lock(queue_mutex);
|
||||
return queue.size();
|
||||
}
|
||||
|
||||
bool isFinished() const
|
||||
{
|
||||
std::lock_guard lock(queue_mutex);
|
||||
return is_finished;
|
||||
}
|
||||
|
||||
bool finish()
|
||||
{
|
||||
std::lock_guard<std::mutex> lock(queue_mutex);
|
||||
bool was_finished_before = is_finished;
|
||||
is_finished = true;
|
||||
|
||||
cv.notify_all();
|
||||
|
||||
return was_finished_before;
|
||||
}
|
||||
};
|
||||
|
||||
}
|
@ -1159,7 +1159,7 @@ void testLogAndStateMachine(Coordination::CoordinationSettingsPtr settings, uint
|
||||
ChangelogDirTest snapshots("./snapshots");
|
||||
ChangelogDirTest logs("./logs");
|
||||
|
||||
ResponsesQueue queue;
|
||||
ResponsesQueue queue(std::numeric_limits<size_t>::max());
|
||||
SnapshotsQueue snapshots_queue{1};
|
||||
auto state_machine = std::make_shared<KeeperStateMachine>(queue, snapshots_queue, "./snapshots", settings);
|
||||
state_machine->init();
|
||||
@ -1310,7 +1310,7 @@ TEST_P(CoordinationTest, TestEphemeralNodeRemove)
|
||||
ChangelogDirTest snapshots("./snapshots");
|
||||
CoordinationSettingsPtr settings = std::make_shared<CoordinationSettings>();
|
||||
|
||||
ResponsesQueue queue;
|
||||
ResponsesQueue queue(std::numeric_limits<size_t>::max());
|
||||
SnapshotsQueue snapshots_queue{1};
|
||||
auto state_machine = std::make_shared<KeeperStateMachine>(queue, snapshots_queue, "./snapshots", settings);
|
||||
state_machine->init();
|
||||
|
@ -9,7 +9,7 @@ WriteBuffer LazyOutputFormat::out(nullptr, 0);
|
||||
|
||||
Chunk LazyOutputFormat::getChunk(UInt64 milliseconds)
|
||||
{
|
||||
if (queue.isFinished())
|
||||
if (isFinished())
|
||||
return {};
|
||||
|
||||
Chunk chunk;
|
||||
|
@ -23,7 +23,7 @@ public:
|
||||
Chunk getTotals();
|
||||
Chunk getExtremes();
|
||||
|
||||
bool isFinished() { return queue.isFinished(); }
|
||||
bool isFinished() { return queue.isFinishedAndEmpty(); }
|
||||
|
||||
BlockStreamProfileInfo & getProfileInfo() { return info; }
|
||||
|
||||
@ -31,7 +31,7 @@ public:
|
||||
|
||||
void onCancel() override
|
||||
{
|
||||
queue.finish();
|
||||
queue.clearAndFinish();
|
||||
}
|
||||
|
||||
void finalize() override
|
||||
@ -44,8 +44,7 @@ public:
|
||||
protected:
|
||||
void consume(Chunk chunk) override
|
||||
{
|
||||
if (!queue.isFinished())
|
||||
(void)(queue.emplace(std::move(chunk)));
|
||||
(void)(queue.emplace(std::move(chunk)));
|
||||
}
|
||||
|
||||
void consumeTotals(Chunk chunk) override { totals = std::move(chunk); }
|
||||
|
@ -198,7 +198,7 @@ KeeperTCPHandler::KeeperTCPHandler(IServer & server_, const Poco::Net::StreamSoc
|
||||
, operation_timeout(0, global_context->getConfigRef().getUInt("keeper_server.operation_timeout_ms", Coordination::DEFAULT_OPERATION_TIMEOUT_MS) * 1000)
|
||||
, session_timeout(0, global_context->getConfigRef().getUInt("keeper_server.session_timeout_ms", Coordination::DEFAULT_SESSION_TIMEOUT_MS) * 1000)
|
||||
, poll_wrapper(std::make_unique<SocketInterruptablePollWrapper>(socket_))
|
||||
, responses(std::make_unique<ThreadSafeResponseQueue>())
|
||||
, responses(std::make_unique<ThreadSafeResponseQueue>(std::numeric_limits<size_t>::max()))
|
||||
{
|
||||
}
|
||||
|
||||
@ -314,7 +314,7 @@ void KeeperTCPHandler::runImpl()
|
||||
auto response_fd = poll_wrapper->getResponseFD();
|
||||
auto response_callback = [this, response_fd] (const Coordination::ZooKeeperResponsePtr & response)
|
||||
{
|
||||
responses->push(response);
|
||||
[[maybe_unused]] bool push_result = responses->push(response);
|
||||
UInt8 single_byte = 1;
|
||||
[[maybe_unused]] int result = write(response_fd, &single_byte, sizeof(single_byte));
|
||||
};
|
||||
|
@ -13,10 +13,10 @@
|
||||
#include <Interpreters/Context.h>
|
||||
#include <Common/ZooKeeper/ZooKeeperCommon.h>
|
||||
#include <Common/ZooKeeper/ZooKeeperConstants.h>
|
||||
#include <Common/ConcurrentBoundedQueue.h>
|
||||
#include <Coordination/KeeperDispatcher.h>
|
||||
#include <IO/WriteBufferFromPocoSocket.h>
|
||||
#include <IO/ReadBufferFromPocoSocket.h>
|
||||
#include <Coordination/ThreadSafeQueue.h>
|
||||
#include <unordered_map>
|
||||
|
||||
namespace DB
|
||||
@ -25,7 +25,7 @@ namespace DB
|
||||
struct SocketInterruptablePollWrapper;
|
||||
using SocketInterruptablePollWrapperPtr = std::unique_ptr<SocketInterruptablePollWrapper>;
|
||||
|
||||
using ThreadSafeResponseQueue = ThreadSafeQueue<Coordination::ZooKeeperResponsePtr>;
|
||||
using ThreadSafeResponseQueue = ConcurrentBoundedQueue<Coordination::ZooKeeperResponsePtr>;
|
||||
|
||||
using ThreadSafeResponseQueuePtr = std::unique_ptr<ThreadSafeResponseQueue>;
|
||||
|
||||
|
@ -14,6 +14,11 @@
|
||||
namespace DB
|
||||
{
|
||||
|
||||
namespace ErrorCodes
|
||||
{
|
||||
extern const int LOGICAL_ERROR;
|
||||
}
|
||||
|
||||
ReadBufferFromRabbitMQConsumer::ReadBufferFromRabbitMQConsumer(
|
||||
ChannelPtr consumer_channel_,
|
||||
RabbitMQHandler & event_handler_,
|
||||
|
@ -21,6 +21,7 @@ static const auto RETURNED_LIMIT = 50000;
|
||||
namespace ErrorCodes
|
||||
{
|
||||
extern const int CANNOT_CONNECT_RABBITMQ;
|
||||
extern const int LOGICAL_ERROR;
|
||||
}
|
||||
|
||||
WriteBufferToRabbitMQProducer::WriteBufferToRabbitMQProducer(
|
||||
@ -181,7 +182,7 @@ void WriteBufferToRabbitMQProducer::removeRecord(UInt64 received_delivery_tag, b
|
||||
else
|
||||
{
|
||||
if (republish)
|
||||
if (returned.push(record_iter->second))
|
||||
if (!returned.push(record_iter->second))
|
||||
throw Exception(ErrorCodes::LOGICAL_ERROR, "Could not push to returned queue");
|
||||
|
||||
delivery_record.erase(record_iter);
|
||||
|
@ -59,7 +59,7 @@ int main(int argc, char *argv[])
|
||||
Poco::Logger::root().setLevel("trace");
|
||||
}
|
||||
auto * logger = &Poco::Logger::get("keeper-dumper");
|
||||
ResponsesQueue queue;
|
||||
ResponsesQueue queue(std::numeric_limits<size_t>::max());
|
||||
SnapshotsQueue snapshots_queue{1};
|
||||
CoordinationSettingsPtr settings = std::make_shared<CoordinationSettings>();
|
||||
auto state_machine = std::make_shared<KeeperStateMachine>(queue, snapshots_queue, argv[1], settings);
|
||||
|
Loading…
Reference in New Issue
Block a user