kafka.cpp 38 KB

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