2017-10-03 23:39:24 +00:00
|
|
|
#pragma once
|
2017-10-14 00:53:01 +00:00
|
|
|
#include <Common/config.h>
|
|
|
|
#if USE_RDKAFKA
|
2017-10-03 23:39:24 +00:00
|
|
|
|
|
|
|
#include <mutex>
|
|
|
|
|
|
|
|
#include <ext/shared_ptr_helper.h>
|
|
|
|
#include <Core/NamesAndTypes.h>
|
|
|
|
#include <Storages/IStorage.h>
|
|
|
|
#include <DataStreams/IBlockOutputStream.h>
|
|
|
|
#include <Poco/Event.h>
|
2017-12-16 01:45:11 +00:00
|
|
|
#include <Poco/Semaphore.h>
|
2017-10-03 23:39:24 +00:00
|
|
|
|
|
|
|
struct rd_kafka_s;
|
|
|
|
struct rd_kafka_conf_s;
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
|
|
|
class StorageKafka;
|
|
|
|
|
|
|
|
/** Implements a Kafka queue table engine that can be used as a persistent queue / buffer,
|
|
|
|
* or as a basic building block for creating pipelines with a continuous insertion / ETL.
|
|
|
|
*/
|
|
|
|
class StorageKafka : public ext::shared_ptr_helper<StorageKafka>, public IStorage
|
|
|
|
{
|
|
|
|
friend class KafkaBlockInputStream;
|
|
|
|
friend class KafkaBlockOutputStream;
|
|
|
|
|
|
|
|
public:
|
|
|
|
std::string getName() const override { return "Kafka"; }
|
|
|
|
std::string getTableName() const override { return table_name; }
|
|
|
|
std::string getDatabaseName() const { return database_name; }
|
|
|
|
|
2017-12-25 21:32:33 +00:00
|
|
|
const NamesAndTypes & getColumnsListImpl() const override { return columns; }
|
2017-10-03 23:39:24 +00:00
|
|
|
|
|
|
|
void startup() override;
|
|
|
|
void shutdown() override;
|
|
|
|
|
|
|
|
BlockInputStreams read(
|
|
|
|
const Names & column_names,
|
|
|
|
const SelectQueryInfo & query_info,
|
|
|
|
const Context & context,
|
|
|
|
QueryProcessingStage::Enum & processed_stage,
|
|
|
|
size_t max_block_size,
|
|
|
|
unsigned num_streams) override;
|
|
|
|
|
2017-12-01 21:13:25 +00:00
|
|
|
void rename(const String & /*new_path_to_db*/, const String & new_database_name, const String & new_table_name) override
|
2017-10-03 23:39:24 +00:00
|
|
|
{
|
|
|
|
table_name = new_table_name;
|
|
|
|
database_name = new_database_name;
|
|
|
|
}
|
|
|
|
|
|
|
|
void updateDependencies() override;
|
|
|
|
|
|
|
|
private:
|
2017-12-16 01:45:11 +00:00
|
|
|
/// Each engine typically has one consumer (able to process 1..N partitions)
|
2017-12-19 18:34:48 +00:00
|
|
|
/// It is however possible to create multiple consumers per table, as long
|
|
|
|
/// as the total number of consumers is <= number of partitions.
|
2017-12-16 01:45:11 +00:00
|
|
|
struct Consumer
|
|
|
|
{
|
|
|
|
Consumer(struct rd_kafka_conf_s * conf);
|
|
|
|
~Consumer();
|
|
|
|
|
|
|
|
void subscribe(const Names & topics);
|
|
|
|
void unsubscribe();
|
|
|
|
|
|
|
|
struct rd_kafka_s * stream = nullptr;
|
|
|
|
};
|
|
|
|
using ConsumerPtr = std::shared_ptr<Consumer>;
|
|
|
|
|
|
|
|
// Configuration and state
|
2017-10-03 23:39:24 +00:00
|
|
|
String table_name;
|
|
|
|
String database_name;
|
|
|
|
Context & context;
|
2017-12-25 21:32:33 +00:00
|
|
|
NamesAndTypes columns;
|
2017-10-03 23:39:24 +00:00
|
|
|
Names topics;
|
2017-12-16 01:45:11 +00:00
|
|
|
const String brokers;
|
|
|
|
const String group;
|
2017-10-03 23:39:24 +00:00
|
|
|
const String format_name;
|
|
|
|
const String schema_name;
|
2017-12-16 01:45:11 +00:00
|
|
|
size_t num_consumers; /// Total number of created consumers
|
|
|
|
Poco::Logger * log;
|
2017-10-03 23:39:24 +00:00
|
|
|
|
2017-12-16 01:45:11 +00:00
|
|
|
// Consumer list
|
|
|
|
Poco::Semaphore semaphore;
|
2017-10-03 23:39:24 +00:00
|
|
|
std::mutex mutex;
|
2017-12-16 01:45:11 +00:00
|
|
|
std::vector<ConsumerPtr> consumers; /// Available consumers
|
|
|
|
|
|
|
|
// Stream thread
|
|
|
|
Poco::Event event_update;
|
2017-10-03 23:39:24 +00:00
|
|
|
std::thread stream_thread;
|
2017-12-16 01:45:11 +00:00
|
|
|
std::atomic<bool> stream_cancelled{false};
|
|
|
|
|
|
|
|
void consumerConfiguration(struct rd_kafka_conf_s * conf);
|
2017-12-20 19:09:13 +00:00
|
|
|
ConsumerPtr claimConsumer();
|
|
|
|
ConsumerPtr tryClaimConsumer(long wait_ms);
|
2017-12-16 01:45:11 +00:00
|
|
|
void pushConsumer(ConsumerPtr c);
|
2017-10-03 23:39:24 +00:00
|
|
|
|
2017-11-03 21:50:22 +00:00
|
|
|
void streamThread();
|
|
|
|
void streamToViews();
|
|
|
|
|
|
|
|
protected:
|
2017-10-03 23:39:24 +00:00
|
|
|
StorageKafka(
|
|
|
|
const std::string & table_name_,
|
|
|
|
const std::string & database_name_,
|
|
|
|
Context & context_,
|
2017-12-25 21:32:33 +00:00
|
|
|
const NamesAndTypes & columns_,
|
|
|
|
const NamesAndTypes & materialized_columns_,
|
|
|
|
const NamesAndTypes & alias_columns_,
|
2017-10-03 23:39:24 +00:00
|
|
|
const ColumnDefaults & column_defaults_,
|
|
|
|
const String & brokers_, const String & group_, const Names & topics_,
|
2017-12-16 01:45:11 +00:00
|
|
|
const String & format_name_, const String & schema_name_, size_t num_consumers_);
|
2017-10-03 23:39:24 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|
2017-10-14 00:53:01 +00:00
|
|
|
|
|
|
|
#endif
|