mirror of
https://github.com/Telecominfraproject/wlan-cloud-lib-cppkafka.git
synced 2025-11-03 12:07:57 +00:00
Add some other methods to TopicPartitionList
This commit is contained in:
@@ -28,6 +28,10 @@ public:
|
|||||||
TopicPartitionList& operator=(TopicPartitionList&&) = default;
|
TopicPartitionList& operator=(TopicPartitionList&&) = default;
|
||||||
|
|
||||||
void add(const TopicPartition& topic_partition);
|
void add(const TopicPartition& topic_partition);
|
||||||
|
void update(const TopicPartition& topic_partition);
|
||||||
|
bool remove(const TopicPartition& topic_partition);
|
||||||
|
|
||||||
|
bool contains(const TopicPartition& topic_partition) const;
|
||||||
size_t size() const;
|
size_t size() const;
|
||||||
bool empty() const;
|
bool empty() const;
|
||||||
|
|
||||||
@@ -40,6 +44,8 @@ private:
|
|||||||
|
|
||||||
static HandlePtr make_handle(rd_kafka_topic_partition_list_t* ptr);
|
static HandlePtr make_handle(rd_kafka_topic_partition_list_t* ptr);
|
||||||
|
|
||||||
|
rd_kafka_topic_partition_t* get_topic_partition(const TopicPartition& topic_partition) const;
|
||||||
|
|
||||||
HandlePtr handle_;
|
HandlePtr handle_;
|
||||||
};
|
};
|
||||||
|
|
||||||
|
|||||||
@@ -1,5 +1,6 @@
|
|||||||
#include "topic_partition_list.h"
|
#include "topic_partition_list.h"
|
||||||
#include "topic_partition.h"
|
#include "topic_partition.h"
|
||||||
|
#include "exceptions.h"
|
||||||
|
|
||||||
namespace cppkafka {
|
namespace cppkafka {
|
||||||
|
|
||||||
@@ -33,6 +34,27 @@ void TopicPartitionList::add(const TopicPartition& topic_partition) {
|
|||||||
element->offset = topic_partition.get_offset();
|
element->offset = topic_partition.get_offset();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
void TopicPartitionList::update(const TopicPartition& topic_partition) {
|
||||||
|
rd_kafka_resp_err_t error;
|
||||||
|
error = rd_kafka_topic_partition_list_set_offset(get_handle(),
|
||||||
|
topic_partition.get_topic().data(),
|
||||||
|
topic_partition.get_partition(),
|
||||||
|
topic_partition.get_offset());
|
||||||
|
if (error != RD_KAFKA_RESP_ERR_NO_ERROR) {
|
||||||
|
throw HandleException(error);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
bool TopicPartitionList::remove(const TopicPartition& topic_partition) {
|
||||||
|
return rd_kafka_topic_partition_list_del(get_handle(),
|
||||||
|
topic_partition.get_topic().data(),
|
||||||
|
topic_partition.get_partition()) == 1;
|
||||||
|
}
|
||||||
|
|
||||||
|
bool TopicPartitionList::contains(const TopicPartition& topic_partition) const {
|
||||||
|
return get_topic_partition(topic_partition) != nullptr;
|
||||||
|
}
|
||||||
|
|
||||||
size_t TopicPartitionList::size() const {
|
size_t TopicPartitionList::size() const {
|
||||||
return handle_->cnt;
|
return handle_->cnt;
|
||||||
}
|
}
|
||||||
@@ -50,4 +72,11 @@ TopicPartitionList::make_handle(rd_kafka_topic_partition_list_t* ptr) {
|
|||||||
return HandlePtr(ptr, &rd_kafka_topic_partition_list_destroy);
|
return HandlePtr(ptr, &rd_kafka_topic_partition_list_destroy);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
rd_kafka_topic_partition_t*
|
||||||
|
TopicPartitionList::get_topic_partition(const TopicPartition& topic_partition) const {
|
||||||
|
return rd_kafka_topic_partition_list_find(get_handle(),
|
||||||
|
topic_partition.get_topic().data(),
|
||||||
|
topic_partition.get_partition());
|
||||||
|
}
|
||||||
|
|
||||||
} // cppkafka
|
} // cppkafka
|
||||||
|
|||||||
Reference in New Issue
Block a user