client.go 41 KB

12345678910111213141516171819202122232425262728293031323334353637383940414243444546474849505152535455565758596061626364656667686970717273747576777879808182838485868788899091929394959697989910010110210310410510610710810911011111211311411511611711811912012112212312412512612712812913013113213313413513613713813914014114214314414514614714814915015115215315415515615715815916016116216316416516616716816917017117217317417517617717817918018118218318418518618718818919019119219319419519619719819920020120220320420520620720820921021121221321421521621721821922022122222322422522622722822923023123223323423523623723823924024124224324424524624724824925025125225325425525625725825926026126226326426526626726826927027127227327427527627727827928028128228328428528628728828929029129229329429529629729829930030130230330430530630730830931031131231331431531631731831932032132232332432532632732832933033133233333433533633733833934034134234334434534634734834935035135235335435535635735835936036136236336436536636736836937037137237337437537637737837938038138238338438538638738838939039139239339439539639739839940040140240340440540640740840941041141241341441541641741841942042142242342442542642742842943043143243343443543643743843944044144244344444544644744844945045145245345445545645745845946046146246346446546646746846947047147247347447547647747847948048148248348448548648748848949049149249349449549649749849950050150250350450550650750850951051151251351451551651751851952052152252352452552652752852953053153253353453553653753853954054154254354454554654754854955055155255355455555655755855956056156256356456556656756856957057157257357457557657757857958058158258358458558658758858959059159259359459559659759859960060160260360460560660760860961061161261361461561661761861962062162262362462562662762862963063163263363463563663763863964064164264364464564664764864965065165265365465565665765865966066166266366466566666766866967067167267367467567667767867968068168268368468568668768868969069169269369469569669769869970070170270370470570670770870971071171271371471571671771871972072172272372472572672772872973073173273373473573673773873974074174274374474574674774874975075175275375475575675775875976076176276376476576676776876977077177277377477577677777877978078178278378478578678778878979079179279379479579679779879980080180280380480580680780880981081181281381481581681781881982082182282382482582682782882983083183283383483583683783883984084184284384484584684784884985085185285385485585685785885986086186286386486586686786886987087187287387487587687787887988088188288388488588688788888989089189289389489589689789889990090190290390490590690790890991091191291391491591691791891992092192292392492592692792892993093193293393493593693793893994094194294394494594694794894995095195295395495595695795895996096196296396496596696796896997097197297397497597697797897998098198298398498598698798898999099199299399499599699799899910001001100210031004100510061007100810091010101110121013101410151016101710181019102010211022102310241025102610271028102910301031103210331034103510361037103810391040104110421043104410451046104710481049105010511052105310541055105610571058105910601061106210631064106510661067106810691070107110721073107410751076107710781079108010811082108310841085108610871088108910901091109210931094109510961097109810991100110111021103110411051106110711081109111011111112111311141115111611171118111911201121112211231124112511261127112811291130113111321133113411351136113711381139114011411142114311441145114611471148114911501151115211531154115511561157115811591160116111621163116411651166116711681169117011711172117311741175117611771178117911801181118211831184118511861187118811891190119111921193119411951196119711981199120012011202120312041205120612071208120912101211121212131214121512161217121812191220122112221223122412251226122712281229123012311232123312341235123612371238123912401241124212431244124512461247124812491250125112521253125412551256125712581259126012611262126312641265126612671268126912701271127212731274127512761277127812791280128112821283128412851286128712881289129012911292129312941295129612971298129913001301130213031304130513061307130813091310131113121313131413151316131713181319132013211322132313241325132613271328132913301331
  1. package sarama
  2. import (
  3. "context"
  4. "errors"
  5. "math"
  6. "math/rand"
  7. "net"
  8. "sort"
  9. "strings"
  10. "sync"
  11. "sync/atomic"
  12. "time"
  13. "golang.org/x/net/proxy"
  14. )
  15. // Client is a generic Kafka client. It manages connections to one or more Kafka brokers.
  16. // You MUST call Close() on a client to avoid leaks, it will not be garbage-collected
  17. // automatically when it passes out of scope. It is safe to share a client amongst many
  18. // users, however Kafka will process requests from a single client strictly in serial,
  19. // so it is generally more efficient to use the default one client per producer/consumer.
  20. type Client interface {
  21. // Config returns the Config struct of the client. This struct should not be
  22. // altered after it has been created.
  23. Config() *Config
  24. // Controller returns the cluster controller broker. It will return a
  25. // locally cached value if it's available. You can call RefreshController
  26. // to update the cached value. Requires Kafka 0.10 or higher.
  27. Controller() (*Broker, error)
  28. // RefreshController retrieves the cluster controller from fresh metadata
  29. // and stores it in the local cache. Requires Kafka 0.10 or higher.
  30. RefreshController() (*Broker, error)
  31. // Brokers returns the current set of active brokers as retrieved from cluster metadata.
  32. Brokers() []*Broker
  33. // Broker returns the active Broker if available for the broker ID.
  34. Broker(brokerID int32) (*Broker, error)
  35. // Topics returns the set of available topics as retrieved from cluster metadata.
  36. Topics() ([]string, error)
  37. // Partitions returns the sorted list of all partition IDs for the given topic.
  38. Partitions(topic string) ([]int32, error)
  39. // WritablePartitions returns the sorted list of all writable partition IDs for
  40. // the given topic, where "writable" means "having a valid leader accepting
  41. // writes".
  42. WritablePartitions(topic string) ([]int32, error)
  43. // Leader returns the broker object that is the leader of the current
  44. // topic/partition, as determined by querying the cluster metadata.
  45. Leader(topic string, partitionID int32) (*Broker, error)
  46. // LeaderAndEpoch returns the leader and its epoch for the current
  47. // topic/partition, as determined by querying the cluster metadata.
  48. LeaderAndEpoch(topic string, partitionID int32) (*Broker, int32, error)
  49. // Replicas returns the set of all replica IDs for the given partition.
  50. Replicas(topic string, partitionID int32) ([]int32, error)
  51. // InSyncReplicas returns the set of all in-sync replica IDs for the given
  52. // partition. In-sync replicas are replicas which are fully caught up with
  53. // the partition leader.
  54. InSyncReplicas(topic string, partitionID int32) ([]int32, error)
  55. // OfflineReplicas returns the set of all offline replica IDs for the given
  56. // partition. Offline replicas are replicas which are offline
  57. OfflineReplicas(topic string, partitionID int32) ([]int32, error)
  58. // RefreshBrokers takes a list of addresses to be used as seed brokers.
  59. // Existing broker connections are closed and the updated list of seed brokers
  60. // will be used for the next metadata fetch.
  61. RefreshBrokers(addrs []string) error
  62. // RefreshMetadata takes a list of topics and queries the cluster to refresh the
  63. // available metadata for those topics. If no topics are provided, it will refresh
  64. // metadata for all topics.
  65. RefreshMetadata(topics ...string) error
  66. // GetOffset queries the cluster to get the most recent available offset at the
  67. // given time (in milliseconds) on the topic/partition combination.
  68. // Time should be OffsetOldest for the earliest available offset,
  69. // OffsetNewest for the offset of the message that will be produced next, or a time.
  70. GetOffset(topic string, partitionID int32, time int64) (int64, error)
  71. // Coordinator returns the coordinating broker for a consumer group. It will
  72. // return a locally cached value if it's available. You can call
  73. // RefreshCoordinator to update the cached value. This function only works on
  74. // Kafka 0.8.2 and higher.
  75. Coordinator(consumerGroup string) (*Broker, error)
  76. // RefreshCoordinator retrieves the coordinator for a consumer group and stores it
  77. // in local cache. This function only works on Kafka 0.8.2 and higher.
  78. RefreshCoordinator(consumerGroup string) error
  79. // Coordinator returns the coordinating broker for a transaction id. It will
  80. // return a locally cached value if it's available. You can call
  81. // RefreshCoordinator to update the cached value. This function only works on
  82. // Kafka 0.11.0.0 and higher.
  83. TransactionCoordinator(transactionID string) (*Broker, error)
  84. // RefreshCoordinator retrieves the coordinator for a transaction id and stores it
  85. // in local cache. This function only works on Kafka 0.11.0.0 and higher.
  86. RefreshTransactionCoordinator(transactionID string) error
  87. // InitProducerID retrieves information required for Idempotent Producer
  88. InitProducerID() (*InitProducerIDResponse, error)
  89. // LeastLoadedBroker retrieves broker that has the least responses pending
  90. LeastLoadedBroker() *Broker
  91. // Close shuts down all broker connections managed by this client. It is required
  92. // to call this function before a client object passes out of scope, as it will
  93. // otherwise leak memory. You must close any Producers or Consumers using a client
  94. // before you close the client.
  95. Close() error
  96. // Closed returns true if the client has already had Close called on it
  97. Closed() bool
  98. }
  99. const (
  100. // OffsetNewest stands for the log head offset, i.e. the offset that will be
  101. // assigned to the next message that will be produced to the partition. You
  102. // can send this to a client's GetOffset method to get this offset, or when
  103. // calling ConsumePartition to start consuming new messages.
  104. OffsetNewest int64 = -1
  105. // OffsetOldest stands for the oldest offset available on the broker for a
  106. // partition. You can send this to a client's GetOffset method to get this
  107. // offset, or when calling ConsumePartition to start consuming from the
  108. // oldest offset that is still available on the broker.
  109. OffsetOldest int64 = -2
  110. )
  111. type client struct {
  112. // updateMetadataMs stores the time at which metadata was lasted updated.
  113. // Note: this accessed atomically so must be the first word in the struct
  114. // as per golang/go#41970
  115. updateMetadataMs int64
  116. conf *Config
  117. closer, closed chan none // for shutting down background metadata updater
  118. // the broker addresses given to us through the constructor are not guaranteed to be returned in
  119. // the cluster metadata (I *think* it only returns brokers who are currently leading partitions?)
  120. // so we store them separately
  121. seedBrokers []*Broker
  122. deadSeeds []*Broker
  123. controllerID int32 // cluster controller broker id
  124. brokers map[int32]*Broker // maps broker ids to brokers
  125. metadata map[string]map[int32]*PartitionMetadata // maps topics to partition ids to metadata
  126. metadataTopics map[string]none // topics that need to collect metadata
  127. coordinators map[string]int32 // Maps consumer group names to coordinating broker IDs
  128. transactionCoordinators map[string]int32 // Maps transaction ids to coordinating broker IDs
  129. // If the number of partitions is large, we can get some churn calling cachedPartitions,
  130. // so the result is cached. It is important to update this value whenever metadata is changed
  131. cachedPartitionsResults map[string][maxPartitionIndex][]int32
  132. lock sync.RWMutex // protects access to the maps that hold cluster state.
  133. }
  134. // NewClient creates a new Client. It connects to one of the given broker addresses
  135. // and uses that broker to automatically fetch metadata on the rest of the kafka cluster. If metadata cannot
  136. // be retrieved from any of the given broker addresses, the client is not created.
  137. func NewClient(addrs []string, conf *Config) (Client, error) {
  138. DebugLogger.Println("Initializing new client")
  139. if conf == nil {
  140. conf = NewConfig()
  141. }
  142. if err := conf.Validate(); err != nil {
  143. return nil, err
  144. }
  145. if len(addrs) < 1 {
  146. return nil, ConfigurationError("You must provide at least one broker address")
  147. }
  148. if strings.Contains(addrs[0], ".servicebus.windows.net") {
  149. if conf.Version.IsAtLeast(V1_1_0_0) || !conf.Version.IsAtLeast(V0_11_0_0) {
  150. Logger.Println("Connecting to Azure Event Hubs, forcing version to V1_0_0_0 for compatibility")
  151. conf.Version = V1_0_0_0
  152. }
  153. }
  154. client := &client{
  155. conf: conf,
  156. closer: make(chan none),
  157. closed: make(chan none),
  158. brokers: make(map[int32]*Broker),
  159. metadata: make(map[string]map[int32]*PartitionMetadata),
  160. metadataTopics: make(map[string]none),
  161. cachedPartitionsResults: make(map[string][maxPartitionIndex][]int32),
  162. coordinators: make(map[string]int32),
  163. transactionCoordinators: make(map[string]int32),
  164. }
  165. if conf.Net.ResolveCanonicalBootstrapServers {
  166. var err error
  167. addrs, err = client.resolveCanonicalNames(addrs)
  168. if err != nil {
  169. return nil, err
  170. }
  171. }
  172. client.randomizeSeedBrokers(addrs)
  173. if conf.Metadata.Full {
  174. // do an initial fetch of all cluster metadata by specifying an empty list of topics
  175. err := client.RefreshMetadata()
  176. if err == nil {
  177. } else if errors.Is(err, ErrLeaderNotAvailable) || errors.Is(err, ErrReplicaNotAvailable) || errors.Is(err, ErrTopicAuthorizationFailed) || errors.Is(err, ErrClusterAuthorizationFailed) {
  178. // indicates that maybe part of the cluster is down, but is not fatal to creating the client
  179. Logger.Println(err)
  180. } else {
  181. close(client.closed) // we haven't started the background updater yet, so we have to do this manually
  182. _ = client.Close()
  183. return nil, err
  184. }
  185. }
  186. go withRecover(client.backgroundMetadataUpdater)
  187. DebugLogger.Println("Successfully initialized new client")
  188. return client, nil
  189. }
  190. func (client *client) Config() *Config {
  191. return client.conf
  192. }
  193. func (client *client) Brokers() []*Broker {
  194. client.lock.RLock()
  195. defer client.lock.RUnlock()
  196. brokers := make([]*Broker, 0, len(client.brokers))
  197. for _, broker := range client.brokers {
  198. brokers = append(brokers, broker)
  199. }
  200. return brokers
  201. }
  202. func (client *client) Broker(brokerID int32) (*Broker, error) {
  203. client.lock.RLock()
  204. defer client.lock.RUnlock()
  205. broker, ok := client.brokers[brokerID]
  206. if !ok {
  207. return nil, ErrBrokerNotFound
  208. }
  209. _ = broker.Open(client.conf)
  210. return broker, nil
  211. }
  212. func (client *client) InitProducerID() (*InitProducerIDResponse, error) {
  213. // FIXME: this InitProducerID seems to only be called from client_test.go (TestInitProducerIDConnectionRefused) and has been superceded by transaction_manager.go?
  214. brokerErrors := make([]error, 0)
  215. for broker := client.LeastLoadedBroker(); broker != nil; broker = client.LeastLoadedBroker() {
  216. request := &InitProducerIDRequest{}
  217. if client.conf.Version.IsAtLeast(V2_7_0_0) {
  218. // Version 4 adds the support for new error code PRODUCER_FENCED.
  219. request.Version = 4
  220. } else if client.conf.Version.IsAtLeast(V2_5_0_0) {
  221. // Version 3 adds ProducerId and ProducerEpoch, allowing producers to try to resume after an INVALID_PRODUCER_EPOCH error
  222. request.Version = 3
  223. } else if client.conf.Version.IsAtLeast(V2_4_0_0) {
  224. // Version 2 is the first flexible version.
  225. request.Version = 2
  226. } else if client.conf.Version.IsAtLeast(V2_0_0_0) {
  227. // Version 1 is the same as version 0.
  228. request.Version = 1
  229. }
  230. response, err := broker.InitProducerID(request)
  231. if err == nil {
  232. return response, nil
  233. } else {
  234. // some error, remove that broker and try again
  235. Logger.Printf("Client got error from broker %d when issuing InitProducerID : %v\n", broker.ID(), err)
  236. _ = broker.Close()
  237. brokerErrors = append(brokerErrors, err)
  238. client.deregisterBroker(broker)
  239. }
  240. }
  241. return nil, Wrap(ErrOutOfBrokers, brokerErrors...)
  242. }
  243. func (client *client) Close() error {
  244. if client.Closed() {
  245. // Chances are this is being called from a defer() and the error will go unobserved
  246. // so we go ahead and log the event in this case.
  247. Logger.Printf("Close() called on already closed client")
  248. return ErrClosedClient
  249. }
  250. // shutdown and wait for the background thread before we take the lock, to avoid races
  251. close(client.closer)
  252. <-client.closed
  253. client.lock.Lock()
  254. defer client.lock.Unlock()
  255. DebugLogger.Println("Closing Client")
  256. for _, broker := range client.brokers {
  257. safeAsyncClose(broker)
  258. }
  259. for _, broker := range client.seedBrokers {
  260. safeAsyncClose(broker)
  261. }
  262. client.brokers = nil
  263. client.metadata = nil
  264. client.metadataTopics = nil
  265. return nil
  266. }
  267. func (client *client) Closed() bool {
  268. client.lock.RLock()
  269. defer client.lock.RUnlock()
  270. return client.brokers == nil
  271. }
  272. func (client *client) Topics() ([]string, error) {
  273. if client.Closed() {
  274. return nil, ErrClosedClient
  275. }
  276. client.lock.RLock()
  277. defer client.lock.RUnlock()
  278. ret := make([]string, 0, len(client.metadata))
  279. for topic := range client.metadata {
  280. ret = append(ret, topic)
  281. }
  282. return ret, nil
  283. }
  284. func (client *client) MetadataTopics() ([]string, error) {
  285. if client.Closed() {
  286. return nil, ErrClosedClient
  287. }
  288. client.lock.RLock()
  289. defer client.lock.RUnlock()
  290. ret := make([]string, 0, len(client.metadataTopics))
  291. for topic := range client.metadataTopics {
  292. ret = append(ret, topic)
  293. }
  294. return ret, nil
  295. }
  296. func (client *client) Partitions(topic string) ([]int32, error) {
  297. if client.Closed() {
  298. return nil, ErrClosedClient
  299. }
  300. partitions := client.cachedPartitions(topic, allPartitions)
  301. if len(partitions) == 0 {
  302. err := client.RefreshMetadata(topic)
  303. if err != nil {
  304. return nil, err
  305. }
  306. partitions = client.cachedPartitions(topic, allPartitions)
  307. }
  308. // no partitions found after refresh metadata
  309. if len(partitions) == 0 {
  310. return nil, ErrUnknownTopicOrPartition
  311. }
  312. return partitions, nil
  313. }
  314. func (client *client) WritablePartitions(topic string) ([]int32, error) {
  315. if client.Closed() {
  316. return nil, ErrClosedClient
  317. }
  318. partitions := client.cachedPartitions(topic, writablePartitions)
  319. // len==0 catches when it's nil (no such topic) and the odd case when every single
  320. // partition is undergoing leader election simultaneously. Callers have to be able to handle
  321. // this function returning an empty slice (which is a valid return value) but catching it
  322. // here the first time (note we *don't* catch it below where we return ErrUnknownTopicOrPartition) triggers
  323. // a metadata refresh as a nicety so callers can just try again and don't have to manually
  324. // trigger a refresh (otherwise they'd just keep getting a stale cached copy).
  325. if len(partitions) == 0 {
  326. err := client.RefreshMetadata(topic)
  327. if err != nil {
  328. return nil, err
  329. }
  330. partitions = client.cachedPartitions(topic, writablePartitions)
  331. }
  332. if partitions == nil {
  333. return nil, ErrUnknownTopicOrPartition
  334. }
  335. return partitions, nil
  336. }
  337. func (client *client) Replicas(topic string, partitionID int32) ([]int32, error) {
  338. if client.Closed() {
  339. return nil, ErrClosedClient
  340. }
  341. metadata := client.cachedMetadata(topic, partitionID)
  342. if metadata == nil {
  343. err := client.RefreshMetadata(topic)
  344. if err != nil {
  345. return nil, err
  346. }
  347. metadata = client.cachedMetadata(topic, partitionID)
  348. }
  349. if metadata == nil {
  350. return nil, ErrUnknownTopicOrPartition
  351. }
  352. if errors.Is(metadata.Err, ErrReplicaNotAvailable) {
  353. return dupInt32Slice(metadata.Replicas), metadata.Err
  354. }
  355. return dupInt32Slice(metadata.Replicas), nil
  356. }
  357. func (client *client) InSyncReplicas(topic string, partitionID int32) ([]int32, error) {
  358. if client.Closed() {
  359. return nil, ErrClosedClient
  360. }
  361. metadata := client.cachedMetadata(topic, partitionID)
  362. if metadata == nil {
  363. err := client.RefreshMetadata(topic)
  364. if err != nil {
  365. return nil, err
  366. }
  367. metadata = client.cachedMetadata(topic, partitionID)
  368. }
  369. if metadata == nil {
  370. return nil, ErrUnknownTopicOrPartition
  371. }
  372. if errors.Is(metadata.Err, ErrReplicaNotAvailable) {
  373. return dupInt32Slice(metadata.Isr), metadata.Err
  374. }
  375. return dupInt32Slice(metadata.Isr), nil
  376. }
  377. func (client *client) OfflineReplicas(topic string, partitionID int32) ([]int32, error) {
  378. if client.Closed() {
  379. return nil, ErrClosedClient
  380. }
  381. metadata := client.cachedMetadata(topic, partitionID)
  382. if metadata == nil {
  383. err := client.RefreshMetadata(topic)
  384. if err != nil {
  385. return nil, err
  386. }
  387. metadata = client.cachedMetadata(topic, partitionID)
  388. }
  389. if metadata == nil {
  390. return nil, ErrUnknownTopicOrPartition
  391. }
  392. if errors.Is(metadata.Err, ErrReplicaNotAvailable) {
  393. return dupInt32Slice(metadata.OfflineReplicas), metadata.Err
  394. }
  395. return dupInt32Slice(metadata.OfflineReplicas), nil
  396. }
  397. func (client *client) Leader(topic string, partitionID int32) (*Broker, error) {
  398. leader, _, err := client.LeaderAndEpoch(topic, partitionID)
  399. return leader, err
  400. }
  401. func (client *client) LeaderAndEpoch(topic string, partitionID int32) (*Broker, int32, error) {
  402. if client.Closed() {
  403. return nil, -1, ErrClosedClient
  404. }
  405. leader, epoch, err := client.cachedLeader(topic, partitionID)
  406. if leader == nil {
  407. err = client.RefreshMetadata(topic)
  408. if err != nil {
  409. return nil, -1, err
  410. }
  411. leader, epoch, err = client.cachedLeader(topic, partitionID)
  412. }
  413. return leader, epoch, err
  414. }
  415. func (client *client) RefreshBrokers(addrs []string) error {
  416. if client.Closed() {
  417. return ErrClosedClient
  418. }
  419. client.lock.Lock()
  420. defer client.lock.Unlock()
  421. for _, broker := range client.brokers {
  422. safeAsyncClose(broker)
  423. }
  424. client.brokers = make(map[int32]*Broker)
  425. for _, broker := range client.seedBrokers {
  426. safeAsyncClose(broker)
  427. }
  428. for _, broker := range client.deadSeeds {
  429. safeAsyncClose(broker)
  430. }
  431. client.seedBrokers = nil
  432. client.deadSeeds = nil
  433. client.randomizeSeedBrokers(addrs)
  434. return nil
  435. }
  436. func (client *client) RefreshMetadata(topics ...string) error {
  437. if client.Closed() {
  438. return ErrClosedClient
  439. }
  440. // Prior to 0.8.2, Kafka will throw exceptions on an empty topic and not return a proper
  441. // error. This handles the case by returning an error instead of sending it
  442. // off to Kafka. See: https://github.com/IBM/sarama/pull/38#issuecomment-26362310
  443. for _, topic := range topics {
  444. if topic == "" {
  445. return ErrInvalidTopic // this is the error that 0.8.2 and later correctly return
  446. }
  447. }
  448. deadline := time.Time{}
  449. if client.conf.Metadata.Timeout > 0 {
  450. deadline = time.Now().Add(client.conf.Metadata.Timeout)
  451. }
  452. return client.tryRefreshMetadata(topics, client.conf.Metadata.Retry.Max, deadline)
  453. }
  454. func (client *client) GetOffset(topic string, partitionID int32, timestamp int64) (int64, error) {
  455. if client.Closed() {
  456. return -1, ErrClosedClient
  457. }
  458. offset, err := client.getOffset(topic, partitionID, timestamp)
  459. if err != nil {
  460. if err := client.RefreshMetadata(topic); err != nil {
  461. return -1, err
  462. }
  463. return client.getOffset(topic, partitionID, timestamp)
  464. }
  465. return offset, err
  466. }
  467. func (client *client) Controller() (*Broker, error) {
  468. if client.Closed() {
  469. return nil, ErrClosedClient
  470. }
  471. if !client.conf.Version.IsAtLeast(V0_10_0_0) {
  472. return nil, ErrUnsupportedVersion
  473. }
  474. controller := client.cachedController()
  475. if controller == nil {
  476. if err := client.refreshMetadata(); err != nil {
  477. return nil, err
  478. }
  479. controller = client.cachedController()
  480. }
  481. if controller == nil {
  482. return nil, ErrControllerNotAvailable
  483. }
  484. _ = controller.Open(client.conf)
  485. return controller, nil
  486. }
  487. // deregisterController removes the cached controllerID
  488. func (client *client) deregisterController() {
  489. client.lock.Lock()
  490. defer client.lock.Unlock()
  491. if controller, ok := client.brokers[client.controllerID]; ok {
  492. _ = controller.Close()
  493. delete(client.brokers, client.controllerID)
  494. }
  495. }
  496. // RefreshController retrieves the cluster controller from fresh metadata
  497. // and stores it in the local cache. Requires Kafka 0.10 or higher.
  498. func (client *client) RefreshController() (*Broker, error) {
  499. if client.Closed() {
  500. return nil, ErrClosedClient
  501. }
  502. client.deregisterController()
  503. if err := client.refreshMetadata(); err != nil {
  504. return nil, err
  505. }
  506. controller := client.cachedController()
  507. if controller == nil {
  508. return nil, ErrControllerNotAvailable
  509. }
  510. _ = controller.Open(client.conf)
  511. return controller, nil
  512. }
  513. func (client *client) Coordinator(consumerGroup string) (*Broker, error) {
  514. if client.Closed() {
  515. return nil, ErrClosedClient
  516. }
  517. coordinator := client.cachedCoordinator(consumerGroup)
  518. if coordinator == nil {
  519. if err := client.RefreshCoordinator(consumerGroup); err != nil {
  520. return nil, err
  521. }
  522. coordinator = client.cachedCoordinator(consumerGroup)
  523. }
  524. if coordinator == nil {
  525. return nil, ErrConsumerCoordinatorNotAvailable
  526. }
  527. _ = coordinator.Open(client.conf)
  528. return coordinator, nil
  529. }
  530. func (client *client) RefreshCoordinator(consumerGroup string) error {
  531. if client.Closed() {
  532. return ErrClosedClient
  533. }
  534. response, err := client.findCoordinator(consumerGroup, CoordinatorGroup, client.conf.Metadata.Retry.Max)
  535. if err != nil {
  536. return err
  537. }
  538. client.lock.Lock()
  539. defer client.lock.Unlock()
  540. client.registerBroker(response.Coordinator)
  541. client.coordinators[consumerGroup] = response.Coordinator.ID()
  542. return nil
  543. }
  544. func (client *client) TransactionCoordinator(transactionID string) (*Broker, error) {
  545. if client.Closed() {
  546. return nil, ErrClosedClient
  547. }
  548. coordinator := client.cachedTransactionCoordinator(transactionID)
  549. if coordinator == nil {
  550. if err := client.RefreshTransactionCoordinator(transactionID); err != nil {
  551. return nil, err
  552. }
  553. coordinator = client.cachedTransactionCoordinator(transactionID)
  554. }
  555. if coordinator == nil {
  556. return nil, ErrConsumerCoordinatorNotAvailable
  557. }
  558. _ = coordinator.Open(client.conf)
  559. return coordinator, nil
  560. }
  561. func (client *client) RefreshTransactionCoordinator(transactionID string) error {
  562. if client.Closed() {
  563. return ErrClosedClient
  564. }
  565. response, err := client.findCoordinator(transactionID, CoordinatorTransaction, client.conf.Metadata.Retry.Max)
  566. if err != nil {
  567. return err
  568. }
  569. client.lock.Lock()
  570. defer client.lock.Unlock()
  571. client.registerBroker(response.Coordinator)
  572. client.transactionCoordinators[transactionID] = response.Coordinator.ID()
  573. return nil
  574. }
  575. // private broker management helpers
  576. func (client *client) randomizeSeedBrokers(addrs []string) {
  577. random := rand.New(rand.NewSource(time.Now().UnixNano()))
  578. for _, index := range random.Perm(len(addrs)) {
  579. client.seedBrokers = append(client.seedBrokers, NewBroker(addrs[index]))
  580. }
  581. }
  582. func (client *client) updateBroker(brokers []*Broker) {
  583. currentBroker := make(map[int32]*Broker, len(brokers))
  584. for _, broker := range brokers {
  585. currentBroker[broker.ID()] = broker
  586. if client.brokers[broker.ID()] == nil { // add new broker
  587. client.brokers[broker.ID()] = broker
  588. DebugLogger.Printf("client/brokers registered new broker #%d at %s", broker.ID(), broker.Addr())
  589. } else if broker.Addr() != client.brokers[broker.ID()].Addr() { // replace broker with new address
  590. safeAsyncClose(client.brokers[broker.ID()])
  591. client.brokers[broker.ID()] = broker
  592. Logger.Printf("client/brokers replaced registered broker #%d with %s", broker.ID(), broker.Addr())
  593. }
  594. }
  595. for id, broker := range client.brokers {
  596. if _, exist := currentBroker[id]; !exist { // remove old broker
  597. safeAsyncClose(broker)
  598. delete(client.brokers, id)
  599. Logger.Printf("client/broker remove invalid broker #%d with %s", broker.ID(), broker.Addr())
  600. }
  601. }
  602. }
  603. // registerBroker makes sure a broker received by a Metadata or Coordinator request is registered
  604. // in the brokers map. It returns the broker that is registered, which may be the provided broker,
  605. // or a previously registered Broker instance. You must hold the write lock before calling this function.
  606. func (client *client) registerBroker(broker *Broker) {
  607. if client.brokers == nil {
  608. Logger.Printf("cannot register broker #%d at %s, client already closed", broker.ID(), broker.Addr())
  609. return
  610. }
  611. if client.brokers[broker.ID()] == nil {
  612. client.brokers[broker.ID()] = broker
  613. DebugLogger.Printf("client/brokers registered new broker #%d at %s", broker.ID(), broker.Addr())
  614. } else if broker.Addr() != client.brokers[broker.ID()].Addr() {
  615. safeAsyncClose(client.brokers[broker.ID()])
  616. client.brokers[broker.ID()] = broker
  617. Logger.Printf("client/brokers replaced registered broker #%d with %s", broker.ID(), broker.Addr())
  618. }
  619. }
  620. // deregisterBroker removes a broker from the broker list, and if it's
  621. // not in the broker list, removes it from seedBrokers.
  622. func (client *client) deregisterBroker(broker *Broker) {
  623. client.lock.Lock()
  624. defer client.lock.Unlock()
  625. _, ok := client.brokers[broker.ID()]
  626. if ok {
  627. Logger.Printf("client/brokers deregistered broker #%d at %s", broker.ID(), broker.Addr())
  628. delete(client.brokers, broker.ID())
  629. return
  630. }
  631. if len(client.seedBrokers) > 0 && broker == client.seedBrokers[0] {
  632. client.deadSeeds = append(client.deadSeeds, broker)
  633. client.seedBrokers = client.seedBrokers[1:]
  634. }
  635. }
  636. func (client *client) resurrectDeadBrokers() {
  637. client.lock.Lock()
  638. defer client.lock.Unlock()
  639. Logger.Printf("client/brokers resurrecting %d dead seed brokers", len(client.deadSeeds))
  640. client.seedBrokers = append(client.seedBrokers, client.deadSeeds...)
  641. client.deadSeeds = nil
  642. }
  643. // LeastLoadedBroker returns the broker with the least pending requests.
  644. // Firstly, choose the broker from cached broker list. If the broker list is empty, choose from seed brokers.
  645. func (client *client) LeastLoadedBroker() *Broker {
  646. client.lock.RLock()
  647. defer client.lock.RUnlock()
  648. var leastLoadedBroker *Broker
  649. pendingRequests := math.MaxInt
  650. for _, broker := range client.brokers {
  651. if pendingRequests > broker.ResponseSize() {
  652. pendingRequests = broker.ResponseSize()
  653. leastLoadedBroker = broker
  654. }
  655. }
  656. if leastLoadedBroker != nil {
  657. _ = leastLoadedBroker.Open(client.conf)
  658. return leastLoadedBroker
  659. }
  660. if len(client.seedBrokers) > 0 {
  661. _ = client.seedBrokers[0].Open(client.conf)
  662. return client.seedBrokers[0]
  663. }
  664. return leastLoadedBroker
  665. }
  666. // private caching/lazy metadata helpers
  667. type partitionType int
  668. const (
  669. allPartitions partitionType = iota
  670. writablePartitions
  671. // If you add any more types, update the partition cache in update()
  672. // Ensure this is the last partition type value
  673. maxPartitionIndex
  674. )
  675. func (client *client) cachedMetadata(topic string, partitionID int32) *PartitionMetadata {
  676. client.lock.RLock()
  677. defer client.lock.RUnlock()
  678. partitions := client.metadata[topic]
  679. if partitions != nil {
  680. return partitions[partitionID]
  681. }
  682. return nil
  683. }
  684. func (client *client) cachedPartitions(topic string, partitionSet partitionType) []int32 {
  685. client.lock.RLock()
  686. defer client.lock.RUnlock()
  687. partitions, exists := client.cachedPartitionsResults[topic]
  688. if !exists {
  689. return nil
  690. }
  691. return partitions[partitionSet]
  692. }
  693. func (client *client) setPartitionCache(topic string, partitionSet partitionType) []int32 {
  694. partitions := client.metadata[topic]
  695. if partitions == nil {
  696. return nil
  697. }
  698. ret := make([]int32, 0, len(partitions))
  699. for _, partition := range partitions {
  700. if partitionSet == writablePartitions && errors.Is(partition.Err, ErrLeaderNotAvailable) {
  701. continue
  702. }
  703. ret = append(ret, partition.ID)
  704. }
  705. sort.Sort(int32Slice(ret))
  706. return ret
  707. }
  708. func (client *client) cachedLeader(topic string, partitionID int32) (*Broker, int32, error) {
  709. client.lock.RLock()
  710. defer client.lock.RUnlock()
  711. partitions := client.metadata[topic]
  712. if partitions != nil {
  713. metadata, ok := partitions[partitionID]
  714. if ok {
  715. if errors.Is(metadata.Err, ErrLeaderNotAvailable) {
  716. return nil, -1, ErrLeaderNotAvailable
  717. }
  718. b := client.brokers[metadata.Leader]
  719. if b == nil {
  720. return nil, -1, ErrLeaderNotAvailable
  721. }
  722. _ = b.Open(client.conf)
  723. return b, metadata.LeaderEpoch, nil
  724. }
  725. }
  726. return nil, -1, ErrUnknownTopicOrPartition
  727. }
  728. func (client *client) getOffset(topic string, partitionID int32, timestamp int64) (int64, error) {
  729. broker, err := client.Leader(topic, partitionID)
  730. if err != nil {
  731. return -1, err
  732. }
  733. request := &OffsetRequest{}
  734. if client.conf.Version.IsAtLeast(V2_1_0_0) {
  735. // Version 4 adds the current leader epoch, which is used for fencing.
  736. request.Version = 4
  737. } else if client.conf.Version.IsAtLeast(V2_0_0_0) {
  738. // Version 3 is the same as version 2.
  739. request.Version = 3
  740. } else if client.conf.Version.IsAtLeast(V0_11_0_0) {
  741. // Version 2 adds the isolation level, which is used for transactional reads.
  742. request.Version = 2
  743. } else if client.conf.Version.IsAtLeast(V0_10_1_0) {
  744. // Version 1 removes MaxNumOffsets. From this version forward, only a single
  745. // offset can be returned.
  746. request.Version = 1
  747. }
  748. request.AddBlock(topic, partitionID, timestamp, 1)
  749. response, err := broker.GetAvailableOffsets(request)
  750. if err != nil {
  751. _ = broker.Close()
  752. return -1, err
  753. }
  754. block := response.GetBlock(topic, partitionID)
  755. if block == nil {
  756. _ = broker.Close()
  757. return -1, ErrIncompleteResponse
  758. }
  759. if !errors.Is(block.Err, ErrNoError) {
  760. return -1, block.Err
  761. }
  762. if len(block.Offsets) != 1 {
  763. return -1, ErrOffsetOutOfRange
  764. }
  765. return block.Offsets[0], nil
  766. }
  767. // core metadata update logic
  768. func (client *client) backgroundMetadataUpdater() {
  769. defer close(client.closed)
  770. if client.conf.Metadata.RefreshFrequency == time.Duration(0) {
  771. return
  772. }
  773. ticker := time.NewTicker(client.conf.Metadata.RefreshFrequency)
  774. defer ticker.Stop()
  775. for {
  776. select {
  777. case <-ticker.C:
  778. if err := client.refreshMetadata(); err != nil {
  779. Logger.Println("Client background metadata update:", err)
  780. }
  781. case <-client.closer:
  782. return
  783. }
  784. }
  785. }
  786. func (client *client) refreshMetadata() error {
  787. var topics []string
  788. if !client.conf.Metadata.Full {
  789. if specificTopics, err := client.MetadataTopics(); err != nil {
  790. return err
  791. } else if len(specificTopics) == 0 {
  792. return ErrNoTopicsToUpdateMetadata
  793. } else {
  794. topics = specificTopics
  795. }
  796. }
  797. if err := client.RefreshMetadata(topics...); err != nil {
  798. return err
  799. }
  800. return nil
  801. }
  802. func (client *client) tryRefreshMetadata(topics []string, attemptsRemaining int, deadline time.Time) error {
  803. pastDeadline := func(backoff time.Duration) bool {
  804. if !deadline.IsZero() && time.Now().Add(backoff).After(deadline) {
  805. // we are past the deadline
  806. return true
  807. }
  808. return false
  809. }
  810. retry := func(err error) error {
  811. if attemptsRemaining > 0 {
  812. backoff := client.computeBackoff(attemptsRemaining)
  813. if pastDeadline(backoff) {
  814. Logger.Println("client/metadata skipping last retries as we would go past the metadata timeout")
  815. return err
  816. }
  817. if backoff > 0 {
  818. time.Sleep(backoff)
  819. }
  820. t := atomic.LoadInt64(&client.updateMetadataMs)
  821. if time.Since(time.UnixMilli(t)) < backoff {
  822. return err
  823. }
  824. attemptsRemaining--
  825. Logger.Printf("client/metadata retrying after %dms... (%d attempts remaining)\n", backoff/time.Millisecond, attemptsRemaining)
  826. return client.tryRefreshMetadata(topics, attemptsRemaining, deadline)
  827. }
  828. return err
  829. }
  830. broker := client.LeastLoadedBroker()
  831. brokerErrors := make([]error, 0)
  832. for ; broker != nil && !pastDeadline(0); broker = client.LeastLoadedBroker() {
  833. allowAutoTopicCreation := client.conf.Metadata.AllowAutoTopicCreation
  834. if len(topics) > 0 {
  835. DebugLogger.Printf("client/metadata fetching metadata for %v from broker %s\n", topics, broker.addr)
  836. } else {
  837. allowAutoTopicCreation = false
  838. DebugLogger.Printf("client/metadata fetching metadata for all topics from broker %s\n", broker.addr)
  839. }
  840. req := NewMetadataRequest(client.conf.Version, topics)
  841. req.AllowAutoTopicCreation = allowAutoTopicCreation
  842. atomic.StoreInt64(&client.updateMetadataMs, time.Now().UnixMilli())
  843. response, err := broker.GetMetadata(req)
  844. var kerror KError
  845. var packetEncodingError PacketEncodingError
  846. if err == nil {
  847. // When talking to the startup phase of a broker, it is possible to receive an empty metadata set. We should remove that broker and try next broker (https://issues.apache.org/jira/browse/KAFKA-7924).
  848. if len(response.Brokers) == 0 {
  849. Logger.Println("client/metadata receiving empty brokers from the metadata response when requesting the broker #%d at %s", broker.ID(), broker.addr)
  850. _ = broker.Close()
  851. client.deregisterBroker(broker)
  852. continue
  853. }
  854. allKnownMetaData := len(topics) == 0
  855. // valid response, use it
  856. shouldRetry, err := client.updateMetadata(response, allKnownMetaData)
  857. if shouldRetry {
  858. Logger.Println("client/metadata found some partitions to be leaderless")
  859. return retry(err) // note: err can be nil
  860. }
  861. return err
  862. } else if errors.As(err, &packetEncodingError) {
  863. // didn't even send, return the error
  864. return err
  865. } else if errors.As(err, &kerror) {
  866. // if SASL auth error return as this _should_ be a non retryable err for all brokers
  867. if errors.Is(err, ErrSASLAuthenticationFailed) {
  868. Logger.Println("client/metadata failed SASL authentication")
  869. return err
  870. }
  871. if errors.Is(err, ErrTopicAuthorizationFailed) {
  872. Logger.Println("client is not authorized to access this topic. The topics were: ", topics)
  873. return err
  874. }
  875. // else remove that broker and try again
  876. Logger.Printf("client/metadata got error from broker %d while fetching metadata: %v\n", broker.ID(), err)
  877. _ = broker.Close()
  878. client.deregisterBroker(broker)
  879. } else {
  880. // some other error, remove that broker and try again
  881. Logger.Printf("client/metadata got error from broker %d while fetching metadata: %v\n", broker.ID(), err)
  882. brokerErrors = append(brokerErrors, err)
  883. _ = broker.Close()
  884. client.deregisterBroker(broker)
  885. }
  886. }
  887. error := Wrap(ErrOutOfBrokers, brokerErrors...)
  888. if broker != nil {
  889. Logger.Printf("client/metadata not fetching metadata from broker %s as we would go past the metadata timeout\n", broker.addr)
  890. return retry(error)
  891. }
  892. Logger.Println("client/metadata no available broker to send metadata request to")
  893. client.resurrectDeadBrokers()
  894. return retry(error)
  895. }
  896. // if no fatal error, returns a list of topics that need retrying due to ErrLeaderNotAvailable
  897. func (client *client) updateMetadata(data *MetadataResponse, allKnownMetaData bool) (retry bool, err error) {
  898. if client.Closed() {
  899. return
  900. }
  901. client.lock.Lock()
  902. defer client.lock.Unlock()
  903. // For all the brokers we received:
  904. // - if it is a new ID, save it
  905. // - if it is an existing ID, but the address we have is stale, discard the old one and save it
  906. // - if some brokers is not exist in it, remove old broker
  907. // - otherwise ignore it, replacing our existing one would just bounce the connection
  908. client.updateBroker(data.Brokers)
  909. client.controllerID = data.ControllerID
  910. if allKnownMetaData {
  911. client.metadata = make(map[string]map[int32]*PartitionMetadata)
  912. client.metadataTopics = make(map[string]none)
  913. client.cachedPartitionsResults = make(map[string][maxPartitionIndex][]int32)
  914. }
  915. for _, topic := range data.Topics {
  916. // topics must be added firstly to `metadataTopics` to guarantee that all
  917. // requested topics must be recorded to keep them trackable for periodically
  918. // metadata refresh.
  919. if _, exists := client.metadataTopics[topic.Name]; !exists {
  920. client.metadataTopics[topic.Name] = none{}
  921. }
  922. delete(client.metadata, topic.Name)
  923. delete(client.cachedPartitionsResults, topic.Name)
  924. switch topic.Err {
  925. case ErrNoError:
  926. // no-op
  927. case ErrInvalidTopic, ErrTopicAuthorizationFailed: // don't retry, don't store partial results
  928. err = topic.Err
  929. continue
  930. case ErrUnknownTopicOrPartition: // retry, do not store partial partition results
  931. err = topic.Err
  932. retry = true
  933. continue
  934. case ErrLeaderNotAvailable: // retry, but store partial partition results
  935. retry = true
  936. default: // don't retry, don't store partial results
  937. Logger.Printf("Unexpected topic-level metadata error: %s", topic.Err)
  938. err = topic.Err
  939. continue
  940. }
  941. client.metadata[topic.Name] = make(map[int32]*PartitionMetadata, len(topic.Partitions))
  942. for _, partition := range topic.Partitions {
  943. client.metadata[topic.Name][partition.ID] = partition
  944. if errors.Is(partition.Err, ErrLeaderNotAvailable) {
  945. retry = true
  946. }
  947. }
  948. var partitionCache [maxPartitionIndex][]int32
  949. partitionCache[allPartitions] = client.setPartitionCache(topic.Name, allPartitions)
  950. partitionCache[writablePartitions] = client.setPartitionCache(topic.Name, writablePartitions)
  951. client.cachedPartitionsResults[topic.Name] = partitionCache
  952. }
  953. return
  954. }
  955. func (client *client) cachedCoordinator(consumerGroup string) *Broker {
  956. client.lock.RLock()
  957. defer client.lock.RUnlock()
  958. if coordinatorID, ok := client.coordinators[consumerGroup]; ok {
  959. return client.brokers[coordinatorID]
  960. }
  961. return nil
  962. }
  963. func (client *client) cachedTransactionCoordinator(transactionID string) *Broker {
  964. client.lock.RLock()
  965. defer client.lock.RUnlock()
  966. if coordinatorID, ok := client.transactionCoordinators[transactionID]; ok {
  967. return client.brokers[coordinatorID]
  968. }
  969. return nil
  970. }
  971. func (client *client) cachedController() *Broker {
  972. client.lock.RLock()
  973. defer client.lock.RUnlock()
  974. return client.brokers[client.controllerID]
  975. }
  976. func (client *client) computeBackoff(attemptsRemaining int) time.Duration {
  977. if client.conf.Metadata.Retry.BackoffFunc != nil {
  978. maxRetries := client.conf.Metadata.Retry.Max
  979. retries := maxRetries - attemptsRemaining
  980. return client.conf.Metadata.Retry.BackoffFunc(retries, maxRetries)
  981. }
  982. return client.conf.Metadata.Retry.Backoff
  983. }
  984. func (client *client) findCoordinator(coordinatorKey string, coordinatorType CoordinatorType, attemptsRemaining int) (*FindCoordinatorResponse, error) {
  985. retry := func(err error) (*FindCoordinatorResponse, error) {
  986. if attemptsRemaining > 0 {
  987. backoff := client.computeBackoff(attemptsRemaining)
  988. attemptsRemaining--
  989. Logger.Printf("client/coordinator retrying after %dms... (%d attempts remaining)\n", backoff/time.Millisecond, attemptsRemaining)
  990. time.Sleep(backoff)
  991. return client.findCoordinator(coordinatorKey, coordinatorType, attemptsRemaining)
  992. }
  993. return nil, err
  994. }
  995. brokerErrors := make([]error, 0)
  996. for broker := client.LeastLoadedBroker(); broker != nil; broker = client.LeastLoadedBroker() {
  997. DebugLogger.Printf("client/coordinator requesting coordinator for %s from %s\n", coordinatorKey, broker.Addr())
  998. request := new(FindCoordinatorRequest)
  999. request.CoordinatorKey = coordinatorKey
  1000. request.CoordinatorType = coordinatorType
  1001. // Version 1 adds KeyType.
  1002. if client.conf.Version.IsAtLeast(V0_11_0_0) {
  1003. request.Version = 1
  1004. }
  1005. // Version 2 is the same as version 1.
  1006. if client.conf.Version.IsAtLeast(V2_0_0_0) {
  1007. request.Version = 2
  1008. }
  1009. response, err := broker.FindCoordinator(request)
  1010. if err != nil {
  1011. Logger.Printf("client/coordinator request to broker %s failed: %s\n", broker.Addr(), err)
  1012. var packetEncodingError PacketEncodingError
  1013. if errors.As(err, &packetEncodingError) {
  1014. return nil, err
  1015. } else {
  1016. _ = broker.Close()
  1017. brokerErrors = append(brokerErrors, err)
  1018. client.deregisterBroker(broker)
  1019. continue
  1020. }
  1021. }
  1022. if errors.Is(response.Err, ErrNoError) {
  1023. DebugLogger.Printf("client/coordinator coordinator for %s is #%d (%s)\n", coordinatorKey, response.Coordinator.ID(), response.Coordinator.Addr())
  1024. return response, nil
  1025. } else if errors.Is(response.Err, ErrConsumerCoordinatorNotAvailable) {
  1026. Logger.Printf("client/coordinator coordinator for %s is not available\n", coordinatorKey)
  1027. // This is very ugly, but this scenario will only happen once per cluster.
  1028. // The __consumer_offsets topic only has to be created one time.
  1029. // The number of partitions not configurable, but partition 0 should always exist.
  1030. if _, err := client.Leader("__consumer_offsets", 0); err != nil {
  1031. Logger.Printf("client/coordinator the __consumer_offsets topic is not initialized completely yet. Waiting 2 seconds...\n")
  1032. time.Sleep(2 * time.Second)
  1033. }
  1034. if coordinatorType == CoordinatorTransaction {
  1035. if _, err := client.Leader("__transaction_state", 0); err != nil {
  1036. Logger.Printf("client/coordinator the __transaction_state topic is not initialized completely yet. Waiting 2 seconds...\n")
  1037. time.Sleep(2 * time.Second)
  1038. }
  1039. }
  1040. return retry(ErrConsumerCoordinatorNotAvailable)
  1041. } else if errors.Is(response.Err, ErrGroupAuthorizationFailed) {
  1042. Logger.Printf("client was not authorized to access group %s while attempting to find coordinator", coordinatorKey)
  1043. return retry(ErrGroupAuthorizationFailed)
  1044. } else {
  1045. return nil, response.Err
  1046. }
  1047. }
  1048. Logger.Println("client/coordinator no available broker to send consumer metadata request to")
  1049. client.resurrectDeadBrokers()
  1050. return retry(Wrap(ErrOutOfBrokers, brokerErrors...))
  1051. }
  1052. func (client *client) resolveCanonicalNames(addrs []string) ([]string, error) {
  1053. ctx := context.Background()
  1054. dialer := client.Config().getDialer()
  1055. resolver := net.Resolver{
  1056. Dial: func(ctx context.Context, network, address string) (net.Conn, error) {
  1057. // dial func should only be called once, so switching within is acceptable
  1058. switch d := dialer.(type) {
  1059. case proxy.ContextDialer:
  1060. return d.DialContext(ctx, network, address)
  1061. default:
  1062. // we have no choice but to ignore the context
  1063. return d.Dial(network, address)
  1064. }
  1065. },
  1066. }
  1067. canonicalAddrs := make(map[string]struct{}, len(addrs)) // dedupe as we go
  1068. for _, addr := range addrs {
  1069. host, port, err := net.SplitHostPort(addr)
  1070. if err != nil {
  1071. return nil, err // message includes addr
  1072. }
  1073. ips, err := resolver.LookupHost(ctx, host)
  1074. if err != nil {
  1075. return nil, err // message includes host
  1076. }
  1077. for _, ip := range ips {
  1078. ptrs, err := resolver.LookupAddr(ctx, ip)
  1079. if err != nil {
  1080. return nil, err // message includes ip
  1081. }
  1082. // unlike the Java client, we do not further check that PTRs resolve
  1083. ptr := strings.TrimSuffix(ptrs[0], ".") // trailing dot breaks GSSAPI
  1084. canonicalAddrs[net.JoinHostPort(ptr, port)] = struct{}{}
  1085. }
  1086. }
  1087. addrs = make([]string, 0, len(canonicalAddrs))
  1088. for addr := range canonicalAddrs {
  1089. addrs = append(addrs, addr)
  1090. }
  1091. return addrs, nil
  1092. }
  1093. // nopCloserClient embeds an existing Client, but disables
  1094. // the Close method (yet all other methods pass
  1095. // through unchanged). This is for use in larger structs
  1096. // where it is undesirable to close the client that was
  1097. // passed in by the caller.
  1098. type nopCloserClient struct {
  1099. Client
  1100. }
  1101. // Close intercepts and purposely does not call the underlying
  1102. // client's Close() method.
  1103. func (ncc *nopCloserClient) Close() error {
  1104. return nil
  1105. }