mockbroker.go 12 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415
  1. package sarama
  2. import (
  3. "bytes"
  4. "encoding/binary"
  5. "fmt"
  6. "io"
  7. "net"
  8. "reflect"
  9. "strconv"
  10. "sync"
  11. "time"
  12. "github.com/davecgh/go-spew/spew"
  13. )
  14. const (
  15. expectationTimeout = 500 * time.Millisecond
  16. )
  17. type GSSApiHandlerFunc func([]byte) []byte
  18. type requestHandlerFunc func(req *request) (res encoderWithHeader)
  19. // RequestNotifierFunc is invoked when a mock broker processes a request successfully
  20. // and will provides the number of bytes read and written.
  21. type RequestNotifierFunc func(bytesRead, bytesWritten int)
  22. // MockBroker is a mock Kafka broker that is used in unit tests. It is exposed
  23. // to facilitate testing of higher level or specialized consumers and producers
  24. // built on top of Sarama. Note that it does not 'mimic' the Kafka API protocol,
  25. // but rather provides a facility to do that. It takes care of the TCP
  26. // transport, request unmarshaling, response marshaling, and makes it the test
  27. // writer responsibility to program correct according to the Kafka API protocol
  28. // MockBroker behaviour.
  29. //
  30. // MockBroker is implemented as a TCP server listening on a kernel-selected
  31. // localhost port that can accept many connections. It reads Kafka requests
  32. // from that connection and returns responses programmed by the SetHandlerByMap
  33. // function. If a MockBroker receives a request that it has no programmed
  34. // response for, then it returns nothing and the request times out.
  35. //
  36. // A set of MockRequest builders to define mappings used by MockBroker is
  37. // provided by Sarama. But users can develop MockRequests of their own and use
  38. // them along with or instead of the standard ones.
  39. //
  40. // When running tests with MockBroker it is strongly recommended to specify
  41. // a timeout to `go test` so that if the broker hangs waiting for a response,
  42. // the test panics.
  43. //
  44. // It is not necessary to prefix message length or correlation ID to your
  45. // response bytes, the server does that automatically as a convenience.
  46. type MockBroker struct {
  47. brokerID int32
  48. port int32
  49. closing chan none
  50. stopper chan none
  51. expectations chan encoderWithHeader
  52. listener net.Listener
  53. t TestReporter
  54. latency time.Duration
  55. handler requestHandlerFunc
  56. notifier RequestNotifierFunc
  57. history []RequestResponse
  58. lock sync.Mutex
  59. gssApiHandler GSSApiHandlerFunc
  60. }
  61. // RequestResponse represents a Request/Response pair processed by MockBroker.
  62. type RequestResponse struct {
  63. Request protocolBody
  64. Response encoder
  65. }
  66. // SetLatency makes broker pause for the specified period every time before
  67. // replying.
  68. func (b *MockBroker) SetLatency(latency time.Duration) {
  69. b.latency = latency
  70. }
  71. // SetHandlerByMap defines mapping of Request types to MockResponses. When a
  72. // request is received by the broker, it looks up the request type in the map
  73. // and uses the found MockResponse instance to generate an appropriate reply.
  74. // If the request type is not found in the map then nothing is sent.
  75. func (b *MockBroker) SetHandlerByMap(handlerMap map[string]MockResponse) {
  76. b.setHandler(func(req *request) (res encoderWithHeader) {
  77. reqTypeName := reflect.TypeOf(req.body).Elem().Name()
  78. mockResponse := handlerMap[reqTypeName]
  79. if mockResponse == nil {
  80. return nil
  81. }
  82. return mockResponse.For(req.body)
  83. })
  84. }
  85. // SetNotifier set a function that will get invoked whenever a request has been
  86. // processed successfully and will provide the number of bytes read and written
  87. func (b *MockBroker) SetNotifier(notifier RequestNotifierFunc) {
  88. b.lock.Lock()
  89. b.notifier = notifier
  90. b.lock.Unlock()
  91. }
  92. // BrokerID returns broker ID assigned to the broker.
  93. func (b *MockBroker) BrokerID() int32 {
  94. return b.brokerID
  95. }
  96. // History returns a slice of RequestResponse pairs in the order they were
  97. // processed by the broker. Note that in case of multiple connections to the
  98. // broker the order expected by a test can be different from the order recorded
  99. // in the history, unless some synchronization is implemented in the test.
  100. func (b *MockBroker) History() []RequestResponse {
  101. b.lock.Lock()
  102. history := make([]RequestResponse, len(b.history))
  103. copy(history, b.history)
  104. b.lock.Unlock()
  105. return history
  106. }
  107. // Port returns the TCP port number the broker is listening for requests on.
  108. func (b *MockBroker) Port() int32 {
  109. return b.port
  110. }
  111. // Addr returns the broker connection string in the form "<address>:<port>".
  112. func (b *MockBroker) Addr() string {
  113. return b.listener.Addr().String()
  114. }
  115. // Close terminates the broker blocking until it stops internal goroutines and
  116. // releases all resources.
  117. func (b *MockBroker) Close() {
  118. close(b.expectations)
  119. if len(b.expectations) > 0 {
  120. buf := bytes.NewBufferString(fmt.Sprintf("mockbroker/%d: not all expectations were satisfied! Still waiting on:\n", b.BrokerID()))
  121. for e := range b.expectations {
  122. _, _ = buf.WriteString(spew.Sdump(e))
  123. }
  124. b.t.Error(buf.String())
  125. }
  126. close(b.closing)
  127. <-b.stopper
  128. }
  129. // setHandler sets the specified function as the request handler. Whenever
  130. // a mock broker reads a request from the wire it passes the request to the
  131. // function and sends back whatever the handler function returns.
  132. func (b *MockBroker) setHandler(handler requestHandlerFunc) {
  133. b.lock.Lock()
  134. b.handler = handler
  135. b.lock.Unlock()
  136. }
  137. func (b *MockBroker) serverLoop() {
  138. defer close(b.stopper)
  139. var err error
  140. var conn net.Conn
  141. go func() {
  142. <-b.closing
  143. err := b.listener.Close()
  144. if err != nil {
  145. b.t.Error(err)
  146. }
  147. }()
  148. wg := &sync.WaitGroup{}
  149. i := 0
  150. for conn, err = b.listener.Accept(); err == nil; conn, err = b.listener.Accept() {
  151. wg.Add(1)
  152. go b.handleRequests(conn, i, wg)
  153. i++
  154. }
  155. wg.Wait()
  156. Logger.Printf("*** mockbroker/%d: listener closed, err=%v", b.BrokerID(), err)
  157. }
  158. func (b *MockBroker) SetGSSAPIHandler(handler GSSApiHandlerFunc) {
  159. b.gssApiHandler = handler
  160. }
  161. func (b *MockBroker) readToBytes(r io.Reader) ([]byte, error) {
  162. var (
  163. bytesRead int
  164. lengthBytes = make([]byte, 4)
  165. )
  166. if _, err := io.ReadFull(r, lengthBytes); err != nil {
  167. return nil, err
  168. }
  169. bytesRead += len(lengthBytes)
  170. length := int32(binary.BigEndian.Uint32(lengthBytes))
  171. if length <= 4 || length > MaxRequestSize {
  172. return nil, PacketDecodingError{fmt.Sprintf("message of length %d too large or too small", length)}
  173. }
  174. encodedReq := make([]byte, length)
  175. if _, err := io.ReadFull(r, encodedReq); err != nil {
  176. return nil, err
  177. }
  178. bytesRead += len(encodedReq)
  179. fullBytes := append(lengthBytes, encodedReq...)
  180. return fullBytes, nil
  181. }
  182. func (b *MockBroker) isGSSAPI(buffer []byte) bool {
  183. return buffer[4] == 0x60 || bytes.Equal(buffer[4:6], []byte{0x05, 0x04})
  184. }
  185. func (b *MockBroker) handleRequests(conn io.ReadWriteCloser, idx int, wg *sync.WaitGroup) {
  186. defer wg.Done()
  187. defer func() {
  188. _ = conn.Close()
  189. }()
  190. Logger.Printf("*** mockbroker/%d/%d: connection opened", b.BrokerID(), idx)
  191. var err error
  192. abort := make(chan none)
  193. defer close(abort)
  194. go func() {
  195. select {
  196. case <-b.closing:
  197. _ = conn.Close()
  198. case <-abort:
  199. }
  200. }()
  201. var bytesWritten int
  202. var bytesRead int
  203. for {
  204. buffer, err := b.readToBytes(conn)
  205. if err != nil {
  206. Logger.Printf("*** mockbroker/%d/%d: invalid request: err=%+v, %+v", b.brokerID, idx, err, spew.Sdump(buffer))
  207. b.serverError(err)
  208. break
  209. }
  210. bytesWritten = 0
  211. if !b.isGSSAPI(buffer) {
  212. req, br, err := decodeRequest(bytes.NewReader(buffer))
  213. bytesRead = br
  214. if err != nil {
  215. Logger.Printf("*** mockbroker/%d/%d: invalid request: err=%+v, %+v", b.brokerID, idx, err, spew.Sdump(req))
  216. b.serverError(err)
  217. break
  218. }
  219. if b.latency > 0 {
  220. time.Sleep(b.latency)
  221. }
  222. b.lock.Lock()
  223. res := b.handler(req)
  224. b.history = append(b.history, RequestResponse{req.body, res})
  225. b.lock.Unlock()
  226. if res == nil {
  227. Logger.Printf("*** mockbroker/%d/%d: ignored %v", b.brokerID, idx, spew.Sdump(req))
  228. continue
  229. }
  230. Logger.Printf("*** mockbroker/%d/%d: served %v -> %v", b.brokerID, idx, req, res)
  231. encodedRes, err := encode(res, nil)
  232. if err != nil {
  233. b.serverError(err)
  234. break
  235. }
  236. if len(encodedRes) == 0 {
  237. b.lock.Lock()
  238. if b.notifier != nil {
  239. b.notifier(bytesRead, 0)
  240. }
  241. b.lock.Unlock()
  242. continue
  243. }
  244. resHeader := b.encodeHeader(res.headerVersion(), req.correlationID, uint32(len(encodedRes)))
  245. if _, err = conn.Write(resHeader); err != nil {
  246. b.serverError(err)
  247. break
  248. }
  249. if _, err = conn.Write(encodedRes); err != nil {
  250. b.serverError(err)
  251. break
  252. }
  253. bytesWritten = len(resHeader) + len(encodedRes)
  254. } else {
  255. // GSSAPI is not part of kafka protocol, but is supported for authentication proposes.
  256. // Don't support history for this kind of request as is only used for test GSSAPI authentication mechanism
  257. b.lock.Lock()
  258. res := b.gssApiHandler(buffer)
  259. b.lock.Unlock()
  260. if res == nil {
  261. Logger.Printf("*** mockbroker/%d/%d: ignored %v", b.brokerID, idx, spew.Sdump(buffer))
  262. continue
  263. }
  264. if _, err = conn.Write(res); err != nil {
  265. b.serverError(err)
  266. break
  267. }
  268. bytesWritten = len(res)
  269. }
  270. b.lock.Lock()
  271. if b.notifier != nil {
  272. b.notifier(bytesRead, bytesWritten)
  273. }
  274. b.lock.Unlock()
  275. }
  276. Logger.Printf("*** mockbroker/%d/%d: connection closed, err=%v", b.BrokerID(), idx, err)
  277. }
  278. func (b *MockBroker) encodeHeader(headerVersion int16, correlationId int32, payloadLength uint32) []byte {
  279. headerLength := uint32(8)
  280. if headerVersion >= 1 {
  281. headerLength = 9
  282. }
  283. resHeader := make([]byte, headerLength)
  284. binary.BigEndian.PutUint32(resHeader, payloadLength+headerLength-4)
  285. binary.BigEndian.PutUint32(resHeader[4:], uint32(correlationId))
  286. if headerVersion >= 1 {
  287. binary.PutUvarint(resHeader[8:], 0)
  288. }
  289. return resHeader
  290. }
  291. func (b *MockBroker) defaultRequestHandler(req *request) (res encoderWithHeader) {
  292. select {
  293. case res, ok := <-b.expectations:
  294. if !ok {
  295. return nil
  296. }
  297. return res
  298. case <-time.After(expectationTimeout):
  299. return nil
  300. }
  301. }
  302. func (b *MockBroker) serverError(err error) {
  303. isConnectionClosedError := false
  304. if _, ok := err.(*net.OpError); ok {
  305. isConnectionClosedError = true
  306. } else if err == io.EOF {
  307. isConnectionClosedError = true
  308. } else if err.Error() == "use of closed network connection" {
  309. isConnectionClosedError = true
  310. }
  311. if isConnectionClosedError {
  312. return
  313. }
  314. b.t.Errorf(err.Error())
  315. }
  316. // NewMockBroker launches a fake Kafka broker. It takes a TestReporter as provided by the
  317. // test framework and a channel of responses to use. If an error occurs it is
  318. // simply logged to the TestReporter and the broker exits.
  319. func NewMockBroker(t TestReporter, brokerID int32) *MockBroker {
  320. return NewMockBrokerAddr(t, brokerID, "localhost:0")
  321. }
  322. // NewMockBrokerAddr behaves like newMockBroker but listens on the address you give
  323. // it rather than just some ephemeral port.
  324. func NewMockBrokerAddr(t TestReporter, brokerID int32, addr string) *MockBroker {
  325. listener, err := net.Listen("tcp", addr)
  326. if err != nil {
  327. t.Fatal(err)
  328. }
  329. return NewMockBrokerListener(t, brokerID, listener)
  330. }
  331. // NewMockBrokerListener behaves like newMockBrokerAddr but accepts connections on the listener specified.
  332. func NewMockBrokerListener(t TestReporter, brokerID int32, listener net.Listener) *MockBroker {
  333. var err error
  334. broker := &MockBroker{
  335. closing: make(chan none),
  336. stopper: make(chan none),
  337. t: t,
  338. brokerID: brokerID,
  339. expectations: make(chan encoderWithHeader, 512),
  340. listener: listener,
  341. }
  342. broker.handler = broker.defaultRequestHandler
  343. Logger.Printf("*** mockbroker/%d listening on %s\n", brokerID, broker.listener.Addr().String())
  344. _, portStr, err := net.SplitHostPort(broker.listener.Addr().String())
  345. if err != nil {
  346. t.Fatal(err)
  347. }
  348. tmp, err := strconv.ParseInt(portStr, 10, 32)
  349. if err != nil {
  350. t.Fatal(err)
  351. }
  352. broker.port = int32(tmp)
  353. go broker.serverLoop()
  354. return broker
  355. }
  356. func (b *MockBroker) Returns(e encoderWithHeader) {
  357. b.expectations <- e
  358. }