-
Notifications
You must be signed in to change notification settings - Fork 3.2k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Header support for C++ API #1959
Changes from 3 commits
cba970d
f2f8466
fe84db6
1f79384
fe68e1c
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change | ||||
---|---|---|---|---|---|---|
@@ -0,0 +1,49 @@ | ||||||
/* | ||||||
* librdkafka - Apache Kafka C/C++ library | ||||||
* | ||||||
* Copyright (c) 2014 Magnus Edenhill | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
Suggested change
|
||||||
* All rights reserved. | ||||||
* | ||||||
* Redistribution and use in source and binary forms, with or without | ||||||
* modification, are permitted provided that the following conditions are met: | ||||||
* | ||||||
* 1. Redistributions of source code must retain the above copyright notice, | ||||||
* this list of conditions and the following disclaimer. | ||||||
* 2. Redistributions in binary form must reproduce the above copyright notice, | ||||||
* this list of conditions and the following disclaimer in the documentation | ||||||
* and/or other materials provided with the distribution. | ||||||
* | ||||||
* THIS SOFTWARE IS PROVIDED BY THE COPYRIGHT HOLDERS AND CONTRIBUTORS "AS IS" | ||||||
* AND ANY EXPRESS OR IMPLIED WARRANTIES, INCLUDING, BUT NOT LIMITED TO, THE | ||||||
* IMPLIED WARRANTIES OF MERCHANTABILITY AND FITNESS FOR A PARTICULAR PURPOSE | ||||||
* ARE DISCLAIMED. IN NO EVENT SHALL THE COPYRIGHT OWNER OR CONTRIBUTORS BE | ||||||
* LIABLE FOR ANY DIRECT, INDIRECT, INCIDENTAL, SPECIAL, EXEMPLARY, OR | ||||||
* CONSEQUENTIAL DAMAGES (INCLUDING, BUT NOT LIMITED TO, PROCUREMENT OF | ||||||
* SUBSTITUTE GOODS OR SERVICES; LOSS OF USE, DATA, OR PROFITS; OR BUSINESS | ||||||
* INTERRUPTION) HOWEVER CAUSED AND ON ANY THEORY OF LIABILITY, WHETHER IN | ||||||
* CONTRACT, STRICT LIABILITY, OR TORT (INCLUDING NEGLIGENCE OR OTHERWISE) | ||||||
* ARISING IN ANY WAY OUT OF THE USE OF THIS SOFTWARE, EVEN IF ADVISED OF THE | ||||||
* POSSIBILITY OF SUCH DAMAGE. | ||||||
*/ | ||||||
|
||||||
#include <iostream> | ||||||
#include <string> | ||||||
#include <list> | ||||||
#include <cerrno> | ||||||
|
||||||
#include "rdkafkacpp_int.h" | ||||||
|
||||||
RdKafka::Headers *RdKafka::Headers::create(size_t initial_count, bool free_rd_headers) { | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. two spaces after type, should only be one. |
||||||
return new RdKafka::HeadersImpl(initial_count, free_rd_headers); | ||||||
} | ||||||
|
||||||
RdKafka::Headers *RdKafka::Headers::create(const std::vector<Header> &headers, bool free_rd_headers) { | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. free_rd_headers is an implementation detail, remove it from the public API. |
||||||
if (headers.size() > 0) { | ||||||
return new RdKafka::HeadersImpl(headers, free_rd_headers); | ||||||
} else { | ||||||
return 0; | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I would've assumed an empty Headers list here rather than NULL |
||||||
} | ||||||
|
||||||
} | ||||||
|
||||||
RdKafka::Headers::~Headers() {} |
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -143,14 +143,19 @@ RdKafka::ProducerImpl::produce (RdKafka::Topic *topic, | |
|
||
} | ||
|
||
|
||
RdKafka::ErrorCode | ||
RdKafka::ProducerImpl::produce (const std::string topic_name, | ||
int32_t partition, int msgflags, | ||
void *payload, size_t len, | ||
const void *key, size_t key_len, | ||
int64_t timestamp, | ||
void *msg_opaque) { | ||
int64_t timestamp, void *msg_opaque, | ||
RdKafka::Headers *headers) { | ||
rd_kafka_headers_t *hdrs; | ||
if (headers) { | ||
hdrs = headers->c_headers(); | ||
} else { | ||
hdrs = 0; | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. use NULL for pointers, not 0. And in this case you can initialize |
||
} | ||
return | ||
static_cast<RdKafka::ErrorCode> | ||
( | ||
|
@@ -162,6 +167,7 @@ RdKafka::ProducerImpl::produce (const std::string topic_name, | |
RD_KAFKA_V_KEY(key, key_len), | ||
RD_KAFKA_V_TIMESTAMP(timestamp), | ||
RD_KAFKA_V_OPAQUE(msg_opaque), | ||
RD_KAFKA_V_HEADERS(hdrs), | ||
RD_KAFKA_V_END) | ||
); | ||
} |
Original file line number | Diff line number | Diff line change | ||||
---|---|---|---|---|---|---|
|
@@ -75,11 +75,11 @@ extern "C" { | |||||
struct rd_kafka_s; | ||||||
struct rd_kafka_topic_s; | ||||||
struct rd_kafka_message_s; | ||||||
struct rd_kafka_headers_s; | ||||||
}; | ||||||
|
||||||
namespace RdKafka { | ||||||
|
||||||
|
||||||
/** | ||||||
* @name Miscellaneous APIs | ||||||
* @{ | ||||||
|
@@ -396,6 +396,7 @@ std::string err2str(RdKafka::ErrorCode err); | |||||
/* Forward declarations */ | ||||||
class Producer; | ||||||
class Message; | ||||||
class Headers; | ||||||
class Queue; | ||||||
class Event; | ||||||
class Topic; | ||||||
|
@@ -1365,7 +1366,139 @@ class RD_EXPORT MessageTimestamp { | |||||
int64_t timestamp; /**< Milliseconds since epoch (UTC). */ | ||||||
}; | ||||||
|
||||||
/** | ||||||
* @brief Headers object | ||||||
* | ||||||
* This object encapsulates the C implementation logic into a C++ object | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. We shouldn't reference the C API from the public C++ API: that'll require users to know both the C and C++ API. |
||||||
* for use in RdKafka::Messages object. | ||||||
*/ | ||||||
class RD_EXPORT Headers { | ||||||
public: | ||||||
virtual ~Headers() = 0; | ||||||
|
||||||
/** | ||||||
* @brief Header object | ||||||
* | ||||||
* This object represents a single Header with key value pair | ||||||
* and an ErrorCode | ||||||
* | ||||||
* @remark dynamic allocation of this object is not supported. | ||||||
* | ||||||
*/ | ||||||
class Header { | ||||||
public: | ||||||
Header(const std::string& key, | ||||||
const char* value, | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
|
||||||
RdKafka::ErrorCode err = ERR_NO_ERROR): | ||||||
key_(key), err_(err) { | ||||||
value_container_.assign(value); | ||||||
}; | ||||||
|
||||||
std::string key() const { | ||||||
return key_; | ||||||
} | ||||||
|
||||||
const char* value() const { | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. header values may be NULL, and are to be considered binary, so use const void * and a size_t length instead |
||||||
return value_container_.c_str(); | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. values may be NULL |
||||||
} | ||||||
|
||||||
RdKafka::ErrorCode err() const { | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. What's the use of the error field? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. The error field was just so when you got one individual Header when it was returned you could pass back if there was an error or not on when trying to get that Header |
||||||
return err_; | ||||||
} | ||||||
|
||||||
private: | ||||||
std::string key_; | ||||||
RdKafka::ErrorCode err_; | ||||||
std::string value_container_; | ||||||
void *operator new(size_t); /* Prevent dynamic allocation */ | ||||||
}; | ||||||
|
||||||
/** | ||||||
* @brief create a new instance of the Headers object | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Capital "C" in "Create", same for the other functions |
||||||
*/ | ||||||
static Headers *create(size_t initial_size = 8, bool free_rd_headers = true); | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Don't use default param values: Document the I don't think free_rd_headers should be part of the public API |
||||||
|
||||||
/** | ||||||
* @brief create a new instance of the Headers object from a std::vector | ||||||
*/ | ||||||
static Headers *create(const std::vector<Header> &headers, bool free_rd_headers = true); | ||||||
|
||||||
/** | ||||||
* @brief adds a Header to the end | ||||||
* | ||||||
* @returns An ErrorCode signalling a success or failure to add the Header. | ||||||
*/ | ||||||
virtual ErrorCode add(const std::string& key, const char* value) = 0; | ||||||
|
||||||
/** | ||||||
* @brief removes all the Headers of a given key | ||||||
* | ||||||
* @returns An ErrorCode signalling a success or failure to remove the Header. | ||||||
*/ | ||||||
virtual ErrorCode remove(const std::string& key) = 0; | ||||||
|
||||||
/** | ||||||
* @brief gets all of the Headers of a given key | ||||||
* | ||||||
* @returns a std::vector containing all the Headers of the given key. | ||||||
*/ | ||||||
virtual std::vector<Header> get(const std::string &key) const = 0; | ||||||
|
||||||
/** | ||||||
* @brief gets the last occurrence of a Header of a given key | ||||||
* | ||||||
* @returns the Header if found, otherwise a Header with an ErrorCode | ||||||
*/ | ||||||
virtual Header get_last(const std::string& key) const = 0; | ||||||
|
||||||
/** | ||||||
* @brief returns all the Headers of a Message | ||||||
* | ||||||
* @returns a std::vector containing all of the Headers of a message | ||||||
*/ | ||||||
virtual std::vector<Header> get_all() const = 0; | ||||||
|
||||||
/** | ||||||
* @brief the count of all the Headers | ||||||
* | ||||||
* @returns a size_t count of all the headers | ||||||
*/ | ||||||
virtual size_t size() const = 0; | ||||||
|
||||||
/** | ||||||
* @brief Returns the underlying librdkafka C rd_kafka_headers_t handle. | ||||||
* | ||||||
* @warning Calling the C API on this handle is not recommended and there | ||||||
* is no official support for it, but for cases where the C++ API | ||||||
* does not provide the underlying functionality this C handle can be | ||||||
* used to interact directly with the core librdkafka API. | ||||||
* | ||||||
* @remark The lifetime of the returned pointer can be different than the lifetime | ||||||
* of the Headers message due to how the producev function in the C API works | ||||||
* if there is no error then the producev will take care of deallocation | ||||||
* but if there is an error then it is the responsibility of the calling | ||||||
* object to deallocate the underlying C implementation if an instance | ||||||
* of the Headers object is created with free_rd_headers set to `false` | ||||||
* | ||||||
* @remark Include <rdkafka/rdkafka.h> prior to including | ||||||
* <rdkafka/rdkafkacpp.h> | ||||||
* | ||||||
* @returns \c rd_kafka_headers_t* | ||||||
*/ | ||||||
virtual struct rd_kafka_headers_s *c_headers() = 0; | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
Suggested change
|
||||||
|
||||||
/** | ||||||
* @brief cleans up the underlying alocated C implementation headers if called | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. allocated There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. when would you use this method? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. It's probably an artifact of some experimentation I was doing, with your comment mentioning detaching the headers from the rkmessage I could completely encapsulate the Headers within in this instance and let the destructor manage the lifetime of the rd_headers There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Actually come to think of it, I think this is still needed in the case where producev fails and doesn't clean up the C headers and in that case you will have to manually clean them up yourself because producev can't set the internal C headers ptr to NULL so the destructor will fail for a double free |
||||||
* | ||||||
* @remark Safe to call even if the Headers object is set to clean up when | ||||||
* when the destructor is called | ||||||
* | ||||||
* @remark Safe to call even if the underlyng C pointer is set to null | ||||||
* | ||||||
* @returns an ErrorCode signalling if the the operation was attempted | ||||||
*/ | ||||||
virtual ErrorCode destroy_headers() = 0; | ||||||
}; | ||||||
|
||||||
/** | ||||||
* @brief Message object | ||||||
|
@@ -1428,6 +1561,9 @@ class RD_EXPORT Message { | |||||
/** @returns The \p msg_opaque as provided to RdKafka::Producer::produce() */ | ||||||
virtual void *msg_opaque () const = 0; | ||||||
|
||||||
/** @returns The Headers instance for this Message (if applicable) */ | ||||||
virtual RdKafka::Headers *get_headers() = 0; | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Add new methods to end of the class. |
||||||
|
||||||
virtual ~Message () = 0; | ||||||
|
||||||
/** @returns the latency in microseconds for a produced message measured | ||||||
|
@@ -2147,8 +2283,8 @@ class RD_EXPORT Producer : public virtual Handle { | |||||
int msgflags, | ||||||
void *payload, size_t len, | ||||||
const void *key, size_t key_len, | ||||||
int64_t timestamp, | ||||||
void *msg_opaque) = 0; | ||||||
int64_t timestamp, void *msg_opaque, | ||||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Must not change existing APIs since that will break the API stability guarantee There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Noted thank you, I'll overload the method |
||||||
RdKafka::Headers *headers) = 0; | ||||||
|
||||||
|
||||||
/** | ||||||
|
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
There is no need for headers support in the verifiable_client, please remove these changes.
kafkatest_verifiable_client is used to run the Java system tests with the librdkafka client.