kafka.cpp 40 KB

12345678910111213141516171819202122232425262728293031323334353637383940414243444546474849505152535455565758596061626364656667686970717273747576777879808182838485868788899091929394959697989910010110210310410510610710810911011111211311411511611711811912012112212312412512612712812913013113213313413513613713813914014114214314414514614714814915015115215315415515615715815916016116216316416516616716816917017117217317417517617717817918018118218318418518618718818919019119219319419519619719819920020120220320420520620720820921021121221321421521621721821922022122222322422522622722822923023123223323423523623723823924024124224324424524624724824925025125225325425525625725825926026126226326426526626726826927027127227327427527627727827928028128228328428528628728828929029129229329429529629729829930030130230330430530630730830931031131231331431531631731831932032132232332432532632732832933033133233333433533633733833934034134234334434534634734834935035135235335435535635735835936036136236336436536636736836937037137237337437537637737837938038138238338438538638738838939039139239339439539639739839940040140240340440540640740840941041141241341441541641741841942042142242342442542642742842943043143243343443543643743843944044144244344444544644744844945045145245345445545645745845946046146246346446546646746846947047147247347447547647747847948048148248348448548648748848949049149249349449549649749849950050150250350450550650750850951051151251351451551651751851952052152252352452552652752852953053153253353453553653753853954054154254354454554654754854955055155255355455555655755855956056156256356456556656756856957057157257357457557657757857958058158258358458558658758858959059159259359459559659759859960060160260360460560660760860961061161261361461561661761861962062162262362462562662762862963063163263363463563663763863964064164264364464564664764864965065165265365465565665765865966066166266366466566666766866967067167267367467567667767867968068168268368468568668768868969069169269369469569669769869970070170270370470570670770870971071171271371471571671771871972072172272372472572672772872973073173273373473573673773873974074174274374474574674774874975075175275375475575675775875976076176276376476576676776876977077177277377477577677777877978078178278378478578678778878979079179279379479579679779879980080180280380480580680780880981081181281381481581681781881982082182282382482582682782882983083183283383483583683783883984084184284384484584684784884985085185285385485585685785885986086186286386486586686786886987087187287387487587687787887988088188288388488588688788888989089189289389489589689789889990090190290390490590690790890991091191291391491591691791891992092192292392492592692792892993093193293393493593693793893994094194294394494594694794894995095195295395495595695795895996096196296396496596696796896997097197297397497597697797897998098198298398498598698798898999099199299399499599699799899910001001100210031004100510061007100810091010101110121013101410151016101710181019102010211022102310241025102610271028102910301031103210331034103510361037103810391040104110421043104410451046104710481049105010511052105310541055105610571058105910601061106210631064106510661067106810691070107110721073107410751076107710781079108010811082108310841085108610871088108910901091109210931094109510961097109810991100110111021103110411051106110711081109111011111112111311141115111611171118111911201121112211231124112511261127
  1. /*##############################################################################
  2. HPCC SYSTEMS software Copyright (C) 2015 HPCC Systems®.
  3. Licensed under the Apache License, Version 2.0 (the "License");
  4. you may not use this file except in compliance with the License.
  5. You may obtain a copy of the License at
  6. http://www.apache.org/licenses/LICENSE-2.0
  7. Unless required by applicable law or agreed to in writing, software
  8. distributed under the License is distributed on an "AS IS" BASIS,
  9. WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  10. See the License for the specific language governing permissions and
  11. limitations under the License.
  12. ############################################################################## */
  13. #include "kafka.hpp"
  14. #include "rtlds_imp.hpp"
  15. #include "jlog.hpp"
  16. #include "jmutex.hpp"
  17. #include "jprop.hpp"
  18. #include "jfile.hpp"
  19. #include "librdkafka/rdkafka.h"
  20. #include <map>
  21. #include <fstream>
  22. #include <mutex>
  23. //==============================================================================
  24. // Kafka Interface Code
  25. //==============================================================================
  26. namespace KafkaPlugin
  27. {
  28. //--------------------------------------------------------------------------
  29. // File Constants
  30. //--------------------------------------------------------------------------
  31. // Filename of global Kafka configuration file
  32. const char* GLOBAL_CONFIG_FILENAME = "kafka_global.conf";
  33. // The minimum number of seconds that a cached object can live
  34. // without activity
  35. const time_t OBJECT_EXPIRE_TIMEOUT_SECONDS = 60 * 2;
  36. // The number of milliseconds given to librdkafka to perform explicit
  37. // background activity
  38. const __int32 POLL_TIMEOUT = 1000;
  39. //--------------------------------------------------------------------------
  40. // Static Variables
  41. //--------------------------------------------------------------------------
  42. static std::once_flag pubCacheInitFlag;
  43. //--------------------------------------------------------------------------
  44. // Static Methods (internal)
  45. //--------------------------------------------------------------------------
  46. /**
  47. * Look for an optional configuration file and apply any found configuration
  48. * parameters to a librdkafka configuration object.
  49. *
  50. * @param configFilePath The path to a configuration file; it is not
  51. * necessary for the file to exist
  52. * @param globalConfigPtr A pointer to the configuration object that
  53. * will receive any found parameters
  54. * @param traceLevel The current log trace level
  55. */
  56. static void applyConfig(const char* configFilePath, RdKafka::Conf* globalConfigPtr, int traceLevel)
  57. {
  58. if (configFilePath && *configFilePath && globalConfigPtr)
  59. {
  60. std::string errStr;
  61. StringBuffer fullConfigPath;
  62. fullConfigPath.append(hpccBuildInfo.configDir).append(PATHSEPSTR).append(configFilePath);
  63. Owned<IProperties> properties = createProperties(fullConfigPath.str(), true);
  64. Owned<IPropertyIterator> props = properties->getIterator();
  65. ForEach(*props)
  66. {
  67. StringBuffer key(props->getPropKey());
  68. key.trim();
  69. if (key.length() > 0 && key.charAt(0) != '#')
  70. {
  71. if (strcmp(key.str(), "metadata.broker.list") != 0)
  72. {
  73. const char* value = properties->queryProp(key);
  74. if (value && *value)
  75. {
  76. if (globalConfigPtr->set(key.str(), value, errStr) != RdKafka::Conf::CONF_OK)
  77. {
  78. DBGLOG("Kafka: Failed to set config param from file %s: '%s' = '%s'; error: '%s'", configFilePath, key.str(), value, errStr.c_str());
  79. }
  80. else if (traceLevel > 4)
  81. {
  82. DBGLOG("Kafka: Set config param from file %s: '%s' = '%s'", configFilePath, key.str(), value);
  83. }
  84. }
  85. }
  86. else
  87. {
  88. DBGLOG("Kafka: Setting '%s' ignored in config file %s", key.str(), configFilePath);
  89. }
  90. }
  91. }
  92. }
  93. }
  94. //--------------------------------------------------------------------------
  95. // Plugin Classes
  96. //--------------------------------------------------------------------------
  97. KafkaStreamedDataset::KafkaStreamedDataset(Consumer* _consumerPtr, IEngineRowAllocator* _resultAllocator, int _traceLevel, __int64 _maxRecords)
  98. : consumerPtr(_consumerPtr),
  99. resultAllocator(_resultAllocator),
  100. traceLevel(_traceLevel),
  101. maxRecords(_maxRecords)
  102. {
  103. shouldRead = true;
  104. consumedRecCount = 0;
  105. lastMsgOffset = 0;
  106. }
  107. KafkaStreamedDataset::~KafkaStreamedDataset()
  108. {
  109. if (consumerPtr)
  110. {
  111. if (consumedRecCount > 0)
  112. {
  113. consumerPtr->commitOffset(lastMsgOffset);
  114. }
  115. delete(consumerPtr);
  116. }
  117. }
  118. const void* KafkaStreamedDataset::nextRow()
  119. {
  120. const void* result = NULL;
  121. __int32 maxAttempts = 10; //!< Maximum number of tries if local queue is full
  122. __int32 timeoutWait = 100; //!< Amount of time to wait between retries
  123. __int32 attemptNum = 0;
  124. if (consumerPtr && (maxRecords <= 0 || consumedRecCount < maxRecords))
  125. {
  126. RdKafka::Message* messageObjPtr = NULL;
  127. bool messageConsumed = false;
  128. while (!messageConsumed && shouldRead && attemptNum < maxAttempts)
  129. {
  130. messageObjPtr = consumerPtr->getOneMessage(); // messageObjPtr must be deleted when we are through with it
  131. if (messageObjPtr)
  132. {
  133. try
  134. {
  135. switch (messageObjPtr->err())
  136. {
  137. case RdKafka::ERR_NO_ERROR:
  138. {
  139. RtlDynamicRowBuilder rowBuilder(resultAllocator);
  140. unsigned len = sizeof(__int32) + sizeof(__int64) + sizeof(size32_t) + messageObjPtr->len();
  141. byte* row = rowBuilder.ensureCapacity(len, NULL);
  142. // Populating this structure:
  143. // EXPORT KafkaMessage := RECORD
  144. // UNSIGNED4 partitionNum;
  145. // UNSIGNED8 offset;
  146. // UTF8 message;
  147. // END;
  148. *(__int32*)(row) = messageObjPtr->partition();
  149. *(__int64*)(row + sizeof(__int32)) = messageObjPtr->offset();
  150. *(size32_t*)(row + sizeof(__int32) + sizeof(__int64)) = rtlUtf8Length(messageObjPtr->len(), messageObjPtr->payload());
  151. memcpy(row + sizeof(__int32) + sizeof(__int64) + sizeof(size32_t), messageObjPtr->payload(), messageObjPtr->len());
  152. result = rowBuilder.finalizeRowClear(len);
  153. lastMsgOffset = messageObjPtr->offset();
  154. ++consumedRecCount;
  155. // Give opportunity for consumer to pull in any additional messages
  156. consumerPtr->handle()->poll(0);
  157. // Mark as loaded so we don't retry
  158. messageConsumed = true;
  159. }
  160. break;
  161. case RdKafka::ERR__TIMED_OUT:
  162. // No new messages arrived and we timed out waiting
  163. ++attemptNum;
  164. consumerPtr->handle()->poll(timeoutWait);
  165. break;
  166. case RdKafka::ERR__PARTITION_EOF:
  167. // We reached the end of the messages in the partition
  168. if (traceLevel > 4)
  169. {
  170. DBGLOG("Kafka: EOF reading message from partition %d", messageObjPtr->partition());
  171. }
  172. shouldRead = false;
  173. break;
  174. case RdKafka::ERR__UNKNOWN_PARTITION:
  175. // Unknown partition; don't throw an error here because
  176. // in some configurations (e.g. more Thor slaves than
  177. // partitions) not all consumers will have a partition
  178. // to read
  179. if (traceLevel > 4)
  180. {
  181. DBGLOG("Kafka: Unknown partition while trying to read");
  182. }
  183. shouldRead = false;
  184. break;
  185. case RdKafka::ERR__UNKNOWN_TOPIC:
  186. throw MakeStringException(-1, "Kafka: Error while reading message: '%s'", messageObjPtr->errstr().c_str());
  187. break;
  188. }
  189. }
  190. catch (...)
  191. {
  192. delete(messageObjPtr);
  193. throw;
  194. }
  195. delete(messageObjPtr);
  196. messageObjPtr = NULL;
  197. }
  198. }
  199. }
  200. return result;
  201. }
  202. void KafkaStreamedDataset::stop()
  203. {
  204. shouldRead = false;
  205. }
  206. //--------------------------------------------------------------------------
  207. Poller::Poller(KafkaObj* _parentPtr, __int32 _pollTimeout)
  208. : Thread("Kafka::Poller"),
  209. parentPtr(_parentPtr),
  210. pollTimeout(_pollTimeout),
  211. shouldRun(false)
  212. {
  213. }
  214. void Poller::start()
  215. {
  216. if (!isAlive() && parentPtr)
  217. {
  218. shouldRun = true;
  219. Thread::start();
  220. }
  221. }
  222. void Poller::stop()
  223. {
  224. if (isAlive())
  225. {
  226. shouldRun = false;
  227. join();
  228. }
  229. }
  230. int Poller::run()
  231. {
  232. RdKafka::Handle* handle = parentPtr->handle();
  233. while (shouldRun)
  234. {
  235. handle->poll(pollTimeout);
  236. }
  237. return 0;
  238. }
  239. //--------------------------------------------------------------------------
  240. Publisher::Publisher(const std::string& _brokers, const std::string& _topic, __int32 _pollTimeout, int _traceLevel)
  241. : brokers(_brokers),
  242. topic(_topic),
  243. pollTimeout(_pollTimeout),
  244. traceLevel(_traceLevel)
  245. {
  246. producerPtr = NULL;
  247. topicPtr = NULL;
  248. pollerPtr = new Poller(this, _pollTimeout);
  249. updateTimeTouched();
  250. }
  251. Publisher::~Publisher()
  252. {
  253. delete(pollerPtr);
  254. delete(topicPtr.load());
  255. delete(producerPtr);
  256. }
  257. RdKafka::Handle* Publisher::handle()
  258. {
  259. return static_cast<RdKafka::Handle*>(producerPtr);
  260. }
  261. time_t Publisher::updateTimeTouched()
  262. {
  263. timeCreated = time(NULL);
  264. return timeCreated;
  265. }
  266. time_t Publisher::getTimeTouched() const
  267. {
  268. return timeCreated;
  269. }
  270. void Publisher::shutdownPoller()
  271. {
  272. if (pollerPtr)
  273. {
  274. // Wait until we send all messages
  275. while (messagesWaitingInQueue() > 0)
  276. {
  277. usleep(pollTimeout);
  278. }
  279. // Tell poller to stop
  280. pollerPtr->stop();
  281. }
  282. }
  283. __int32 Publisher::messagesWaitingInQueue()
  284. {
  285. __int32 queueLength = 0;
  286. if (producerPtr)
  287. {
  288. queueLength = producerPtr->outq_len();
  289. }
  290. return queueLength;
  291. }
  292. void Publisher::ensureSetup()
  293. {
  294. if (!topicPtr.load(std::memory_order_acquire))
  295. {
  296. CriticalBlock block(lock);
  297. if (!topicPtr.load(std::memory_order_relaxed))
  298. {
  299. std::string errStr;
  300. RdKafka::Conf* globalConfig = RdKafka::Conf::create(RdKafka::Conf::CONF_GLOBAL);
  301. if (globalConfig)
  302. {
  303. // Set global configuration parameters, used mainly at the producer level
  304. globalConfig->set("metadata.broker.list", brokers, errStr);
  305. globalConfig->set("queue.buffering.max.messages", "1000000", errStr);
  306. globalConfig->set("compression.codec", "snappy", errStr);
  307. globalConfig->set("message.send.max.retries", "3", errStr);
  308. globalConfig->set("retry.backoff.ms", "500", errStr);
  309. // Set any global configurations from file, allowing
  310. // overrides of above settings
  311. applyConfig(GLOBAL_CONFIG_FILENAME, globalConfig, traceLevel);
  312. // Set producer callbacks
  313. globalConfig->set("event_cb", static_cast<RdKafka::EventCb*>(this), errStr);
  314. globalConfig->set("dr_cb", static_cast<RdKafka::DeliveryReportCb*>(this), errStr);
  315. // Create the producer
  316. producerPtr = RdKafka::Producer::create(globalConfig, errStr);
  317. delete globalConfig;
  318. if (producerPtr)
  319. {
  320. RdKafka::Conf* topicConfPtr = RdKafka::Conf::create(RdKafka::Conf::CONF_TOPIC);
  321. // Set any topic configurations from file
  322. std::string confName = "kafka_publisher_topic_" + topic + ".conf";
  323. applyConfig(confName.c_str(), topicConfPtr, traceLevel);
  324. // Create the topic
  325. topicPtr.store(RdKafka::Topic::create(producerPtr, topic, topicConfPtr, errStr), std::memory_order_release);
  326. delete topicConfPtr;
  327. if (topicPtr)
  328. {
  329. // Start the attached background poller
  330. pollerPtr->start();
  331. }
  332. else
  333. {
  334. throw MakeStringException(-1, "Kafka: Unable to create producer topic object for topic '%s'; error: '%s'", topic.c_str(), errStr.c_str());
  335. }
  336. }
  337. else
  338. {
  339. throw MakeStringException(-1, "Kafka: Unable to create producer object for brokers '%s'; error: '%s'", brokers.c_str(), errStr.c_str());
  340. }
  341. }
  342. else
  343. {
  344. throw MakeStringException(-1, "Kafka: Unable to create producer global configuration object for brokers '%s'; error: '%s'", brokers.c_str(), errStr.c_str());
  345. }
  346. }
  347. }
  348. }
  349. void Publisher::sendMessage(const std::string& message, const std::string& key)
  350. {
  351. __int32 maxAttempts = 10; //!< Maximum number of tries if local queue is full
  352. __int32 attemptNum = 0;
  353. // Make sure we have a valid connection to the Kafka cluster
  354. ensureSetup();
  355. // Actually send the message
  356. while (true)
  357. {
  358. RdKafka::ErrorCode resp = producerPtr->produce(topicPtr, RdKafka::Topic::PARTITION_UA, RdKafka::Producer::RK_MSG_COPY, const_cast<char*>(message.c_str()), message.size(), (key.empty() ? NULL : &key), NULL);
  359. if (resp == RdKafka::ERR_NO_ERROR)
  360. {
  361. break;
  362. }
  363. else if (resp == RdKafka::ERR__QUEUE_FULL)
  364. {
  365. if (attemptNum < maxAttempts)
  366. {
  367. usleep(pollTimeout);
  368. ++attemptNum;
  369. }
  370. else
  371. {
  372. throw MakeStringException(-1, "Kafka: Unable to send message to topic '%s'; error: '%s'", topic.c_str(), RdKafka::err2str(resp).c_str());
  373. }
  374. }
  375. else
  376. {
  377. throw MakeStringException(-1, "Kafka: Unable to send message to topic '%s'; error: '%s'", topic.c_str(), RdKafka::err2str(resp).c_str());
  378. }
  379. }
  380. }
  381. void Publisher::event_cb(RdKafka::Event& event)
  382. {
  383. if (traceLevel > 4)
  384. {
  385. switch (event.type())
  386. {
  387. case RdKafka::Event::EVENT_ERROR:
  388. DBGLOG("Kafka: Error: %s", event.str().c_str());
  389. break;
  390. case RdKafka::Event::EVENT_STATS:
  391. DBGLOG("Kafka: Stats: %s", event.str().c_str());
  392. break;
  393. case RdKafka::Event::EVENT_LOG:
  394. DBGLOG("Kafka: Log: %s", event.str().c_str());
  395. break;
  396. }
  397. }
  398. }
  399. void Publisher::dr_cb (RdKafka::Message& message)
  400. {
  401. if (message.err() != RdKafka::ERR_NO_ERROR)
  402. {
  403. StringBuffer payloadStr;
  404. if (message.len() == 0)
  405. payloadStr.append("<no message>");
  406. else
  407. payloadStr.append(message.len(), static_cast<const char*>(message.payload()));
  408. DBGLOG("Kafka: Error publishing message: %d (%s); message: '%s'", message.err(), message.errstr().c_str(), payloadStr.str());
  409. }
  410. }
  411. //--------------------------------------------------------------------------
  412. Consumer::Consumer(const std::string& _brokers, const std::string& _topic, const std::string& _consumerGroup, __int32 _partitionNum, int _traceLevel)
  413. : brokers(_brokers),
  414. topic(_topic),
  415. consumerGroup(_consumerGroup),
  416. partitionNum(_partitionNum),
  417. traceLevel(_traceLevel)
  418. {
  419. consumerPtr = NULL;
  420. topicPtr = NULL;
  421. char cpath[_MAX_DIR];
  422. GetCurrentDirectory(_MAX_DIR, cpath);
  423. offsetPath.append(cpath);
  424. addPathSepChar(offsetPath);
  425. offsetPath.append(topic.c_str());
  426. offsetPath.append("-");
  427. offsetPath.append(partitionNum);
  428. if (!consumerGroup.empty())
  429. {
  430. offsetPath.append("-");
  431. offsetPath.append(consumerGroup.c_str());
  432. }
  433. offsetPath.append(".offset");
  434. }
  435. Consumer::~Consumer()
  436. {
  437. if (consumerPtr && topicPtr)
  438. {
  439. consumerPtr->stop(topicPtr, partitionNum);
  440. }
  441. delete(topicPtr.load());
  442. delete(consumerPtr);
  443. }
  444. RdKafka::Handle* Consumer::handle()
  445. {
  446. return static_cast<RdKafka::Handle*>(consumerPtr);
  447. }
  448. void Consumer::ensureSetup()
  449. {
  450. if (!topicPtr.load(std::memory_order_acquire))
  451. {
  452. CriticalBlock block(lock);
  453. if (!topicPtr.load(std::memory_order_relaxed))
  454. {
  455. initFileOffsetIfNotExist();
  456. std::string errStr;
  457. RdKafka::Conf* globalConfig = RdKafka::Conf::create(RdKafka::Conf::CONF_GLOBAL);
  458. if (globalConfig)
  459. {
  460. // Set global configuration parameters, used mainly at the consumer level
  461. globalConfig->set("metadata.broker.list", brokers, errStr);
  462. globalConfig->set("compression.codec", "snappy", errStr);
  463. globalConfig->set("queued.max.messages.kbytes", "10000000", errStr);
  464. globalConfig->set("fetch.message.max.bytes", "10000000", errStr);
  465. // Set any global configurations from file, allowing
  466. // overrides of above settings
  467. applyConfig(GLOBAL_CONFIG_FILENAME, globalConfig, traceLevel);
  468. // Set consumer callbacks
  469. globalConfig->set("event_cb", static_cast<RdKafka::EventCb*>(this), errStr);
  470. // Create the consumer
  471. consumerPtr = RdKafka::Consumer::create(globalConfig, errStr);
  472. delete globalConfig;
  473. if (consumerPtr)
  474. {
  475. RdKafka::Conf* topicConfPtr = RdKafka::Conf::create(RdKafka::Conf::CONF_TOPIC);
  476. // Set the per-topic configuration parameters
  477. topicConfPtr->set("group.id", consumerGroup, errStr);
  478. topicConfPtr->set("auto.offset.reset", "smallest", errStr);
  479. // Set any topic configurations from file, allowing
  480. // overrides of above settings
  481. std::string confName = "kafka_consumer_topic_" + topic + ".conf";
  482. applyConfig(confName.c_str(), topicConfPtr, traceLevel);
  483. // Ensure that some items are set a certain way
  484. // by setting them after loading the external conf
  485. topicConfPtr->set("auto.commit.enable", "false", errStr);
  486. // Additional settings for updated librdkafka
  487. topicConfPtr->set("enable.auto.commit", "false", errStr);
  488. topicConfPtr->set("offset.store.method", "file", errStr);
  489. topicConfPtr->set("offset.store.path", offsetPath.str(), errStr);
  490. // Create the topic
  491. topicPtr.store(RdKafka::Topic::create(consumerPtr, topic, topicConfPtr, errStr), std::memory_order_release);
  492. delete topicConfPtr;
  493. if (!topicPtr)
  494. {
  495. throw MakeStringException(-1, "Kafka: Unable to create consumer topic object for topic '%s'; error: '%s'", topic.c_str(), errStr.c_str());
  496. }
  497. }
  498. else
  499. {
  500. throw MakeStringException(-1, "Kafka: Unable to create consumer object for brokers '%s'; error: '%s'", brokers.c_str(), errStr.c_str());
  501. }
  502. }
  503. else
  504. {
  505. throw MakeStringException(-1, "Kafka: Unable to create consumer global configuration object for brokers '%s'; error: '%s'", brokers.c_str(), errStr.c_str());
  506. }
  507. }
  508. }
  509. }
  510. RdKafka::Message* Consumer::getOneMessage()
  511. {
  512. return consumerPtr->consume(topicPtr, partitionNum, POLL_TIMEOUT);
  513. }
  514. void Consumer::prepForMessageFetch()
  515. {
  516. // Make sure we have a valid connection to the Kafka cluster
  517. ensureSetup();
  518. // Start the local read queue
  519. RdKafka::ErrorCode startErr = consumerPtr->start(topicPtr, partitionNum, RdKafka::Topic::OFFSET_STORED);
  520. if (startErr == RdKafka::ERR_NO_ERROR)
  521. {
  522. if (traceLevel > 4)
  523. {
  524. DBGLOG("Kafka: Started Consumer for %s:%d @ %s", topic.c_str(), partitionNum, brokers.c_str());
  525. }
  526. }
  527. else
  528. {
  529. throw MakeStringException(-1, "Kafka: Failed to start Consumer read for %s:%d @ %s; error: %d", topic.c_str(), partitionNum, brokers.c_str(), startErr);
  530. }
  531. }
  532. void Consumer::commitOffset(__int64 offset) const
  533. {
  534. if (offset >= 0)
  535. {
  536. // Not using librdkafka's offset_store because it seems to be broken
  537. // topicPtr->offset_store(partitionNum, offset);
  538. // Create/overwrite a file using the same naming convention and
  539. // file contents that librdkafka uses so it can pick up where
  540. // we left off; NOTE: librdkafka does not clean the topic name
  541. // or consumer group name when constructing this path
  542. // (which is actually a security concern), so we can't clean, either
  543. std::ofstream outFile(offsetPath.str(), std::ofstream::trunc);
  544. outFile << offset;
  545. if (traceLevel > 4)
  546. {
  547. DBGLOG("Kafka: Saved offset %lld to %s", offset, offsetPath.str());
  548. }
  549. }
  550. }
  551. void Consumer::initFileOffsetIfNotExist() const
  552. {
  553. if (!checkFileExists(offsetPath.str()))
  554. {
  555. commitOffset(0);
  556. if (traceLevel > 4)
  557. {
  558. DBGLOG("Kafka: Creating initial offset file %s", offsetPath.str());
  559. }
  560. }
  561. }
  562. void Consumer::event_cb(RdKafka::Event& event)
  563. {
  564. if (traceLevel > 4)
  565. {
  566. switch (event.type())
  567. {
  568. case RdKafka::Event::EVENT_ERROR:
  569. DBGLOG("Kafka: Error: %s", event.str().c_str());
  570. break;
  571. case RdKafka::Event::EVENT_STATS:
  572. DBGLOG("Kafka: Stats: %s", event.str().c_str());
  573. break;
  574. case RdKafka::Event::EVENT_LOG:
  575. DBGLOG("Kafka: Log: %s", event.str().c_str());
  576. break;
  577. }
  578. }
  579. }
  580. //--------------------------------------------------------------------------
  581. /** @class PublisherCacheObj
  582. *
  583. * Class used to create and cache publisher objects and connections
  584. */
  585. static class PublisherCacheObj
  586. {
  587. private:
  588. typedef std::map<std::string, Publisher*> ObjMap;
  589. public:
  590. /**
  591. * Constructor
  592. *
  593. * @param _traceLevel The current logging level
  594. */
  595. PublisherCacheObj(int _traceLevel)
  596. : traceLevel(_traceLevel)
  597. {
  598. }
  599. void deleteAll()
  600. {
  601. CriticalBlock block(lock);
  602. for (ObjMap::iterator x = cachedPublishers.begin(); x != cachedPublishers.end(); x++)
  603. {
  604. if (x->second)
  605. {
  606. // Shutdown the attached poller before deleting
  607. x->second->shutdownPoller();
  608. // Now delete
  609. delete(x->second);
  610. }
  611. }
  612. cachedPublishers.clear();
  613. }
  614. /**
  615. * Remove previously-created objects that have been inactive
  616. * for awhile
  617. */
  618. void expire()
  619. {
  620. if (!cachedPublishers.empty())
  621. {
  622. CriticalBlock block(lock);
  623. time_t oldestAllowedTime = time(NULL) - OBJECT_EXPIRE_TIMEOUT_SECONDS;
  624. __int32 expireCount = 0;
  625. for (ObjMap::iterator x = cachedPublishers.begin(); x != cachedPublishers.end(); /* increment handled explicitly */)
  626. {
  627. // Expire only if the publisher has been inactive and if
  628. // there are no messages in the outbound queue
  629. if (x->second && x->second->getTimeTouched() < oldestAllowedTime && x->second->messagesWaitingInQueue() == 0)
  630. {
  631. // Shutdown the attached poller before deleting
  632. x->second->shutdownPoller();
  633. // Delete the object
  634. delete(x->second);
  635. // Erase from map
  636. cachedPublishers.erase(x++);
  637. ++expireCount;
  638. }
  639. else
  640. {
  641. x++;
  642. }
  643. }
  644. if (traceLevel > 4 && expireCount > 0)
  645. {
  646. DBGLOG("Kafka: Expired %d cached publisher%s", expireCount, (expireCount == 1 ? "" : "s"));
  647. }
  648. }
  649. }
  650. /**
  651. * Gets an established Publisher, based on unique broker/topic
  652. * pairs, or creates a new one.
  653. *
  654. * @param brokers One or more Kafka brokers, in the
  655. * format 'name[:port]' where 'name'
  656. * is either a host name or IP address;
  657. * multiple brokers can be delimited
  658. * with commas
  659. * @param topic The name of the topic
  660. * @param pollTimeout The number of milliseconds to give
  661. * to librdkafka when executing
  662. * asynchronous activities
  663. *
  664. * @return A pointer to a Publisher* object.
  665. */
  666. Publisher* getPublisher(const std::string& brokers, const std::string& topic, __int32 pollTimeout)
  667. {
  668. Publisher* pubObjPtr = NULL;
  669. StringBuffer suffixStr;
  670. std::string key;
  671. // Create the key used to look up previously-created objects
  672. suffixStr.append(pollTimeout);
  673. key = brokers + "+" + topic + "+" + suffixStr.str();
  674. {
  675. CriticalBlock block(lock);
  676. // Try to find a cached publisher
  677. pubObjPtr = cachedPublishers[key];
  678. if (pubObjPtr)
  679. {
  680. pubObjPtr->updateTimeTouched();
  681. }
  682. else
  683. {
  684. // Publisher for that set of brokers and topic does not exist; create one
  685. pubObjPtr = new Publisher(brokers, topic, pollTimeout, traceLevel);
  686. cachedPublishers[key] = pubObjPtr;
  687. if (traceLevel > 4)
  688. {
  689. DBGLOG("Kafka: Created and cached new publisher object: %s @ %s", topic.c_str(), brokers.c_str());
  690. }
  691. }
  692. }
  693. if (!pubObjPtr)
  694. {
  695. throw MakeStringException(-1, "Kafka: Unable to create publisher for brokers '%s' and topic '%s'", brokers.c_str(), topic.c_str());
  696. }
  697. return pubObjPtr;
  698. }
  699. private:
  700. ObjMap cachedPublishers; //!< std::map of created Publisher object pointers
  701. CriticalSection lock; //!< Mutex guarding modifications to cachedPublishers
  702. int traceLevel; //!< The current logging level
  703. } *publisherCache;
  704. //--------------------------------------------------------------------------
  705. /** @class PublisherCacheExpirerObj
  706. * Class used to expire old publisher objects held within publisherCache
  707. */
  708. static class PublisherCacheExpirerObj : public Thread
  709. {
  710. public:
  711. PublisherCacheExpirerObj()
  712. : Thread("Kafka::PublisherExpirer"),
  713. shouldRun(false)
  714. {
  715. }
  716. virtual void start()
  717. {
  718. if (!isAlive())
  719. {
  720. shouldRun = true;
  721. Thread::start();
  722. }
  723. }
  724. virtual void stop()
  725. {
  726. if (isAlive())
  727. {
  728. shouldRun = false;
  729. join();
  730. }
  731. }
  732. virtual int run()
  733. {
  734. while (shouldRun)
  735. {
  736. if (publisherCache)
  737. {
  738. publisherCache->expire();
  739. }
  740. usleep(1000);
  741. }
  742. return 0;
  743. }
  744. private:
  745. std::atomic_bool shouldRun; //!< If true, we should execute our thread's main event loop
  746. } *publisherCacheExpirer;
  747. //--------------------------------------------------------------------------
  748. // Lazy Initialization
  749. //--------------------------------------------------------------------------
  750. /**
  751. * Make sure the publisher object cache is initialized as well as the
  752. * associated background thread for expiring idle publishers. This is
  753. * called only once.
  754. *
  755. * @param traceLevel Current logging level
  756. */
  757. static void setupPublisherCache(int traceLevel)
  758. {
  759. KafkaPlugin::publisherCache = new KafkaPlugin::PublisherCacheObj(traceLevel);
  760. KafkaPlugin::publisherCacheExpirer = new KafkaPlugin::PublisherCacheExpirerObj;
  761. KafkaPlugin::publisherCacheExpirer->start();
  762. }
  763. //--------------------------------------------------------------------------
  764. // Advertised Entry Point Functions
  765. //--------------------------------------------------------------------------
  766. ECL_KAFKA_API bool ECL_KAFKA_CALL publishMessage(ICodeContext* ctx, const char* brokers, const char* topic, const char* message, const char* key)
  767. {
  768. std::call_once(pubCacheInitFlag, setupPublisherCache, ctx->queryContextLogger().queryTraceLevel());
  769. Publisher* pubObjPtr = publisherCache->getPublisher(brokers, topic, POLL_TIMEOUT);
  770. pubObjPtr->sendMessage(message, key);
  771. return true;
  772. }
  773. ECL_KAFKA_API bool ECL_KAFKA_CALL publishMessage(ICodeContext* ctx, const char* brokers, const char* topic, size32_t lenMessage, const char* message, size32_t lenKey, const char* key)
  774. {
  775. std::call_once(pubCacheInitFlag, setupPublisherCache, ctx->queryContextLogger().queryTraceLevel());
  776. Publisher* pubObjPtr = publisherCache->getPublisher(brokers, topic, POLL_TIMEOUT);
  777. std::string messageStr(message, rtlUtf8Size(lenMessage, message));
  778. std::string keyStr(key, rtlUtf8Size(lenKey, key));
  779. pubObjPtr->sendMessage(messageStr, keyStr);
  780. return true;
  781. }
  782. ECL_KAFKA_API __int32 ECL_KAFKA_CALL getTopicPartitionCount(ICodeContext* ctx, const char* brokers, const char* topic)
  783. {
  784. // We have to use librdkafka's C API for this right now, as the C++ API
  785. // does not expose a topic's metadata. In addition, there is no easy
  786. // link between the exposed C++ objects and the structs used by the
  787. // C API, so we are basically creating a brand-new connection from
  788. // scratch.
  789. int traceLevel = ctx->queryContextLogger().queryTraceLevel();
  790. __int32 pCount = 0;
  791. char errstr[512];
  792. RdKafka::Conf* globalConfig = RdKafka::Conf::create(RdKafka::Conf::CONF_GLOBAL);
  793. if (globalConfig)
  794. {
  795. // Load global config to pick up any protocol modifications
  796. applyConfig(GLOBAL_CONFIG_FILENAME, globalConfig, traceLevel);
  797. // rd_kafka_new() takes ownership of the lower-level conf object, which in this case is a
  798. // pointer currently owned by globalConfig; we need to pass a duplicate
  799. // the conf pointer to rd_kafka_new() so we don't mangle globalConfig's internals
  800. rd_kafka_conf_t* conf = rd_kafka_conf_dup(globalConfig->c_ptr_global());
  801. rd_kafka_t* rk = rd_kafka_new(RD_KAFKA_CONSUMER, conf, errstr, sizeof(errstr));
  802. delete globalConfig;
  803. if (rk)
  804. {
  805. if (rd_kafka_brokers_add(rk, brokers) != 0)
  806. {
  807. rd_kafka_topic_conf_t* topic_conf = rd_kafka_topic_conf_new();
  808. rd_kafka_topic_t* rkt = rd_kafka_topic_new(rk, topic, topic_conf);
  809. if (rkt)
  810. {
  811. const struct rd_kafka_metadata* metadata = NULL;
  812. rd_kafka_resp_err_t err = rd_kafka_metadata(rk, 0, rkt, &metadata, 5000);
  813. if (err == RD_KAFKA_RESP_ERR_NO_ERROR)
  814. {
  815. pCount = metadata->topics[0].partition_cnt;
  816. rd_kafka_metadata_destroy(metadata);
  817. }
  818. else
  819. {
  820. DBGLOG("Kafka: Error retrieving metadata from topic: %s @ %s: '%s'", topic, brokers, rd_kafka_err2str(err));
  821. }
  822. rd_kafka_topic_destroy(rkt);
  823. }
  824. else
  825. {
  826. if (traceLevel > 4)
  827. {
  828. DBGLOG("Kafka: Could not create topic configuration object: %s @ %s", topic, brokers);
  829. }
  830. }
  831. }
  832. else
  833. {
  834. if (traceLevel > 4)
  835. {
  836. DBGLOG("Kafka: Could not add brokers: %s @ %s", topic, brokers);
  837. }
  838. }
  839. rd_kafka_destroy(rk);
  840. }
  841. else
  842. {
  843. DBGLOG("Kafka: Could not create consumer configuration object : %s @ %s: '%s'", topic, brokers, errstr);
  844. }
  845. }
  846. else
  847. {
  848. if (traceLevel > 4)
  849. {
  850. DBGLOG("Kafka: Could not create global configuration object: %s @ %s", topic, brokers);
  851. }
  852. }
  853. if (pCount == 0)
  854. {
  855. DBGLOG("Kafka: Unable to retrieve partition count from topic: %s @ %s", topic, brokers);
  856. }
  857. return pCount;
  858. }
  859. ECL_KAFKA_API IRowStream* ECL_KAFKA_CALL getMessageDataset(ICodeContext* ctx, IEngineRowAllocator* allocator, const char* brokers, const char* topic, const char* consumerGroup, __int32 partitionNum, __int64 maxRecords)
  860. {
  861. Consumer* consumerObjPtr = new Consumer(brokers, topic, consumerGroup, partitionNum, ctx->queryContextLogger().queryTraceLevel());
  862. try
  863. {
  864. consumerObjPtr->prepForMessageFetch();
  865. }
  866. catch(...)
  867. {
  868. delete(consumerObjPtr);
  869. throw;
  870. }
  871. return new KafkaStreamedDataset(consumerObjPtr, allocator, ctx->queryContextLogger().queryTraceLevel(), maxRecords);
  872. }
  873. ECL_KAFKA_API __int64 ECL_KAFKA_CALL setMessageOffset(ICodeContext* ctx, const char* brokers, const char* topic, const char* consumerGroup, __int32 partitionNum, __int64 newOffset)
  874. {
  875. Consumer consumerObj(brokers, topic, consumerGroup, partitionNum, ctx->queryContextLogger().queryTraceLevel());
  876. consumerObj.commitOffset(newOffset);
  877. return newOffset;
  878. }
  879. }
  880. //==============================================================================
  881. // Plugin Initialization and Teardown
  882. //==============================================================================
  883. #define CURRENT_KAFKA_VERSION "kafka plugin 1.1.0"
  884. static const char* kafkaCompatibleVersions[] = {
  885. "kafka plugin 1.0.0",
  886. CURRENT_KAFKA_VERSION,
  887. NULL };
  888. ECL_KAFKA_API bool getECLPluginDefinition(ECLPluginDefinitionBlock* pb)
  889. {
  890. if (pb->size == sizeof(ECLPluginDefinitionBlockEx))
  891. {
  892. ECLPluginDefinitionBlockEx* pbx = static_cast<ECLPluginDefinitionBlockEx*>(pb);
  893. pbx->compatibleVersions = kafkaCompatibleVersions;
  894. }
  895. else if (pb->size != sizeof(ECLPluginDefinitionBlock))
  896. {
  897. return false;
  898. }
  899. pb->magicVersion = PLUGIN_VERSION;
  900. pb->version = CURRENT_KAFKA_VERSION;
  901. pb->moduleName = "kafka";
  902. pb->ECL = NULL;
  903. pb->flags = PLUGIN_IMPLICIT_MODULE;
  904. pb->description = "ECL plugin library for the C++ API in librdkafka++";
  905. return true;
  906. }
  907. MODULE_INIT(INIT_PRIORITY_STANDARD)
  908. {
  909. KafkaPlugin::publisherCache = NULL;
  910. KafkaPlugin::publisherCacheExpirer = NULL;
  911. return true;
  912. }
  913. MODULE_EXIT()
  914. {
  915. // Delete the background thread expiring items from the publisher cache
  916. // before deleting the publisher cache
  917. if (KafkaPlugin::publisherCacheExpirer)
  918. {
  919. KafkaPlugin::publisherCacheExpirer->stop();
  920. delete(KafkaPlugin::publisherCacheExpirer);
  921. KafkaPlugin::publisherCacheExpirer = NULL;
  922. }
  923. if (KafkaPlugin::publisherCache)
  924. {
  925. KafkaPlugin::publisherCache->deleteAll();
  926. delete(KafkaPlugin::publisherCache);
  927. KafkaPlugin::publisherCache = NULL;
  928. }
  929. RdKafka::wait_destroyed(3000);
  930. }