sink_node.go 16 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567
  1. // Copyright 2022-2023 EMQ Technologies Co., Ltd.
  2. //
  3. // Licensed under the Apache License, Version 2.0 (the "License");
  4. // you may not use this file except in compliance with the License.
  5. // You may obtain a copy of the License at
  6. //
  7. // http://www.apache.org/licenses/LICENSE-2.0
  8. //
  9. // Unless required by applicable law or agreed to in writing, software
  10. // distributed under the License is distributed on an "AS IS" BASIS,
  11. // WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  12. // See the License for the specific language governing permissions and
  13. // limitations under the License.
  14. package node
  15. import (
  16. "fmt"
  17. "strings"
  18. "sync"
  19. "github.com/lf-edge/ekuiper/internal/binder/io"
  20. "github.com/lf-edge/ekuiper/internal/conf"
  21. sinkUtil "github.com/lf-edge/ekuiper/internal/io/sink"
  22. "github.com/lf-edge/ekuiper/internal/topo/context"
  23. "github.com/lf-edge/ekuiper/internal/topo/node/cache"
  24. nodeConf "github.com/lf-edge/ekuiper/internal/topo/node/conf"
  25. "github.com/lf-edge/ekuiper/internal/topo/node/metric"
  26. "github.com/lf-edge/ekuiper/internal/topo/transform"
  27. "github.com/lf-edge/ekuiper/internal/xsql"
  28. "github.com/lf-edge/ekuiper/pkg/api"
  29. "github.com/lf-edge/ekuiper/pkg/cast"
  30. "github.com/lf-edge/ekuiper/pkg/errorx"
  31. "github.com/lf-edge/ekuiper/pkg/infra"
  32. "github.com/lf-edge/ekuiper/pkg/message"
  33. )
  34. type SinkConf struct {
  35. Concurrency int `json:"concurrency"`
  36. Omitempty bool `json:"omitIfEmpty"`
  37. SendSingle bool `json:"sendSingle"`
  38. DataTemplate string `json:"dataTemplate"`
  39. Format string `json:"format"`
  40. SchemaId string `json:"schemaId"`
  41. Delimiter string `json:"delimiter"`
  42. BufferLength int `json:"bufferLength"`
  43. Fields []string `json:"fields"`
  44. DataField string `json:"dataField"`
  45. BatchSize int `json:"batchSize"`
  46. LingerInterval int `json:"lingerInterval"`
  47. conf.SinkConf
  48. }
  49. func (sc *SinkConf) isBatchSinkEnabled() bool {
  50. if sc.BatchSize > 0 || sc.LingerInterval > 0 {
  51. return true
  52. }
  53. return false
  54. }
  55. type SinkNode struct {
  56. *defaultSinkNode
  57. // static
  58. sinkType string
  59. mutex sync.RWMutex
  60. // configs (also static for sinks)
  61. options map[string]interface{}
  62. isMock bool
  63. // states varies after restart
  64. sinks []api.Sink
  65. }
  66. func NewSinkNode(name string, sinkType string, props map[string]interface{}) *SinkNode {
  67. bufferLength := 1024
  68. if c, ok := props["bufferLength"]; ok {
  69. if t, err := cast.ToInt(c, cast.STRICT); err != nil || t <= 0 {
  70. // invalid property bufferLength
  71. } else {
  72. bufferLength = t
  73. }
  74. }
  75. return &SinkNode{
  76. defaultSinkNode: &defaultSinkNode{
  77. input: make(chan interface{}, bufferLength),
  78. defaultNode: &defaultNode{
  79. name: name,
  80. concurrency: 1,
  81. ctx: nil,
  82. },
  83. },
  84. sinkType: sinkType,
  85. options: props,
  86. }
  87. }
  88. // NewSinkNodeWithSink Only for mock source, do not use it in production
  89. func NewSinkNodeWithSink(name string, sink api.Sink, props map[string]interface{}) *SinkNode {
  90. return &SinkNode{
  91. defaultSinkNode: &defaultSinkNode{
  92. input: make(chan interface{}, 1024),
  93. defaultNode: &defaultNode{
  94. name: name,
  95. concurrency: 1,
  96. ctx: nil,
  97. },
  98. },
  99. sinks: []api.Sink{sink},
  100. options: props,
  101. isMock: true,
  102. }
  103. }
  104. func (m *SinkNode) Open(ctx api.StreamContext, result chan<- error) {
  105. m.ctx = ctx
  106. logger := ctx.GetLogger()
  107. logger.Debugf("open sink node %s", m.name)
  108. go func() {
  109. err := infra.SafeRun(func() error {
  110. sconf, err := m.parseConf(logger)
  111. if err != nil {
  112. return err
  113. }
  114. tf, err := transform.GenTransform(sconf.DataTemplate, sconf.Format, sconf.SchemaId, sconf.Delimiter, sconf.DataField, sconf.Fields)
  115. if err != nil {
  116. msg := fmt.Sprintf("property dataTemplate %v is invalid: %v", sconf.DataTemplate, err)
  117. logger.Warnf(msg)
  118. return fmt.Errorf(msg)
  119. }
  120. ctx = context.WithValue(ctx.(*context.DefaultContext), context.TransKey, tf)
  121. m.reset()
  122. logger.Infof("open sink node %d instances", m.concurrency)
  123. for i := 0; i < m.concurrency; i++ { // workers
  124. go func(instance int) {
  125. panicOrError := infra.SafeRun(func() error {
  126. var (
  127. sink api.Sink
  128. err error
  129. )
  130. if !m.isMock {
  131. logger.Debugf("Trying to get sink for rule %s with options %v\n", ctx.GetRuleId(), m.options)
  132. sink, err = getSink(m.sinkType, m.options)
  133. if err != nil {
  134. return err
  135. }
  136. logger.Debugf("Successfully get the sink %s", m.sinkType)
  137. m.mutex.Lock()
  138. m.sinks = append(m.sinks, sink)
  139. m.mutex.Unlock()
  140. logger.Debugf("Now is to open sink for rule %s.\n", ctx.GetRuleId())
  141. if err := sink.Open(ctx); err != nil {
  142. return err
  143. }
  144. logger.Debugf("Successfully open sink for rule %s.\n", ctx.GetRuleId())
  145. } else {
  146. sink = m.sinks[instance]
  147. }
  148. stats, err := metric.NewStatManager(ctx, "sink")
  149. if err != nil {
  150. return err
  151. }
  152. m.mutex.Lock()
  153. m.statManagers = append(m.statManagers, stats)
  154. m.mutex.Unlock()
  155. // The sink flow is: receive -> batch -> cache -> send.
  156. // In the outside loop, send received data to batch/cache by dataCh and receive data be dataOutCh
  157. // Only need to deal with dataOutCh in the outer loop
  158. dataCh := make(chan []map[string]interface{}, sconf.BufferLength)
  159. var (
  160. dataOutCh <-chan []map[string]interface{}
  161. resendCh chan []map[string]interface{}
  162. sendManager *sinkUtil.SendManager
  163. c *cache.SyncCache
  164. rq *cache.SyncCache
  165. )
  166. if sconf.isBatchSinkEnabled() {
  167. sendManager, err = sinkUtil.NewSendManager(sconf.BatchSize, sconf.LingerInterval)
  168. if err != nil {
  169. return err
  170. }
  171. go sendManager.Run(ctx)
  172. }
  173. if !sconf.EnableCache {
  174. if sendManager != nil {
  175. dataOutCh = sendManager.GetOutputChan()
  176. } else {
  177. dataOutCh = dataCh
  178. }
  179. } else {
  180. if sendManager != nil {
  181. c = cache.NewSyncCache(ctx, sendManager.GetOutputChan(), result, &sconf.SinkConf, sconf.BufferLength)
  182. } else {
  183. c = cache.NewSyncCache(ctx, dataCh, result, &sconf.SinkConf, sconf.BufferLength)
  184. }
  185. if sconf.ResendAlterQueue {
  186. resendCh = make(chan []map[string]interface{}, sconf.BufferLength)
  187. rq = cache.NewSyncCache(ctx, resendCh, result, &sconf.SinkConf, sconf.BufferLength)
  188. }
  189. dataOutCh = c.Out
  190. }
  191. receiveQ := func(data interface{}) {
  192. processed := false
  193. if data, processed = m.preprocess(data); processed {
  194. return
  195. }
  196. stats.IncTotalRecordsIn()
  197. stats.SetBufferLength(bufferLen(dataCh, c, rq))
  198. outs := itemToMap(data)
  199. if sconf.Omitempty && (data == nil || len(outs) == 0) {
  200. ctx.GetLogger().Debugf("receive empty in sink")
  201. return
  202. }
  203. if sconf.isBatchSinkEnabled() {
  204. for _, out := range outs {
  205. sendManager.RecvData(out)
  206. }
  207. } else {
  208. select {
  209. case dataCh <- outs:
  210. case <-ctx.Done():
  211. }
  212. }
  213. if resendCh != nil {
  214. select {
  215. case resendCh <- nil:
  216. case <-ctx.Done():
  217. }
  218. }
  219. }
  220. normalQ := func(data []map[string]interface{}) {
  221. stats.ProcessTimeStart()
  222. stats.SetBufferLength(bufferLen(dataCh, c, rq))
  223. ctx.GetLogger().Debugf("sending data: %v", data)
  224. err := doCollectMaps(ctx, sink, sconf, data, stats, false)
  225. if sconf.EnableCache {
  226. ack := checkAck(ctx, data, err)
  227. if sconf.ResendAlterQueue {
  228. // If ack is false, add it to the resend queue
  229. if !ack {
  230. select {
  231. case resendCh <- data:
  232. case <-ctx.Done():
  233. }
  234. }
  235. // Always ack for the normal queue as fail items are handled by the resend queue
  236. select {
  237. case c.Ack <- true:
  238. case <-ctx.Done():
  239. }
  240. } else {
  241. select {
  242. case c.Ack <- ack:
  243. case <-ctx.Done():
  244. }
  245. }
  246. }
  247. stats.ProcessTimeEnd()
  248. }
  249. resendQ := func(data []map[string]interface{}) {
  250. ctx.GetLogger().Debugf("resend data: %v", data)
  251. stats.SetBufferLength(bufferLen(dataCh, c, rq))
  252. if sconf.ResendIndicatorField != "" {
  253. for _, item := range data {
  254. item[sconf.ResendIndicatorField] = true
  255. }
  256. }
  257. err := doCollectMaps(ctx, sink, sconf, data, stats, true)
  258. ack := checkAck(ctx, data, err)
  259. select {
  260. case rq.Ack <- ack:
  261. case <-ctx.Done():
  262. }
  263. }
  264. doneQ := func() {
  265. logger.Infof("sink node %s instance %d done", m.name, instance)
  266. if err := sink.Close(ctx); err != nil {
  267. logger.Warnf("close sink node %s instance %d fails: %v", m.name, instance, err)
  268. }
  269. }
  270. if resendCh == nil { // no resend strategy
  271. for {
  272. select {
  273. case data := <-m.input:
  274. receiveQ(data)
  275. case data := <-dataOutCh:
  276. normalQ(data)
  277. case <-ctx.Done():
  278. doneQ()
  279. return nil
  280. }
  281. }
  282. } else {
  283. if sconf.ResendPriority == 0 {
  284. for {
  285. select {
  286. case data := <-m.input:
  287. receiveQ(data)
  288. case data := <-dataOutCh:
  289. normalQ(data)
  290. case data := <-rq.Out:
  291. resendQ(data)
  292. case <-ctx.Done():
  293. doneQ()
  294. return nil
  295. }
  296. }
  297. } else if sconf.ResendPriority < 0 { // normal queue has higher priority
  298. for {
  299. select {
  300. case data := <-m.input:
  301. receiveQ(data)
  302. case data := <-dataOutCh:
  303. normalQ(data)
  304. default:
  305. select {
  306. case data := <-m.input:
  307. receiveQ(data)
  308. case data := <-dataOutCh:
  309. normalQ(data)
  310. case data := <-rq.Out:
  311. resendQ(data)
  312. case <-ctx.Done():
  313. doneQ()
  314. return nil
  315. }
  316. }
  317. }
  318. } else {
  319. for {
  320. select {
  321. case data := <-m.input:
  322. receiveQ(data)
  323. case data := <-rq.Out:
  324. resendQ(data)
  325. default:
  326. select {
  327. case data := <-m.input:
  328. receiveQ(data)
  329. case data := <-dataOutCh:
  330. normalQ(data)
  331. case data := <-rq.Out:
  332. resendQ(data)
  333. case <-ctx.Done():
  334. doneQ()
  335. return nil
  336. }
  337. }
  338. }
  339. }
  340. }
  341. })
  342. if panicOrError != nil {
  343. infra.DrainError(ctx, panicOrError, result)
  344. }
  345. }(i)
  346. }
  347. return nil
  348. })
  349. if err != nil {
  350. infra.DrainError(ctx, err, result)
  351. }
  352. }()
  353. }
  354. func bufferLen(dataCh chan []map[string]interface{}, c *cache.SyncCache, rq *cache.SyncCache) int64 {
  355. l := len(dataCh)
  356. if c != nil {
  357. l += c.CacheLength
  358. }
  359. if rq != nil {
  360. l += rq.CacheLength
  361. }
  362. return int64(l)
  363. }
  364. func checkAck(ctx api.StreamContext, data interface{}, err error) bool {
  365. if err != nil {
  366. if strings.HasPrefix(err.Error(), errorx.IOErr) { // do not log to prevent a lot of logs!
  367. return false
  368. } else {
  369. ctx.GetLogger().Warnf("sink node %s instance %d publish %s error: %v", ctx.GetOpId(), ctx.GetInstanceId(), data, err)
  370. }
  371. } else {
  372. ctx.GetLogger().Debugf("sent data: %v", data)
  373. }
  374. return true
  375. }
  376. func (m *SinkNode) parseConf(logger api.Logger) (*SinkConf, error) {
  377. sconf := &SinkConf{
  378. Concurrency: 1,
  379. Omitempty: false,
  380. SendSingle: false,
  381. DataTemplate: "",
  382. SinkConf: *conf.Config.Sink,
  383. BufferLength: 1024,
  384. }
  385. err := cast.MapToStruct(m.options, sconf)
  386. if err != nil {
  387. return nil, fmt.Errorf("read properties %v fail with error: %v", m.options, err)
  388. }
  389. if sconf.Concurrency <= 0 {
  390. logger.Warnf("invalid type for concurrency property, should be positive integer but found %t", sconf.Concurrency)
  391. sconf.Concurrency = 1
  392. }
  393. m.concurrency = sconf.Concurrency
  394. if sconf.Format == "" {
  395. sconf.Format = "json"
  396. } else if sconf.Format != message.FormatJson && sconf.Format != message.FormatProtobuf && sconf.Format != message.FormatBinary && sconf.Format != message.FormatCustom && sconf.Format != message.FormatDelimited {
  397. logger.Warnf("invalid type for format property, should be json protobuf or binary but found %s", sconf.Format)
  398. sconf.Format = "json"
  399. }
  400. err = cast.MapToStruct(m.options, &sconf.SinkConf)
  401. if err != nil {
  402. return nil, fmt.Errorf("read properties %v to cache conf fail with error: %v", m.options, err)
  403. }
  404. if sconf.DataField == "" {
  405. if v, ok := m.options["tableDataField"]; ok {
  406. sconf.DataField = v.(string)
  407. }
  408. }
  409. err = sconf.SinkConf.Validate()
  410. if err != nil {
  411. return nil, fmt.Errorf("invalid cache properties: %v", err)
  412. }
  413. return sconf, err
  414. }
  415. func (m *SinkNode) reset() {
  416. if !m.isMock {
  417. m.sinks = nil
  418. }
  419. m.statManagers = nil
  420. }
  421. func doCollectMaps(ctx api.StreamContext, sink api.Sink, sconf *SinkConf, outs []map[string]interface{}, stats metric.StatManager, isResend bool) error {
  422. if !sconf.SendSingle {
  423. return doCollectData(ctx, sink, outs, stats, isResend)
  424. } else {
  425. var err error
  426. for _, d := range outs {
  427. if sconf.Omitempty && (d == nil || len(d) == 0) {
  428. ctx.GetLogger().Debugf("receive empty in sink")
  429. continue
  430. }
  431. newErr := doCollectData(ctx, sink, d, stats, isResend)
  432. if newErr != nil {
  433. err = newErr
  434. }
  435. }
  436. return err
  437. }
  438. }
  439. func itemToMap(item interface{}) []map[string]interface{} {
  440. var outs []map[string]interface{}
  441. switch val := item.(type) {
  442. case error:
  443. outs = []map[string]interface{}{
  444. {"error": val.Error()},
  445. }
  446. break
  447. case xsql.Collection: // The order is important here, because some element is both a collection and a row, such as WindowTuples, JoinTuples, etc.
  448. outs = val.ToMaps()
  449. break
  450. case xsql.Row:
  451. outs = []map[string]interface{}{
  452. val.ToMap(),
  453. }
  454. break
  455. case []map[string]interface{}: // for test only
  456. outs = val
  457. break
  458. case *xsql.WatermarkTuple:
  459. // just ignore
  460. default:
  461. outs = []map[string]interface{}{
  462. {"error": fmt.Sprintf("result is not a map slice but found %#v", val)},
  463. }
  464. }
  465. return outs
  466. }
  467. // doCollectData outData must be map or []map
  468. func doCollectData(ctx api.StreamContext, sink api.Sink, outData interface{}, stats metric.StatManager, isResend bool) error {
  469. select {
  470. case <-ctx.Done():
  471. ctx.GetLogger().Infof("sink node %s instance %d stops data resending", ctx.GetOpId(), ctx.GetInstanceId())
  472. return nil
  473. default:
  474. if isResend {
  475. return resendDataToSink(ctx, sink, outData, stats)
  476. } else {
  477. return sendDataToSink(ctx, sink, outData, stats)
  478. }
  479. }
  480. }
  481. func sendDataToSink(ctx api.StreamContext, sink api.Sink, outData interface{}, stats metric.StatManager) error {
  482. if err := sink.Collect(ctx, outData); err != nil {
  483. stats.IncTotalExceptions(err.Error())
  484. return err
  485. } else {
  486. ctx.GetLogger().Debugf("success")
  487. stats.IncTotalRecordsOut()
  488. return nil
  489. }
  490. }
  491. func resendDataToSink(ctx api.StreamContext, sink api.Sink, outData interface{}, stats metric.StatManager) error {
  492. var err error
  493. switch st := sink.(type) {
  494. case api.ResendSink:
  495. err = st.CollectResend(ctx, outData)
  496. default:
  497. err = st.Collect(ctx, outData)
  498. }
  499. if err != nil {
  500. stats.IncTotalExceptions(err.Error())
  501. return err
  502. } else {
  503. ctx.GetLogger().Debugf("success resend")
  504. return nil
  505. }
  506. }
  507. func getSink(name string, action map[string]interface{}) (api.Sink, error) {
  508. var (
  509. s api.Sink
  510. err error
  511. )
  512. s, err = io.Sink(name)
  513. if s != nil {
  514. newAction := nodeConf.GetSinkConf(name, action)
  515. err = s.Configure(newAction)
  516. if err != nil {
  517. return nil, err
  518. }
  519. return s, nil
  520. } else {
  521. if err != nil {
  522. return nil, err
  523. } else {
  524. return nil, fmt.Errorf("sink %s not found", name)
  525. }
  526. }
  527. }
  528. // AddOutput Override defaultNode
  529. func (m *SinkNode) AddOutput(_ chan<- interface{}, name string) error {
  530. return fmt.Errorf("fail to add output %s, sink %s cannot add output", name, m.name)
  531. }
  532. // Broadcast Override defaultNode
  533. func (m *SinkNode) Broadcast(_ interface{}) error {
  534. return fmt.Errorf("sink %s cannot add broadcast", m.name)
  535. }