2019-01-21 14:02:03 +00:00
|
|
|
#include <Storages/Kafka/ReadBufferFromKafkaConsumer.h>
|
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
2019-04-22 13:23:05 +00:00
|
|
|
|
2019-05-27 17:25:34 +00:00
|
|
|
using namespace std::chrono_literals;
|
2019-08-29 15:36:07 +00:00
|
|
|
|
2019-06-19 16:15:30 +00:00
|
|
|
ReadBufferFromKafkaConsumer::ReadBufferFromKafkaConsumer(
|
2019-08-29 15:36:07 +00:00
|
|
|
ConsumerPtr consumer_,
|
|
|
|
Poco::Logger * log_,
|
|
|
|
size_t max_batch_size,
|
|
|
|
size_t poll_timeout_,
|
|
|
|
bool intermediate_commit_,
|
|
|
|
const std::atomic<bool> & stopped_)
|
2019-06-19 16:15:30 +00:00
|
|
|
: ReadBuffer(nullptr, 0)
|
|
|
|
, consumer(consumer_)
|
|
|
|
, log(log_)
|
|
|
|
, batch_size(max_batch_size)
|
|
|
|
, poll_timeout(poll_timeout_)
|
|
|
|
, intermediate_commit(intermediate_commit_)
|
2019-08-29 15:36:07 +00:00
|
|
|
, stopped(stopped_)
|
2019-06-19 16:15:30 +00:00
|
|
|
, current(messages.begin())
|
|
|
|
{
|
|
|
|
}
|
2019-05-27 17:25:34 +00:00
|
|
|
|
|
|
|
ReadBufferFromKafkaConsumer::~ReadBufferFromKafkaConsumer()
|
|
|
|
{
|
|
|
|
/// NOTE: see https://github.com/edenhill/librdkafka/issues/2077
|
|
|
|
consumer->unsubscribe();
|
|
|
|
consumer->unassign();
|
2019-06-03 14:36:59 +00:00
|
|
|
while (consumer->get_consumer_queue().next_event(1s));
|
2019-05-27 17:25:34 +00:00
|
|
|
}
|
|
|
|
|
2019-01-25 12:48:59 +00:00
|
|
|
void ReadBufferFromKafkaConsumer::commit()
|
2019-01-21 14:02:03 +00:00
|
|
|
{
|
2019-08-29 15:36:07 +00:00
|
|
|
auto PrintOffsets = [this] (const char * prefix, const cppkafka::TopicPartitionList & offsets)
|
|
|
|
{
|
|
|
|
for (const auto & topic_part : offsets)
|
|
|
|
{
|
|
|
|
auto print_special_offset = [&topic_part]
|
|
|
|
{
|
|
|
|
switch (topic_part.get_offset())
|
|
|
|
{
|
|
|
|
case cppkafka::TopicPartition::OFFSET_BEGINNING: return "BEGINNING";
|
|
|
|
case cppkafka::TopicPartition::OFFSET_END: return "END";
|
|
|
|
case cppkafka::TopicPartition::OFFSET_STORED: return "STORED";
|
|
|
|
case cppkafka::TopicPartition::OFFSET_INVALID: return "INVALID";
|
|
|
|
default: return "";
|
|
|
|
}
|
|
|
|
};
|
|
|
|
|
|
|
|
if (topic_part.get_offset() < 0)
|
|
|
|
{
|
|
|
|
LOG_TRACE(
|
|
|
|
log,
|
|
|
|
prefix << " " << print_special_offset() << " (topic: " << topic_part.get_topic()
|
|
|
|
<< ", partition: " << topic_part.get_partition() << ")");
|
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
LOG_TRACE(
|
|
|
|
log,
|
|
|
|
prefix << " " << topic_part.get_offset() << " (topic: " << topic_part.get_topic()
|
|
|
|
<< ", partition: " << topic_part.get_partition() << ")");
|
|
|
|
}
|
|
|
|
}
|
|
|
|
};
|
|
|
|
|
|
|
|
PrintOffsets("Polled offset", consumer->get_offsets_position(consumer->get_assignment()));
|
|
|
|
|
2019-06-17 16:27:18 +00:00
|
|
|
if (current != messages.end())
|
|
|
|
{
|
|
|
|
/// Since we can poll more messages than we already processed,
|
|
|
|
/// commit only processed messages.
|
2019-08-29 15:36:07 +00:00
|
|
|
consumer->async_commit(*std::prev(current));
|
2019-06-17 16:27:18 +00:00
|
|
|
}
|
|
|
|
else
|
|
|
|
{
|
|
|
|
/// Commit everything we polled so far because either:
|
|
|
|
/// - read all polled messages (current == messages.end()),
|
|
|
|
/// - read nothing at all (messages.empty()),
|
|
|
|
/// - stalled.
|
|
|
|
consumer->async_commit();
|
|
|
|
}
|
2019-04-22 13:23:05 +00:00
|
|
|
|
2019-08-29 15:36:07 +00:00
|
|
|
PrintOffsets("Committed offset", consumer->get_offsets_committed(consumer->get_assignment()));
|
2019-07-03 16:51:11 +00:00
|
|
|
|
|
|
|
stalled = false;
|
2019-01-25 12:48:59 +00:00
|
|
|
}
|
2019-01-21 14:02:03 +00:00
|
|
|
|
2019-04-22 13:23:05 +00:00
|
|
|
void ReadBufferFromKafkaConsumer::subscribe(const Names & topics)
|
|
|
|
{
|
2019-07-19 15:01:34 +00:00
|
|
|
{
|
2019-08-07 16:10:14 +00:00
|
|
|
String message = "Already subscribed to topics:";
|
2019-07-19 15:01:34 +00:00
|
|
|
for (const auto & topic : consumer->get_subscription())
|
|
|
|
message += " " + topic;
|
|
|
|
LOG_TRACE(log, message);
|
|
|
|
}
|
|
|
|
|
|
|
|
{
|
2019-08-07 16:10:14 +00:00
|
|
|
String message = "Already assigned to topics:";
|
2019-07-19 15:01:34 +00:00
|
|
|
for (const auto & toppar : consumer->get_assignment())
|
|
|
|
message += " " + toppar.get_topic();
|
|
|
|
LOG_TRACE(log, message);
|
|
|
|
}
|
|
|
|
|
2019-04-22 13:23:05 +00:00
|
|
|
// While we wait for an assignment after subscribtion, we'll poll zero messages anyway.
|
|
|
|
// If we're doing a manual select then it's better to get something after a wait, then immediate nothing.
|
2019-08-07 16:10:14 +00:00
|
|
|
// But due to the nature of async pause/resume/subscribe we can't guarantee any persistent state:
|
|
|
|
// see https://github.com/edenhill/librdkafka/issues/2455
|
|
|
|
while (consumer->get_subscription().empty())
|
2019-04-22 13:23:05 +00:00
|
|
|
{
|
2019-08-07 16:10:14 +00:00
|
|
|
stalled = false;
|
|
|
|
|
|
|
|
try
|
|
|
|
{
|
|
|
|
consumer->subscribe(topics);
|
|
|
|
if (nextImpl())
|
|
|
|
break;
|
|
|
|
|
|
|
|
// FIXME: if we failed to receive "subscribe" response while polling and destroy consumer now, then we may hang up.
|
|
|
|
// see https://github.com/edenhill/librdkafka/issues/2077
|
|
|
|
}
|
|
|
|
catch (cppkafka::HandleException & e)
|
|
|
|
{
|
|
|
|
if (e.get_error() == RD_KAFKA_RESP_ERR__TIMED_OUT)
|
|
|
|
continue;
|
|
|
|
throw;
|
|
|
|
}
|
2019-04-22 13:23:05 +00:00
|
|
|
}
|
2019-05-15 16:11:50 +00:00
|
|
|
|
|
|
|
stalled = false;
|
2019-04-22 13:23:05 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
void ReadBufferFromKafkaConsumer::unsubscribe()
|
|
|
|
{
|
|
|
|
LOG_TRACE(log, "Re-joining claimed consumer after failure");
|
2019-07-16 15:27:42 +00:00
|
|
|
|
|
|
|
messages.clear();
|
|
|
|
current = messages.begin();
|
|
|
|
BufferBase::set(nullptr, 0, 0);
|
|
|
|
|
2019-04-22 13:23:05 +00:00
|
|
|
consumer->unsubscribe();
|
|
|
|
}
|
|
|
|
|
2019-06-24 11:42:58 +00:00
|
|
|
/// Do commit messages implicitly after we processed the previous batch.
|
2019-01-25 12:48:59 +00:00
|
|
|
bool ReadBufferFromKafkaConsumer::nextImpl()
|
|
|
|
{
|
2019-05-16 15:20:30 +00:00
|
|
|
/// NOTE: ReadBuffer was implemented with an immutable underlying contents in mind.
|
2019-05-15 16:11:50 +00:00
|
|
|
/// If we failed to poll any message once - don't try again.
|
|
|
|
/// Otherwise, the |poll_timeout| expectations get flawn.
|
2019-08-29 15:36:07 +00:00
|
|
|
if (stalled || stopped)
|
2019-05-15 16:11:50 +00:00
|
|
|
return false;
|
|
|
|
|
2019-01-25 12:48:59 +00:00
|
|
|
if (current == messages.end())
|
2019-01-21 14:02:03 +00:00
|
|
|
{
|
2019-05-16 15:20:30 +00:00
|
|
|
if (intermediate_commit)
|
|
|
|
commit();
|
2019-06-21 14:29:10 +00:00
|
|
|
|
|
|
|
/// Don't drop old messages immediately, since we may need them for virtual columns.
|
|
|
|
auto new_messages = consumer->poll_batch(batch_size, std::chrono::milliseconds(poll_timeout));
|
|
|
|
if (new_messages.empty())
|
|
|
|
{
|
|
|
|
LOG_TRACE(log, "Stalled");
|
|
|
|
stalled = true;
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
messages = std::move(new_messages);
|
2019-01-25 12:48:59 +00:00
|
|
|
current = messages.begin();
|
|
|
|
|
|
|
|
LOG_TRACE(log, "Polled batch of " << messages.size() << " messages");
|
2019-01-21 14:02:03 +00:00
|
|
|
}
|
2019-01-25 12:48:59 +00:00
|
|
|
|
|
|
|
if (auto err = current->get_error())
|
2019-01-21 14:02:03 +00:00
|
|
|
{
|
2019-01-25 12:48:59 +00:00
|
|
|
++current;
|
|
|
|
|
|
|
|
// TODO: should throw exception instead
|
2019-01-21 14:02:03 +00:00
|
|
|
LOG_ERROR(log, "Consumer error: " << err);
|
|
|
|
return false;
|
|
|
|
}
|
|
|
|
|
|
|
|
// XXX: very fishy place with const casting.
|
2019-01-25 12:48:59 +00:00
|
|
|
auto new_position = reinterpret_cast<char *>(const_cast<unsigned char *>(current->get_payload().get_data()));
|
|
|
|
BufferBase::set(new_position, current->get_payload().get_size(), 0);
|
|
|
|
|
|
|
|
++current;
|
2019-01-23 11:00:43 +00:00
|
|
|
|
2019-01-21 14:02:03 +00:00
|
|
|
return true;
|
|
|
|
}
|
|
|
|
|
2019-04-22 13:23:05 +00:00
|
|
|
}
|