Skip to content
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

Fallback to fetch v12 (#10) #4806

Open
wants to merge 1 commit into
base: master
Choose a base branch
from
Open
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
21 changes: 20 additions & 1 deletion src/rdkafka_fetcher.c
Original file line number Diff line number Diff line change
Expand Up @@ -943,7 +943,21 @@ static void rd_kafka_broker_fetch_reply(rd_kafka_t *rk,
}
}

/**
* @brief Check if any toppars have a non-zero topic id.
Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Will change to zero topic id

*
*/
rd_bool_t can_use_topic_id(rd_kafka_broker_t *rkb) {
Copy link
Collaborator

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Add static, better calling it can_use_topic_ids (as in J)

Suggested change
rd_bool_t can_use_topic_id(rd_kafka_broker_t *rkb) {
static rd_bool_t can_use_topic_ids(rd_kafka_broker_t *rkb) {

rd_kafka_toppar_t *rktp = rkb->rkb_active_toppar_next;
do {
if (RD_KAFKA_UUID_IS_ZERO(rktp->rktp_rkt->rkt_topic_id))
return rd_false;
} while ((rktp = CIRCLEQ_LOOP_NEXT(&rkb->rkb_active_toppars, rktp,
rktp_activelink)) !=
rkb->rkb_active_toppar_next);

return rd_true;
}

/**
* @brief Build and send a Fetch request message for all underflowed toppars
Expand Down Expand Up @@ -979,7 +993,12 @@ int rd_kafka_broker_fetch_toppars(rd_kafka_broker_t *rkb, rd_ts_t now) {

ApiVersion = rd_kafka_broker_ApiVersion_supported(rkb, RD_KAFKAP_Fetch,
0, 16, NULL);
rkbuf = rd_kafka_buf_new_flexver_request(

/* Fallback to version 12 if topic id is null which can happen if
* inter.broker.protocol.version is old */
Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Change old to <2.8

ApiVersion = ApiVersion > 12 && can_use_topic_id(rkb) ? ApiVersion : 12;

rkbuf = rd_kafka_buf_new_flexver_request(
rkb, RD_KAFKAP_Fetch, 1,
/* MaxWaitTime+MinBytes+MaxBytes+IsolationLevel+
* SessionId+Epoch+TopicCnt */
Expand Down