broker.go 37 KB

12345678910111213141516171819202122232425262728293031323334353637383940414243444546474849505152535455565758596061626364656667686970717273747576777879808182838485868788899091929394959697989910010110210310410510610710810911011111211311411511611711811912012112212312412512612712812913013113213313413513613713813914014114214314414514614714814915015115215315415515615715815916016116216316416516616716816917017117217317417517617717817918018118218318418518618718818919019119219319419519619719819920020120220320420520620720820921021121221321421521621721821922022122222322422522622722822923023123223323423523623723823924024124224324424524624724824925025125225325425525625725825926026126226326426526626726826927027127227327427527627727827928028128228328428528628728828929029129229329429529629729829930030130230330430530630730830931031131231331431531631731831932032132232332432532632732832933033133233333433533633733833934034134234334434534634734834935035135235335435535635735835936036136236336436536636736836937037137237337437537637737837938038138238338438538638738838939039139239339439539639739839940040140240340440540640740840941041141241341441541641741841942042142242342442542642742842943043143243343443543643743843944044144244344444544644744844945045145245345445545645745845946046146246346446546646746846947047147247347447547647747847948048148248348448548648748848949049149249349449549649749849950050150250350450550650750850951051151251351451551651751851952052152252352452552652752852953053153253353453553653753853954054154254354454554654754854955055155255355455555655755855956056156256356456556656756856957057157257357457557657757857958058158258358458558658758858959059159259359459559659759859960060160260360460560660760860961061161261361461561661761861962062162262362462562662762862963063163263363463563663763863964064164264364464564664764864965065165265365465565665765865966066166266366466566666766866967067167267367467567667767867968068168268368468568668768868969069169269369469569669769869970070170270370470570670770870971071171271371471571671771871972072172272372472572672772872973073173273373473573673773873974074174274374474574674774874975075175275375475575675775875976076176276376476576676776876977077177277377477577677777877978078178278378478578678778878979079179279379479579679779879980080180280380480580680780880981081181281381481581681781881982082182282382482582682782882983083183283383483583683783883984084184284384484584684784884985085185285385485585685785885986086186286386486586686786886987087187287387487587687787887988088188288388488588688788888989089189289389489589689789889990090190290390490590690790890991091191291391491591691791891992092192292392492592692792892993093193293393493593693793893994094194294394494594694794894995095195295395495595695795895996096196296396496596696796896997097197297397497597697797897998098198298398498598698798898999099199299399499599699799899910001001100210031004100510061007100810091010101110121013101410151016101710181019102010211022102310241025102610271028102910301031103210331034103510361037103810391040104110421043104410451046104710481049105010511052105310541055105610571058105910601061106210631064106510661067106810691070107110721073107410751076107710781079108010811082108310841085108610871088108910901091109210931094109510961097109810991100110111021103110411051106110711081109111011111112111311141115111611171118111911201121112211231124112511261127112811291130113111321133113411351136113711381139114011411142114311441145114611471148114911501151115211531154115511561157115811591160116111621163116411651166116711681169117011711172117311741175117611771178117911801181118211831184118511861187118811891190119111921193119411951196119711981199120012011202120312041205120612071208120912101211121212131214121512161217121812191220122112221223122412251226122712281229123012311232123312341235123612371238123912401241124212431244124512461247124812491250125112521253125412551256125712581259126012611262126312641265126612671268126912701271127212731274127512761277127812791280128112821283128412851286128712881289129012911292129312941295129612971298129913001301130213031304130513061307130813091310131113121313131413151316131713181319132013211322132313241325132613271328132913301331133213331334133513361337133813391340134113421343134413451346
  1. package sarama
  2. import (
  3. "crypto/tls"
  4. "encoding/binary"
  5. "fmt"
  6. "io"
  7. "net"
  8. "sort"
  9. "strconv"
  10. "strings"
  11. "sync"
  12. "sync/atomic"
  13. "time"
  14. metrics "github.com/rcrowley/go-metrics"
  15. )
  16. // Broker represents a single Kafka broker connection. All operations on this object are entirely concurrency-safe.
  17. type Broker struct {
  18. conf *Config
  19. rack *string
  20. id int32
  21. addr string
  22. correlationID int32
  23. conn net.Conn
  24. connErr error
  25. lock sync.Mutex
  26. opened int32
  27. responses chan responsePromise
  28. done chan bool
  29. registeredMetrics []string
  30. incomingByteRate metrics.Meter
  31. requestRate metrics.Meter
  32. requestSize metrics.Histogram
  33. requestLatency metrics.Histogram
  34. outgoingByteRate metrics.Meter
  35. responseRate metrics.Meter
  36. responseSize metrics.Histogram
  37. brokerIncomingByteRate metrics.Meter
  38. brokerRequestRate metrics.Meter
  39. brokerRequestSize metrics.Histogram
  40. brokerRequestLatency metrics.Histogram
  41. brokerOutgoingByteRate metrics.Meter
  42. brokerResponseRate metrics.Meter
  43. brokerResponseSize metrics.Histogram
  44. kerberosAuthenticator GSSAPIKerberosAuth
  45. }
  46. // SASLMechanism specifies the SASL mechanism the client uses to authenticate with the broker
  47. type SASLMechanism string
  48. const (
  49. // SASLTypeOAuth represents the SASL/OAUTHBEARER mechanism (Kafka 2.0.0+)
  50. SASLTypeOAuth = "OAUTHBEARER"
  51. // SASLTypePlaintext represents the SASL/PLAIN mechanism
  52. SASLTypePlaintext = "PLAIN"
  53. // SASLTypeSCRAMSHA256 represents the SCRAM-SHA-256 mechanism.
  54. SASLTypeSCRAMSHA256 = "SCRAM-SHA-256"
  55. // SASLTypeSCRAMSHA512 represents the SCRAM-SHA-512 mechanism.
  56. SASLTypeSCRAMSHA512 = "SCRAM-SHA-512"
  57. SASLTypeGSSAPI = "GSSAPI"
  58. // SASLHandshakeV0 is v0 of the Kafka SASL handshake protocol. Client and
  59. // server negotiate SASL auth using opaque packets.
  60. SASLHandshakeV0 = int16(0)
  61. // SASLHandshakeV1 is v1 of the Kafka SASL handshake protocol. Client and
  62. // server negotiate SASL by wrapping tokens with Kafka protocol headers.
  63. SASLHandshakeV1 = int16(1)
  64. // SASLExtKeyAuth is the reserved extension key name sent as part of the
  65. // SASL/OAUTHBEARER intial client response
  66. SASLExtKeyAuth = "auth"
  67. )
  68. // AccessToken contains an access token used to authenticate a
  69. // SASL/OAUTHBEARER client along with associated metadata.
  70. type AccessToken struct {
  71. // Token is the access token payload.
  72. Token string
  73. // Extensions is a optional map of arbitrary key-value pairs that can be
  74. // sent with the SASL/OAUTHBEARER initial client response. These values are
  75. // ignored by the SASL server if they are unexpected. This feature is only
  76. // supported by Kafka >= 2.1.0.
  77. Extensions map[string]string
  78. }
  79. // AccessTokenProvider is the interface that encapsulates how implementors
  80. // can generate access tokens for Kafka broker authentication.
  81. type AccessTokenProvider interface {
  82. // Token returns an access token. The implementation should ensure token
  83. // reuse so that multiple calls at connect time do not create multiple
  84. // tokens. The implementation should also periodically refresh the token in
  85. // order to guarantee that each call returns an unexpired token. This
  86. // method should not block indefinitely--a timeout error should be returned
  87. // after a short period of inactivity so that the broker connection logic
  88. // can log debugging information and retry.
  89. Token() (*AccessToken, error)
  90. }
  91. // SCRAMClient is a an interface to a SCRAM
  92. // client implementation.
  93. type SCRAMClient interface {
  94. // Begin prepares the client for the SCRAM exchange
  95. // with the server with a user name and a password
  96. Begin(userName, password, authzID string) error
  97. // Step steps client through the SCRAM exchange. It is
  98. // called repeatedly until it errors or `Done` returns true.
  99. Step(challenge string) (response string, err error)
  100. // Done should return true when the SCRAM conversation
  101. // is over.
  102. Done() bool
  103. }
  104. type responsePromise struct {
  105. requestTime time.Time
  106. correlationID int32
  107. packets chan []byte
  108. errors chan error
  109. }
  110. // NewBroker creates and returns a Broker targeting the given host:port address.
  111. // This does not attempt to actually connect, you have to call Open() for that.
  112. func NewBroker(addr string) *Broker {
  113. return &Broker{id: -1, addr: addr}
  114. }
  115. // Open tries to connect to the Broker if it is not already connected or connecting, but does not block
  116. // waiting for the connection to complete. This means that any subsequent operations on the broker will
  117. // block waiting for the connection to succeed or fail. To get the effect of a fully synchronous Open call,
  118. // follow it by a call to Connected(). The only errors Open will return directly are ConfigurationError or
  119. // AlreadyConnected. If conf is nil, the result of NewConfig() is used.
  120. func (b *Broker) Open(conf *Config) error {
  121. if !atomic.CompareAndSwapInt32(&b.opened, 0, 1) {
  122. return ErrAlreadyConnected
  123. }
  124. if conf == nil {
  125. conf = NewConfig()
  126. }
  127. err := conf.Validate()
  128. if err != nil {
  129. return err
  130. }
  131. b.lock.Lock()
  132. go withRecover(func() {
  133. defer b.lock.Unlock()
  134. dialer := net.Dialer{
  135. Timeout: conf.Net.DialTimeout,
  136. KeepAlive: conf.Net.KeepAlive,
  137. LocalAddr: conf.Net.LocalAddr,
  138. }
  139. if conf.Net.TLS.Enable {
  140. b.conn, b.connErr = tls.DialWithDialer(&dialer, "tcp", b.addr, conf.Net.TLS.Config)
  141. } else if conf.Net.Proxy.Enable {
  142. b.conn, b.connErr = conf.Net.Proxy.Dialer.Dial("tcp", b.addr)
  143. } else {
  144. b.conn, b.connErr = dialer.Dial("tcp", b.addr)
  145. }
  146. if b.connErr != nil {
  147. Logger.Printf("Failed to connect to broker %s: %s\n", b.addr, b.connErr)
  148. b.conn = nil
  149. atomic.StoreInt32(&b.opened, 0)
  150. return
  151. }
  152. b.conn = newBufConn(b.conn)
  153. b.conf = conf
  154. // Create or reuse the global metrics shared between brokers
  155. b.incomingByteRate = metrics.GetOrRegisterMeter("incoming-byte-rate", conf.MetricRegistry)
  156. b.requestRate = metrics.GetOrRegisterMeter("request-rate", conf.MetricRegistry)
  157. b.requestSize = getOrRegisterHistogram("request-size", conf.MetricRegistry)
  158. b.requestLatency = getOrRegisterHistogram("request-latency-in-ms", conf.MetricRegistry)
  159. b.outgoingByteRate = metrics.GetOrRegisterMeter("outgoing-byte-rate", conf.MetricRegistry)
  160. b.responseRate = metrics.GetOrRegisterMeter("response-rate", conf.MetricRegistry)
  161. b.responseSize = getOrRegisterHistogram("response-size", conf.MetricRegistry)
  162. // Do not gather metrics for seeded broker (only used during bootstrap) because they share
  163. // the same id (-1) and are already exposed through the global metrics above
  164. if b.id >= 0 {
  165. b.registerMetrics()
  166. }
  167. if conf.Net.SASL.Enable {
  168. b.connErr = b.authenticateViaSASL()
  169. if b.connErr != nil {
  170. err = b.conn.Close()
  171. if err == nil {
  172. Logger.Printf("Closed connection to broker %s\n", b.addr)
  173. } else {
  174. Logger.Printf("Error while closing connection to broker %s: %s\n", b.addr, err)
  175. }
  176. b.conn = nil
  177. atomic.StoreInt32(&b.opened, 0)
  178. return
  179. }
  180. }
  181. b.done = make(chan bool)
  182. b.responses = make(chan responsePromise, b.conf.Net.MaxOpenRequests-1)
  183. if b.id >= 0 {
  184. Logger.Printf("Connected to broker at %s (registered as #%d)\n", b.addr, b.id)
  185. } else {
  186. Logger.Printf("Connected to broker at %s (unregistered)\n", b.addr)
  187. }
  188. go withRecover(b.responseReceiver)
  189. })
  190. return nil
  191. }
  192. // Connected returns true if the broker is connected and false otherwise. If the broker is not
  193. // connected but it had tried to connect, the error from that connection attempt is also returned.
  194. func (b *Broker) Connected() (bool, error) {
  195. b.lock.Lock()
  196. defer b.lock.Unlock()
  197. return b.conn != nil, b.connErr
  198. }
  199. //Close closes the broker resources
  200. func (b *Broker) Close() error {
  201. b.lock.Lock()
  202. defer b.lock.Unlock()
  203. if b.conn == nil {
  204. return ErrNotConnected
  205. }
  206. close(b.responses)
  207. <-b.done
  208. err := b.conn.Close()
  209. b.conn = nil
  210. b.connErr = nil
  211. b.done = nil
  212. b.responses = nil
  213. b.unregisterMetrics()
  214. if err == nil {
  215. Logger.Printf("Closed connection to broker %s\n", b.addr)
  216. } else {
  217. Logger.Printf("Error while closing connection to broker %s: %s\n", b.addr, err)
  218. }
  219. atomic.StoreInt32(&b.opened, 0)
  220. return err
  221. }
  222. // ID returns the broker ID retrieved from Kafka's metadata, or -1 if that is not known.
  223. func (b *Broker) ID() int32 {
  224. return b.id
  225. }
  226. // Addr returns the broker address as either retrieved from Kafka's metadata or passed to NewBroker.
  227. func (b *Broker) Addr() string {
  228. return b.addr
  229. }
  230. // Rack returns the broker's rack as retrieved from Kafka's metadata or the
  231. // empty string if it is not known. The returned value corresponds to the
  232. // broker's broker.rack configuration setting. Requires protocol version to be
  233. // at least v0.10.0.0.
  234. func (b *Broker) Rack() string {
  235. if b.rack == nil {
  236. return ""
  237. }
  238. return *b.rack
  239. }
  240. //GetMetadata send a metadata request and returns a metadata response or error
  241. func (b *Broker) GetMetadata(request *MetadataRequest) (*MetadataResponse, error) {
  242. response := new(MetadataResponse)
  243. err := b.sendAndReceive(request, response)
  244. if err != nil {
  245. return nil, err
  246. }
  247. return response, nil
  248. }
  249. //GetConsumerMetadata send a consumer metadata request and returns a consumer metadata response or error
  250. func (b *Broker) GetConsumerMetadata(request *ConsumerMetadataRequest) (*ConsumerMetadataResponse, error) {
  251. response := new(ConsumerMetadataResponse)
  252. err := b.sendAndReceive(request, response)
  253. if err != nil {
  254. return nil, err
  255. }
  256. return response, nil
  257. }
  258. //FindCoordinator sends a find coordinate request and returns a response or error
  259. func (b *Broker) FindCoordinator(request *FindCoordinatorRequest) (*FindCoordinatorResponse, error) {
  260. response := new(FindCoordinatorResponse)
  261. err := b.sendAndReceive(request, response)
  262. if err != nil {
  263. return nil, err
  264. }
  265. return response, nil
  266. }
  267. //GetAvailableOffsets return an offset response or error
  268. func (b *Broker) GetAvailableOffsets(request *OffsetRequest) (*OffsetResponse, error) {
  269. response := new(OffsetResponse)
  270. err := b.sendAndReceive(request, response)
  271. if err != nil {
  272. return nil, err
  273. }
  274. return response, nil
  275. }
  276. //Produce returns a produce response or error
  277. func (b *Broker) Produce(request *ProduceRequest) (*ProduceResponse, error) {
  278. var (
  279. response *ProduceResponse
  280. err error
  281. )
  282. if request.RequiredAcks == NoResponse {
  283. err = b.sendAndReceive(request, nil)
  284. } else {
  285. response = new(ProduceResponse)
  286. err = b.sendAndReceive(request, response)
  287. }
  288. if err != nil {
  289. return nil, err
  290. }
  291. return response, nil
  292. }
  293. //Fetch returns a FetchResponse or error
  294. func (b *Broker) Fetch(request *FetchRequest) (*FetchResponse, error) {
  295. response := new(FetchResponse)
  296. err := b.sendAndReceive(request, response)
  297. if err != nil {
  298. return nil, err
  299. }
  300. return response, nil
  301. }
  302. //CommitOffset return an Offset commit reponse or error
  303. func (b *Broker) CommitOffset(request *OffsetCommitRequest) (*OffsetCommitResponse, error) {
  304. response := new(OffsetCommitResponse)
  305. err := b.sendAndReceive(request, response)
  306. if err != nil {
  307. return nil, err
  308. }
  309. return response, nil
  310. }
  311. //FetchOffset returns an offset fetch response or error
  312. func (b *Broker) FetchOffset(request *OffsetFetchRequest) (*OffsetFetchResponse, error) {
  313. response := new(OffsetFetchResponse)
  314. err := b.sendAndReceive(request, response)
  315. if err != nil {
  316. return nil, err
  317. }
  318. return response, nil
  319. }
  320. //JoinGroup returns a join group response or error
  321. func (b *Broker) JoinGroup(request *JoinGroupRequest) (*JoinGroupResponse, error) {
  322. response := new(JoinGroupResponse)
  323. err := b.sendAndReceive(request, response)
  324. if err != nil {
  325. return nil, err
  326. }
  327. return response, nil
  328. }
  329. //SyncGroup returns a sync group response or error
  330. func (b *Broker) SyncGroup(request *SyncGroupRequest) (*SyncGroupResponse, error) {
  331. response := new(SyncGroupResponse)
  332. err := b.sendAndReceive(request, response)
  333. if err != nil {
  334. return nil, err
  335. }
  336. return response, nil
  337. }
  338. //LeaveGroup return a leave group response or error
  339. func (b *Broker) LeaveGroup(request *LeaveGroupRequest) (*LeaveGroupResponse, error) {
  340. response := new(LeaveGroupResponse)
  341. err := b.sendAndReceive(request, response)
  342. if err != nil {
  343. return nil, err
  344. }
  345. return response, nil
  346. }
  347. //Heartbeat returns a heartbeat response or error
  348. func (b *Broker) Heartbeat(request *HeartbeatRequest) (*HeartbeatResponse, error) {
  349. response := new(HeartbeatResponse)
  350. err := b.sendAndReceive(request, response)
  351. if err != nil {
  352. return nil, err
  353. }
  354. return response, nil
  355. }
  356. //ListGroups return a list group response or error
  357. func (b *Broker) ListGroups(request *ListGroupsRequest) (*ListGroupsResponse, error) {
  358. response := new(ListGroupsResponse)
  359. err := b.sendAndReceive(request, response)
  360. if err != nil {
  361. return nil, err
  362. }
  363. return response, nil
  364. }
  365. //DescribeGroups return describe group response or error
  366. func (b *Broker) DescribeGroups(request *DescribeGroupsRequest) (*DescribeGroupsResponse, error) {
  367. response := new(DescribeGroupsResponse)
  368. err := b.sendAndReceive(request, response)
  369. if err != nil {
  370. return nil, err
  371. }
  372. return response, nil
  373. }
  374. //ApiVersions return api version response or error
  375. func (b *Broker) ApiVersions(request *ApiVersionsRequest) (*ApiVersionsResponse, error) {
  376. response := new(ApiVersionsResponse)
  377. err := b.sendAndReceive(request, response)
  378. if err != nil {
  379. return nil, err
  380. }
  381. return response, nil
  382. }
  383. //CreateTopics send a create topic request and returns create topic response
  384. func (b *Broker) CreateTopics(request *CreateTopicsRequest) (*CreateTopicsResponse, error) {
  385. response := new(CreateTopicsResponse)
  386. err := b.sendAndReceive(request, response)
  387. if err != nil {
  388. return nil, err
  389. }
  390. return response, nil
  391. }
  392. //DeleteTopics sends a delete topic request and returns delete topic response
  393. func (b *Broker) DeleteTopics(request *DeleteTopicsRequest) (*DeleteTopicsResponse, error) {
  394. response := new(DeleteTopicsResponse)
  395. err := b.sendAndReceive(request, response)
  396. if err != nil {
  397. return nil, err
  398. }
  399. return response, nil
  400. }
  401. //CreatePartitions sends a create partition request and returns create
  402. //partitions response or error
  403. func (b *Broker) CreatePartitions(request *CreatePartitionsRequest) (*CreatePartitionsResponse, error) {
  404. response := new(CreatePartitionsResponse)
  405. err := b.sendAndReceive(request, response)
  406. if err != nil {
  407. return nil, err
  408. }
  409. return response, nil
  410. }
  411. //DeleteRecords send a request to delete records and return delete record
  412. //response or error
  413. func (b *Broker) DeleteRecords(request *DeleteRecordsRequest) (*DeleteRecordsResponse, error) {
  414. response := new(DeleteRecordsResponse)
  415. err := b.sendAndReceive(request, response)
  416. if err != nil {
  417. return nil, err
  418. }
  419. return response, nil
  420. }
  421. //DescribeAcls sends a describe acl request and returns a response or error
  422. func (b *Broker) DescribeAcls(request *DescribeAclsRequest) (*DescribeAclsResponse, error) {
  423. response := new(DescribeAclsResponse)
  424. err := b.sendAndReceive(request, response)
  425. if err != nil {
  426. return nil, err
  427. }
  428. return response, nil
  429. }
  430. //CreateAcls sends a create acl request and returns a response or error
  431. func (b *Broker) CreateAcls(request *CreateAclsRequest) (*CreateAclsResponse, error) {
  432. response := new(CreateAclsResponse)
  433. err := b.sendAndReceive(request, response)
  434. if err != nil {
  435. return nil, err
  436. }
  437. return response, nil
  438. }
  439. //DeleteAcls sends a delete acl request and returns a response or error
  440. func (b *Broker) DeleteAcls(request *DeleteAclsRequest) (*DeleteAclsResponse, error) {
  441. response := new(DeleteAclsResponse)
  442. err := b.sendAndReceive(request, response)
  443. if err != nil {
  444. return nil, err
  445. }
  446. return response, nil
  447. }
  448. //InitProducerID sends an init producer request and returns a response or error
  449. func (b *Broker) InitProducerID(request *InitProducerIDRequest) (*InitProducerIDResponse, error) {
  450. response := new(InitProducerIDResponse)
  451. err := b.sendAndReceive(request, response)
  452. if err != nil {
  453. return nil, err
  454. }
  455. return response, nil
  456. }
  457. //AddPartitionsToTxn send a request to add partition to txn and returns
  458. //a response or error
  459. func (b *Broker) AddPartitionsToTxn(request *AddPartitionsToTxnRequest) (*AddPartitionsToTxnResponse, error) {
  460. response := new(AddPartitionsToTxnResponse)
  461. err := b.sendAndReceive(request, response)
  462. if err != nil {
  463. return nil, err
  464. }
  465. return response, nil
  466. }
  467. //AddOffsetsToTxn sends a request to add offsets to txn and returns a response
  468. //or error
  469. func (b *Broker) AddOffsetsToTxn(request *AddOffsetsToTxnRequest) (*AddOffsetsToTxnResponse, error) {
  470. response := new(AddOffsetsToTxnResponse)
  471. err := b.sendAndReceive(request, response)
  472. if err != nil {
  473. return nil, err
  474. }
  475. return response, nil
  476. }
  477. //EndTxn sends a request to end txn and returns a response or error
  478. func (b *Broker) EndTxn(request *EndTxnRequest) (*EndTxnResponse, error) {
  479. response := new(EndTxnResponse)
  480. err := b.sendAndReceive(request, response)
  481. if err != nil {
  482. return nil, err
  483. }
  484. return response, nil
  485. }
  486. //TxnOffsetCommit sends a request to commit transaction offsets and returns
  487. //a response or error
  488. func (b *Broker) TxnOffsetCommit(request *TxnOffsetCommitRequest) (*TxnOffsetCommitResponse, error) {
  489. response := new(TxnOffsetCommitResponse)
  490. err := b.sendAndReceive(request, response)
  491. if err != nil {
  492. return nil, err
  493. }
  494. return response, nil
  495. }
  496. //DescribeConfigs sends a request to describe config and returns a response or
  497. //error
  498. func (b *Broker) DescribeConfigs(request *DescribeConfigsRequest) (*DescribeConfigsResponse, error) {
  499. response := new(DescribeConfigsResponse)
  500. err := b.sendAndReceive(request, response)
  501. if err != nil {
  502. return nil, err
  503. }
  504. return response, nil
  505. }
  506. //AlterConfigs sends a request to alter config and return a response or error
  507. func (b *Broker) AlterConfigs(request *AlterConfigsRequest) (*AlterConfigsResponse, error) {
  508. response := new(AlterConfigsResponse)
  509. err := b.sendAndReceive(request, response)
  510. if err != nil {
  511. return nil, err
  512. }
  513. return response, nil
  514. }
  515. //DeleteGroups sends a request to delete groups and returns a response or error
  516. func (b *Broker) DeleteGroups(request *DeleteGroupsRequest) (*DeleteGroupsResponse, error) {
  517. response := new(DeleteGroupsResponse)
  518. if err := b.sendAndReceive(request, response); err != nil {
  519. return nil, err
  520. }
  521. return response, nil
  522. }
  523. //DescribeLogDirs sends a request to get the broker's log dir paths and sizes
  524. func (b *Broker) DescribeLogDirs(request *DescribeLogDirsRequest) (*DescribeLogDirsResponse, error) {
  525. response := new(DescribeLogDirsResponse)
  526. err := b.sendAndReceive(request, response)
  527. if err != nil {
  528. return nil, err
  529. }
  530. return response, nil
  531. }
  532. // readFull ensures the conn ReadDeadline has been setup before making a
  533. // call to io.ReadFull
  534. func (b *Broker) readFull(buf []byte) (n int, err error) {
  535. if err := b.conn.SetReadDeadline(time.Now().Add(b.conf.Net.ReadTimeout)); err != nil {
  536. return 0, err
  537. }
  538. return io.ReadFull(b.conn, buf)
  539. }
  540. // write ensures the conn WriteDeadline has been setup before making a
  541. // call to conn.Write
  542. func (b *Broker) write(buf []byte) (n int, err error) {
  543. if err := b.conn.SetWriteDeadline(time.Now().Add(b.conf.Net.WriteTimeout)); err != nil {
  544. return 0, err
  545. }
  546. return b.conn.Write(buf)
  547. }
  548. func (b *Broker) send(rb protocolBody, promiseResponse bool) (*responsePromise, error) {
  549. b.lock.Lock()
  550. defer b.lock.Unlock()
  551. if b.conn == nil {
  552. if b.connErr != nil {
  553. return nil, b.connErr
  554. }
  555. return nil, ErrNotConnected
  556. }
  557. if !b.conf.Version.IsAtLeast(rb.requiredVersion()) {
  558. return nil, ErrUnsupportedVersion
  559. }
  560. req := &request{correlationID: b.correlationID, clientID: b.conf.ClientID, body: rb}
  561. buf, err := encode(req, b.conf.MetricRegistry)
  562. if err != nil {
  563. return nil, err
  564. }
  565. requestTime := time.Now()
  566. bytes, err := b.write(buf)
  567. b.updateOutgoingCommunicationMetrics(bytes)
  568. if err != nil {
  569. return nil, err
  570. }
  571. b.correlationID++
  572. if !promiseResponse {
  573. // Record request latency without the response
  574. b.updateRequestLatencyMetrics(time.Since(requestTime))
  575. return nil, nil
  576. }
  577. promise := responsePromise{requestTime, req.correlationID, make(chan []byte), make(chan error)}
  578. b.responses <- promise
  579. return &promise, nil
  580. }
  581. func (b *Broker) sendAndReceive(req protocolBody, res versionedDecoder) error {
  582. promise, err := b.send(req, res != nil)
  583. if err != nil {
  584. return err
  585. }
  586. if promise == nil {
  587. return nil
  588. }
  589. select {
  590. case buf := <-promise.packets:
  591. return versionedDecode(buf, res, req.version())
  592. case err = <-promise.errors:
  593. return err
  594. }
  595. }
  596. func (b *Broker) decode(pd packetDecoder, version int16) (err error) {
  597. b.id, err = pd.getInt32()
  598. if err != nil {
  599. return err
  600. }
  601. host, err := pd.getString()
  602. if err != nil {
  603. return err
  604. }
  605. port, err := pd.getInt32()
  606. if err != nil {
  607. return err
  608. }
  609. if version >= 1 {
  610. b.rack, err = pd.getNullableString()
  611. if err != nil {
  612. return err
  613. }
  614. }
  615. b.addr = net.JoinHostPort(host, fmt.Sprint(port))
  616. if _, _, err := net.SplitHostPort(b.addr); err != nil {
  617. return err
  618. }
  619. return nil
  620. }
  621. func (b *Broker) encode(pe packetEncoder, version int16) (err error) {
  622. host, portstr, err := net.SplitHostPort(b.addr)
  623. if err != nil {
  624. return err
  625. }
  626. port, err := strconv.Atoi(portstr)
  627. if err != nil {
  628. return err
  629. }
  630. pe.putInt32(b.id)
  631. err = pe.putString(host)
  632. if err != nil {
  633. return err
  634. }
  635. pe.putInt32(int32(port))
  636. if version >= 1 {
  637. err = pe.putNullableString(b.rack)
  638. if err != nil {
  639. return err
  640. }
  641. }
  642. return nil
  643. }
  644. func (b *Broker) responseReceiver() {
  645. var dead error
  646. header := make([]byte, 8)
  647. for response := range b.responses {
  648. if dead != nil {
  649. response.errors <- dead
  650. continue
  651. }
  652. bytesReadHeader, err := b.readFull(header)
  653. requestLatency := time.Since(response.requestTime)
  654. if err != nil {
  655. b.updateIncomingCommunicationMetrics(bytesReadHeader, requestLatency)
  656. dead = err
  657. response.errors <- err
  658. continue
  659. }
  660. decodedHeader := responseHeader{}
  661. err = decode(header, &decodedHeader)
  662. if err != nil {
  663. b.updateIncomingCommunicationMetrics(bytesReadHeader, requestLatency)
  664. dead = err
  665. response.errors <- err
  666. continue
  667. }
  668. if decodedHeader.correlationID != response.correlationID {
  669. b.updateIncomingCommunicationMetrics(bytesReadHeader, requestLatency)
  670. // TODO if decoded ID < cur ID, discard until we catch up
  671. // TODO if decoded ID > cur ID, save it so when cur ID catches up we have a response
  672. dead = PacketDecodingError{fmt.Sprintf("correlation ID didn't match, wanted %d, got %d", response.correlationID, decodedHeader.correlationID)}
  673. response.errors <- dead
  674. continue
  675. }
  676. buf := make([]byte, decodedHeader.length-4)
  677. bytesReadBody, err := b.readFull(buf)
  678. b.updateIncomingCommunicationMetrics(bytesReadHeader+bytesReadBody, requestLatency)
  679. if err != nil {
  680. dead = err
  681. response.errors <- err
  682. continue
  683. }
  684. response.packets <- buf
  685. }
  686. close(b.done)
  687. }
  688. func (b *Broker) authenticateViaSASL() error {
  689. switch b.conf.Net.SASL.Mechanism {
  690. case SASLTypeOAuth:
  691. return b.sendAndReceiveSASLOAuth(b.conf.Net.SASL.TokenProvider)
  692. case SASLTypeSCRAMSHA256, SASLTypeSCRAMSHA512:
  693. return b.sendAndReceiveSASLSCRAMv1()
  694. case SASLTypeGSSAPI:
  695. return b.sendAndReceiveKerberos()
  696. default:
  697. return b.sendAndReceiveSASLPlainAuth()
  698. }
  699. }
  700. func (b *Broker) sendAndReceiveKerberos() error {
  701. b.kerberosAuthenticator.Config = &b.conf.Net.SASL.GSSAPI
  702. if b.kerberosAuthenticator.NewKerberosClientFunc == nil {
  703. b.kerberosAuthenticator.NewKerberosClientFunc = NewKerberosClient
  704. }
  705. return b.kerberosAuthenticator.Authorize(b)
  706. }
  707. func (b *Broker) sendAndReceiveSASLHandshake(saslType SASLMechanism, version int16) error {
  708. rb := &SaslHandshakeRequest{Mechanism: string(saslType), Version: version}
  709. req := &request{correlationID: b.correlationID, clientID: b.conf.ClientID, body: rb}
  710. buf, err := encode(req, b.conf.MetricRegistry)
  711. if err != nil {
  712. return err
  713. }
  714. requestTime := time.Now()
  715. bytes, err := b.write(buf)
  716. b.updateOutgoingCommunicationMetrics(bytes)
  717. if err != nil {
  718. Logger.Printf("Failed to send SASL handshake %s: %s\n", b.addr, err.Error())
  719. return err
  720. }
  721. b.correlationID++
  722. header := make([]byte, 8) // response header
  723. _, err = b.readFull(header)
  724. if err != nil {
  725. Logger.Printf("Failed to read SASL handshake header : %s\n", err.Error())
  726. return err
  727. }
  728. length := binary.BigEndian.Uint32(header[:4])
  729. payload := make([]byte, length-4)
  730. n, err := b.readFull(payload)
  731. if err != nil {
  732. Logger.Printf("Failed to read SASL handshake payload : %s\n", err.Error())
  733. return err
  734. }
  735. b.updateIncomingCommunicationMetrics(n+8, time.Since(requestTime))
  736. res := &SaslHandshakeResponse{}
  737. err = versionedDecode(payload, res, 0)
  738. if err != nil {
  739. Logger.Printf("Failed to parse SASL handshake : %s\n", err.Error())
  740. return err
  741. }
  742. if res.Err != ErrNoError {
  743. Logger.Printf("Invalid SASL Mechanism : %s\n", res.Err.Error())
  744. return res.Err
  745. }
  746. Logger.Print("Successful SASL handshake. Available mechanisms: ", res.EnabledMechanisms)
  747. return nil
  748. }
  749. // Kafka 0.10.x supported SASL PLAIN/Kerberos via KAFKA-3149 (KIP-43).
  750. // Kafka 1.x.x onward added a SaslAuthenticate request/response message which
  751. // wraps the SASL flow in the Kafka protocol, which allows for returning
  752. // meaningful errors on authentication failure.
  753. //
  754. // In SASL Plain, Kafka expects the auth header to be in the following format
  755. // Message format (from https://tools.ietf.org/html/rfc4616):
  756. //
  757. // message = [authzid] UTF8NUL authcid UTF8NUL passwd
  758. // authcid = 1*SAFE ; MUST accept up to 255 octets
  759. // authzid = 1*SAFE ; MUST accept up to 255 octets
  760. // passwd = 1*SAFE ; MUST accept up to 255 octets
  761. // UTF8NUL = %x00 ; UTF-8 encoded NUL character
  762. //
  763. // SAFE = UTF1 / UTF2 / UTF3 / UTF4
  764. // ;; any UTF-8 encoded Unicode character except NUL
  765. //
  766. // With SASL v0 handshake and auth then:
  767. // When credentials are valid, Kafka returns a 4 byte array of null characters.
  768. // When credentials are invalid, Kafka closes the connection.
  769. //
  770. // With SASL v1 handshake and auth then:
  771. // When credentials are invalid, Kafka replies with a SaslAuthenticate response
  772. // containing an error code and message detailing the authentication failure.
  773. func (b *Broker) sendAndReceiveSASLPlainAuth() error {
  774. // default to V0 to allow for backward compatability when SASL is enabled
  775. // but not the handshake
  776. if b.conf.Net.SASL.Handshake {
  777. handshakeErr := b.sendAndReceiveSASLHandshake(SASLTypePlaintext, b.conf.Net.SASL.Version)
  778. if handshakeErr != nil {
  779. Logger.Printf("Error while performing SASL handshake %s\n", b.addr)
  780. return handshakeErr
  781. }
  782. }
  783. if b.conf.Net.SASL.Version == SASLHandshakeV1 {
  784. return b.sendAndReceiveV1SASLPlainAuth()
  785. }
  786. return b.sendAndReceiveV0SASLPlainAuth()
  787. }
  788. // sendAndReceiveV0SASLPlainAuth flows the v0 sasl auth NOT wrapped in the kafka protocol
  789. func (b *Broker) sendAndReceiveV0SASLPlainAuth() error {
  790. length := len(b.conf.Net.SASL.AuthIdentity) + 1 + len(b.conf.Net.SASL.User) + 1 + len(b.conf.Net.SASL.Password)
  791. authBytes := make([]byte, length+4) //4 byte length header + auth data
  792. binary.BigEndian.PutUint32(authBytes, uint32(length))
  793. copy(authBytes[4:], []byte(b.conf.Net.SASL.AuthIdentity+"\x00"+b.conf.Net.SASL.User+"\x00"+b.conf.Net.SASL.Password))
  794. requestTime := time.Now()
  795. bytesWritten, err := b.write(authBytes)
  796. b.updateOutgoingCommunicationMetrics(bytesWritten)
  797. if err != nil {
  798. Logger.Printf("Failed to write SASL auth header to broker %s: %s\n", b.addr, err.Error())
  799. return err
  800. }
  801. header := make([]byte, 4)
  802. n, err := b.readFull(header)
  803. b.updateIncomingCommunicationMetrics(n, time.Since(requestTime))
  804. // If the credentials are valid, we would get a 4 byte response filled with null characters.
  805. // Otherwise, the broker closes the connection and we get an EOF
  806. if err != nil {
  807. Logger.Printf("Failed to read response while authenticating with SASL to broker %s: %s\n", b.addr, err.Error())
  808. return err
  809. }
  810. Logger.Printf("SASL authentication successful with broker %s:%v - %v\n", b.addr, n, header)
  811. return nil
  812. }
  813. // sendAndReceiveV1SASLPlainAuth flows the v1 sasl authentication using the kafka protocol
  814. func (b *Broker) sendAndReceiveV1SASLPlainAuth() error {
  815. correlationID := b.correlationID
  816. requestTime := time.Now()
  817. bytesWritten, err := b.sendSASLPlainAuthClientResponse(correlationID)
  818. b.updateOutgoingCommunicationMetrics(bytesWritten)
  819. if err != nil {
  820. Logger.Printf("Failed to write SASL auth header to broker %s: %s\n", b.addr, err.Error())
  821. return err
  822. }
  823. b.correlationID++
  824. bytesRead, err := b.receiveSASLServerResponse(&SaslAuthenticateResponse{}, correlationID)
  825. b.updateIncomingCommunicationMetrics(bytesRead, time.Since(requestTime))
  826. // With v1 sasl we get an error message set in the response we can return
  827. if err != nil {
  828. Logger.Printf("Error returned from broker during SASL flow %s: %s\n", b.addr, err.Error())
  829. return err
  830. }
  831. return nil
  832. }
  833. // sendAndReceiveSASLOAuth performs the authentication flow as described by KIP-255
  834. // https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=75968876
  835. func (b *Broker) sendAndReceiveSASLOAuth(provider AccessTokenProvider) error {
  836. if err := b.sendAndReceiveSASLHandshake(SASLTypeOAuth, SASLHandshakeV1); err != nil {
  837. return err
  838. }
  839. token, err := provider.Token()
  840. if err != nil {
  841. return err
  842. }
  843. message, err := buildClientFirstMessage(token)
  844. if err != nil {
  845. return err
  846. }
  847. challenged, err := b.sendClientMessage(message)
  848. if err != nil {
  849. return err
  850. }
  851. if challenged {
  852. // Abort the token exchange. The broker returns the failure code.
  853. _, err = b.sendClientMessage([]byte(`\x01`))
  854. }
  855. return err
  856. }
  857. // sendClientMessage sends a SASL/OAUTHBEARER client message and returns true
  858. // if the broker responds with a challenge, in which case the token is
  859. // rejected.
  860. func (b *Broker) sendClientMessage(message []byte) (bool, error) {
  861. requestTime := time.Now()
  862. correlationID := b.correlationID
  863. bytesWritten, err := b.sendSASLOAuthBearerClientMessage(message, correlationID)
  864. if err != nil {
  865. return false, err
  866. }
  867. b.updateOutgoingCommunicationMetrics(bytesWritten)
  868. b.correlationID++
  869. res := &SaslAuthenticateResponse{}
  870. bytesRead, err := b.receiveSASLServerResponse(res, correlationID)
  871. requestLatency := time.Since(requestTime)
  872. b.updateIncomingCommunicationMetrics(bytesRead, requestLatency)
  873. isChallenge := len(res.SaslAuthBytes) > 0
  874. if isChallenge && err != nil {
  875. Logger.Printf("Broker rejected authentication token: %s", res.SaslAuthBytes)
  876. }
  877. return isChallenge, err
  878. }
  879. func (b *Broker) sendAndReceiveSASLSCRAMv1() error {
  880. if err := b.sendAndReceiveSASLHandshake(b.conf.Net.SASL.Mechanism, SASLHandshakeV1); err != nil {
  881. return err
  882. }
  883. scramClient := b.conf.Net.SASL.SCRAMClientGeneratorFunc()
  884. if err := scramClient.Begin(b.conf.Net.SASL.User, b.conf.Net.SASL.Password, b.conf.Net.SASL.SCRAMAuthzID); err != nil {
  885. return fmt.Errorf("failed to start SCRAM exchange with the server: %s", err.Error())
  886. }
  887. msg, err := scramClient.Step("")
  888. if err != nil {
  889. return fmt.Errorf("failed to advance the SCRAM exchange: %s", err.Error())
  890. }
  891. for !scramClient.Done() {
  892. requestTime := time.Now()
  893. correlationID := b.correlationID
  894. bytesWritten, err := b.sendSaslAuthenticateRequest(correlationID, []byte(msg))
  895. if err != nil {
  896. Logger.Printf("Failed to write SASL auth header to broker %s: %s\n", b.addr, err.Error())
  897. return err
  898. }
  899. b.updateOutgoingCommunicationMetrics(bytesWritten)
  900. b.correlationID++
  901. challenge, err := b.receiveSaslAuthenticateResponse(correlationID)
  902. if err != nil {
  903. Logger.Printf("Failed to read response while authenticating with SASL to broker %s: %s\n", b.addr, err.Error())
  904. return err
  905. }
  906. b.updateIncomingCommunicationMetrics(len(challenge), time.Since(requestTime))
  907. msg, err = scramClient.Step(string(challenge))
  908. if err != nil {
  909. Logger.Println("SASL authentication failed", err)
  910. return err
  911. }
  912. }
  913. Logger.Println("SASL authentication succeeded")
  914. return nil
  915. }
  916. func (b *Broker) sendSaslAuthenticateRequest(correlationID int32, msg []byte) (int, error) {
  917. rb := &SaslAuthenticateRequest{msg}
  918. req := &request{correlationID: correlationID, clientID: b.conf.ClientID, body: rb}
  919. buf, err := encode(req, b.conf.MetricRegistry)
  920. if err != nil {
  921. return 0, err
  922. }
  923. return b.write(buf)
  924. }
  925. func (b *Broker) receiveSaslAuthenticateResponse(correlationID int32) ([]byte, error) {
  926. buf := make([]byte, responseLengthSize+correlationIDSize)
  927. _, err := b.readFull(buf)
  928. if err != nil {
  929. return nil, err
  930. }
  931. header := responseHeader{}
  932. err = decode(buf, &header)
  933. if err != nil {
  934. return nil, err
  935. }
  936. if header.correlationID != correlationID {
  937. return nil, fmt.Errorf("correlation ID didn't match, wanted %d, got %d", b.correlationID, header.correlationID)
  938. }
  939. buf = make([]byte, header.length-correlationIDSize)
  940. _, err = b.readFull(buf)
  941. if err != nil {
  942. return nil, err
  943. }
  944. res := &SaslAuthenticateResponse{}
  945. if err := versionedDecode(buf, res, 0); err != nil {
  946. return nil, err
  947. }
  948. if res.Err != ErrNoError {
  949. return nil, res.Err
  950. }
  951. return res.SaslAuthBytes, nil
  952. }
  953. // Build SASL/OAUTHBEARER initial client response as described by RFC-7628
  954. // https://tools.ietf.org/html/rfc7628
  955. func buildClientFirstMessage(token *AccessToken) ([]byte, error) {
  956. var ext string
  957. if token.Extensions != nil && len(token.Extensions) > 0 {
  958. if _, ok := token.Extensions[SASLExtKeyAuth]; ok {
  959. return []byte{}, fmt.Errorf("the extension `%s` is invalid", SASLExtKeyAuth)
  960. }
  961. ext = "\x01" + mapToString(token.Extensions, "=", "\x01")
  962. }
  963. resp := []byte(fmt.Sprintf("n,,\x01auth=Bearer %s%s\x01\x01", token.Token, ext))
  964. return resp, nil
  965. }
  966. // mapToString returns a list of key-value pairs ordered by key.
  967. // keyValSep separates the key from the value. elemSep separates each pair.
  968. func mapToString(extensions map[string]string, keyValSep string, elemSep string) string {
  969. buf := make([]string, 0, len(extensions))
  970. for k, v := range extensions {
  971. buf = append(buf, k+keyValSep+v)
  972. }
  973. sort.Strings(buf)
  974. return strings.Join(buf, elemSep)
  975. }
  976. func (b *Broker) sendSASLPlainAuthClientResponse(correlationID int32) (int, error) {
  977. authBytes := []byte(b.conf.Net.SASL.AuthIdentity + "\x00" + b.conf.Net.SASL.User + "\x00" + b.conf.Net.SASL.Password)
  978. rb := &SaslAuthenticateRequest{authBytes}
  979. req := &request{correlationID: correlationID, clientID: b.conf.ClientID, body: rb}
  980. buf, err := encode(req, b.conf.MetricRegistry)
  981. if err != nil {
  982. return 0, err
  983. }
  984. return b.write(buf)
  985. }
  986. func (b *Broker) sendSASLOAuthBearerClientMessage(initialResp []byte, correlationID int32) (int, error) {
  987. rb := &SaslAuthenticateRequest{initialResp}
  988. req := &request{correlationID: correlationID, clientID: b.conf.ClientID, body: rb}
  989. buf, err := encode(req, b.conf.MetricRegistry)
  990. if err != nil {
  991. return 0, err
  992. }
  993. return b.write(buf)
  994. }
  995. func (b *Broker) receiveSASLServerResponse(res *SaslAuthenticateResponse, correlationID int32) (int, error) {
  996. buf := make([]byte, responseLengthSize+correlationIDSize)
  997. bytesRead, err := b.readFull(buf)
  998. if err != nil {
  999. return bytesRead, err
  1000. }
  1001. header := responseHeader{}
  1002. err = decode(buf, &header)
  1003. if err != nil {
  1004. return bytesRead, err
  1005. }
  1006. if header.correlationID != correlationID {
  1007. return bytesRead, fmt.Errorf("correlation ID didn't match, wanted %d, got %d", b.correlationID, header.correlationID)
  1008. }
  1009. buf = make([]byte, header.length-correlationIDSize)
  1010. c, err := b.readFull(buf)
  1011. bytesRead += c
  1012. if err != nil {
  1013. return bytesRead, err
  1014. }
  1015. if err := versionedDecode(buf, res, 0); err != nil {
  1016. return bytesRead, err
  1017. }
  1018. if res.Err != ErrNoError {
  1019. return bytesRead, res.Err
  1020. }
  1021. return bytesRead, nil
  1022. }
  1023. func (b *Broker) updateIncomingCommunicationMetrics(bytes int, requestLatency time.Duration) {
  1024. b.updateRequestLatencyMetrics(requestLatency)
  1025. b.responseRate.Mark(1)
  1026. if b.brokerResponseRate != nil {
  1027. b.brokerResponseRate.Mark(1)
  1028. }
  1029. responseSize := int64(bytes)
  1030. b.incomingByteRate.Mark(responseSize)
  1031. if b.brokerIncomingByteRate != nil {
  1032. b.brokerIncomingByteRate.Mark(responseSize)
  1033. }
  1034. b.responseSize.Update(responseSize)
  1035. if b.brokerResponseSize != nil {
  1036. b.brokerResponseSize.Update(responseSize)
  1037. }
  1038. }
  1039. func (b *Broker) updateRequestLatencyMetrics(requestLatency time.Duration) {
  1040. requestLatencyInMs := int64(requestLatency / time.Millisecond)
  1041. b.requestLatency.Update(requestLatencyInMs)
  1042. if b.brokerRequestLatency != nil {
  1043. b.brokerRequestLatency.Update(requestLatencyInMs)
  1044. }
  1045. }
  1046. func (b *Broker) updateOutgoingCommunicationMetrics(bytes int) {
  1047. b.requestRate.Mark(1)
  1048. if b.brokerRequestRate != nil {
  1049. b.brokerRequestRate.Mark(1)
  1050. }
  1051. requestSize := int64(bytes)
  1052. b.outgoingByteRate.Mark(requestSize)
  1053. if b.brokerOutgoingByteRate != nil {
  1054. b.brokerOutgoingByteRate.Mark(requestSize)
  1055. }
  1056. b.requestSize.Update(requestSize)
  1057. if b.brokerRequestSize != nil {
  1058. b.brokerRequestSize.Update(requestSize)
  1059. }
  1060. }
  1061. func (b *Broker) registerMetrics() {
  1062. b.brokerIncomingByteRate = b.registerMeter("incoming-byte-rate")
  1063. b.brokerRequestRate = b.registerMeter("request-rate")
  1064. b.brokerRequestSize = b.registerHistogram("request-size")
  1065. b.brokerRequestLatency = b.registerHistogram("request-latency-in-ms")
  1066. b.brokerOutgoingByteRate = b.registerMeter("outgoing-byte-rate")
  1067. b.brokerResponseRate = b.registerMeter("response-rate")
  1068. b.brokerResponseSize = b.registerHistogram("response-size")
  1069. }
  1070. func (b *Broker) unregisterMetrics() {
  1071. for _, name := range b.registeredMetrics {
  1072. b.conf.MetricRegistry.Unregister(name)
  1073. }
  1074. }
  1075. func (b *Broker) registerMeter(name string) metrics.Meter {
  1076. nameForBroker := getMetricNameForBroker(name, b)
  1077. b.registeredMetrics = append(b.registeredMetrics, nameForBroker)
  1078. return metrics.GetOrRegisterMeter(nameForBroker, b.conf.MetricRegistry)
  1079. }
  1080. func (b *Broker) registerHistogram(name string) metrics.Histogram {
  1081. nameForBroker := getMetricNameForBroker(name, b)
  1082. b.registeredMetrics = append(b.registeredMetrics, nameForBroker)
  1083. return getOrRegisterHistogram(nameForBroker, b.conf.MetricRegistry)
  1084. }