2022-10-28 16:41:10 +00:00
|
|
|
#pragma once
|
|
|
|
|
|
|
|
#pragma once
|
|
|
|
|
|
|
|
#include <Processors/Sinks/SinkToStorage.h>
|
|
|
|
#include <Interpreters/Context.h>
|
|
|
|
#include <Core/BackgroundSchedulePool.h>
|
|
|
|
|
2023-02-13 12:15:58 +00:00
|
|
|
namespace Poco { class Logger; }
|
|
|
|
|
2022-10-28 16:41:10 +00:00
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
/// Interface for producing messages in streaming storages.
|
|
|
|
/// It's used in MessageQueueSink.
|
|
|
|
class IMessageProducer
|
|
|
|
{
|
|
|
|
public:
|
2024-01-23 17:04:50 +00:00
|
|
|
explicit IMessageProducer(LoggerPtr log_);
|
2023-02-13 12:15:58 +00:00
|
|
|
|
2022-10-28 16:41:10 +00:00
|
|
|
/// Do some preparations.
|
|
|
|
virtual void start(const ContextPtr & context) = 0;
|
|
|
|
|
|
|
|
/// Produce single message.
|
|
|
|
virtual void produce(const String & message, size_t rows_in_message, const Columns & columns, size_t last_row) = 0;
|
|
|
|
|
|
|
|
/// Finalize producer.
|
|
|
|
virtual void finish() = 0;
|
|
|
|
|
|
|
|
virtual ~IMessageProducer() = default;
|
2023-02-13 12:15:58 +00:00
|
|
|
|
|
|
|
protected:
|
2024-01-23 17:04:50 +00:00
|
|
|
LoggerPtr log;
|
2022-10-28 16:41:10 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
/// Implements interface for concurrent message producing.
|
2022-12-15 19:47:10 +00:00
|
|
|
class AsynchronousMessageProducer : public IMessageProducer
|
2022-10-28 16:41:10 +00:00
|
|
|
{
|
|
|
|
public:
|
2024-01-23 17:04:50 +00:00
|
|
|
explicit AsynchronousMessageProducer(LoggerPtr log_) : IMessageProducer(log_) {}
|
2023-02-13 12:15:58 +00:00
|
|
|
|
2022-10-28 16:41:10 +00:00
|
|
|
/// Create and schedule task in BackgroundSchedulePool that will produce messages.
|
|
|
|
void start(const ContextPtr & context) override;
|
|
|
|
|
|
|
|
/// Stop producing task, wait for ot to finish and finalize.
|
|
|
|
void finish() override;
|
|
|
|
|
|
|
|
/// In this method producer should not do any hard work and send message
|
|
|
|
/// to producing task, for example, by using ConcurrentBoundedQueue.
|
|
|
|
void produce(const String & message, size_t rows_in_message, const Columns & columns, size_t last_row) override = 0;
|
|
|
|
|
|
|
|
protected:
|
|
|
|
/// Do some initialization before scheduling producing task.
|
|
|
|
virtual void initialize() {}
|
|
|
|
/// Tell producer to finish all work and stop producing task
|
|
|
|
virtual void stopProducingTask() = 0;
|
|
|
|
/// Do some finalization after producing task is stopped.
|
|
|
|
virtual void finishImpl() {}
|
|
|
|
|
|
|
|
virtual String getProducingTaskName() const = 0;
|
2022-10-28 17:09:08 +00:00
|
|
|
/// Method that is called inside producing task, all producing work should be done here.
|
2022-12-15 19:47:10 +00:00
|
|
|
virtual void startProducingTaskLoop() = 0;
|
2022-10-28 16:41:10 +00:00
|
|
|
|
|
|
|
private:
|
|
|
|
/// Flag, indicated that finish() method was called.
|
|
|
|
/// It's used to prevent doing finish logic more than once.
|
|
|
|
std::atomic<bool> finished = false;
|
|
|
|
|
|
|
|
BackgroundSchedulePool::TaskHolder producing_task;
|
2023-02-13 12:15:58 +00:00
|
|
|
|
|
|
|
std::atomic<bool> scheduled;
|
2022-10-28 16:41:10 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
|
|
|
|
}
|