sink_node.go 13 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448
  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. var sendManager *sinkUtil.SendManager
  156. if sconf.isBatchSinkEnabled() {
  157. sendManager, err = sinkUtil.NewSendManager(sconf.BatchSize, sconf.LingerInterval)
  158. if err != nil {
  159. return err
  160. }
  161. go sendManager.Run(ctx)
  162. go doCollectDataInBatch(ctx, sink, sendManager, stats)
  163. }
  164. if !sconf.EnableCache {
  165. for {
  166. select {
  167. case data := <-m.input:
  168. if temp, processed := m.preprocess(data); !processed {
  169. data = temp
  170. } else {
  171. break
  172. }
  173. stats.SetBufferLength(int64(len(m.input)))
  174. stats.IncTotalRecordsIn()
  175. err := doCollect(ctx, sink, data, sendManager, stats, sconf)
  176. if err != nil {
  177. logger.Warnf("sink collect error: %v", err)
  178. }
  179. case <-ctx.Done():
  180. logger.Infof("sink node %s instance %d done", m.name, instance)
  181. if err := sink.Close(ctx); err != nil {
  182. logger.Warnf("close sink node %s instance %d fails: %v", m.name, instance, err)
  183. }
  184. return nil
  185. }
  186. }
  187. } else {
  188. logger.Infof("Creating sink cache")
  189. { // sync mode, the ack is already in order
  190. dataCh := make(chan []map[string]interface{}, sconf.BufferLength)
  191. c := cache.NewSyncCache(ctx, dataCh, result, stats, &sconf.SinkConf, sconf.BufferLength)
  192. for {
  193. select {
  194. case data := <-m.input:
  195. if temp, processed := m.preprocess(data); !processed {
  196. data = temp
  197. } else {
  198. break
  199. }
  200. stats.IncTotalRecordsIn()
  201. outs := itemToMap(data)
  202. if sconf.Omitempty && (data == nil || len(outs) == 0) {
  203. ctx.GetLogger().Debugf("receive empty in sink")
  204. return nil
  205. }
  206. select {
  207. case dataCh <- outs:
  208. case <-ctx.Done():
  209. }
  210. case data := <-c.Out:
  211. stats.ProcessTimeStart()
  212. ack := true
  213. err := doCollectMaps(ctx, sink, sconf, data, sendManager, stats)
  214. // Only recoverable error should be cached
  215. if err != nil {
  216. if strings.HasPrefix(err.Error(), errorx.IOErr) { // do not log to prevent a lot of logs!
  217. ack = false
  218. } else {
  219. ctx.GetLogger().Warnf("sink node %s instance %d publish %s error: %v", ctx.GetOpId(), ctx.GetInstanceId(), data, err)
  220. }
  221. } else {
  222. ctx.GetLogger().Debugf("sent data to MQTT: %v", data)
  223. }
  224. select {
  225. case c.Ack <- ack:
  226. case <-ctx.Done():
  227. }
  228. stats.ProcessTimeEnd()
  229. case <-ctx.Done():
  230. logger.Infof("sink node %s instance %d done", m.name, instance)
  231. if err := sink.Close(ctx); err != nil {
  232. logger.Warnf("close sink node %s instance %d fails: %v", m.name, instance, err)
  233. }
  234. return nil
  235. }
  236. }
  237. }
  238. }
  239. })
  240. if panicOrError != nil {
  241. infra.DrainError(ctx, panicOrError, result)
  242. }
  243. }(i)
  244. }
  245. return nil
  246. })
  247. if err != nil {
  248. infra.DrainError(ctx, err, result)
  249. }
  250. }()
  251. }
  252. func (m *SinkNode) parseConf(logger api.Logger) (*SinkConf, error) {
  253. sconf := &SinkConf{
  254. Concurrency: 1,
  255. Omitempty: false,
  256. SendSingle: false,
  257. DataTemplate: "",
  258. SinkConf: *conf.Config.Sink,
  259. BufferLength: 1024,
  260. }
  261. err := cast.MapToStruct(m.options, sconf)
  262. if err != nil {
  263. return nil, fmt.Errorf("read properties %v fail with error: %v", m.options, err)
  264. }
  265. if sconf.Concurrency <= 0 {
  266. logger.Warnf("invalid type for concurrency property, should be positive integer but found %t", sconf.Concurrency)
  267. sconf.Concurrency = 1
  268. }
  269. m.concurrency = sconf.Concurrency
  270. if sconf.Format == "" {
  271. sconf.Format = "json"
  272. } else if sconf.Format != message.FormatJson && sconf.Format != message.FormatProtobuf && sconf.Format != message.FormatBinary && sconf.Format != message.FormatCustom && sconf.Format != message.FormatDelimited {
  273. logger.Warnf("invalid type for format property, should be json protobuf or binary but found %s", sconf.Format)
  274. sconf.Format = "json"
  275. }
  276. err = cast.MapToStruct(m.options, &sconf.SinkConf)
  277. if err != nil {
  278. return nil, fmt.Errorf("read properties %v to cache conf fail with error: %v", m.options, err)
  279. }
  280. if sconf.DataField == "" {
  281. if v, ok := m.options["tableDataField"]; ok {
  282. sconf.DataField = v.(string)
  283. }
  284. }
  285. err = sconf.SinkConf.Validate()
  286. if err != nil {
  287. return nil, fmt.Errorf("invalid cache properties: %v", err)
  288. }
  289. return sconf, err
  290. }
  291. func (m *SinkNode) reset() {
  292. if !m.isMock {
  293. m.sinks = nil
  294. }
  295. m.statManagers = nil
  296. }
  297. func doCollect(ctx api.StreamContext, sink api.Sink, item interface{}, sendManager *sinkUtil.SendManager, stats metric.StatManager, sconf *SinkConf) error {
  298. stats.ProcessTimeStart()
  299. defer stats.ProcessTimeEnd()
  300. outs := itemToMap(item)
  301. if sconf.Omitempty && (item == nil || len(outs) == 0) {
  302. ctx.GetLogger().Debugf("receive empty in sink")
  303. return nil
  304. }
  305. return doCollectMaps(ctx, sink, sconf, outs, sendManager, stats)
  306. }
  307. func doCollectMaps(ctx api.StreamContext, sink api.Sink, sconf *SinkConf, outs []map[string]interface{}, sendManager *sinkUtil.SendManager, stats metric.StatManager) error {
  308. if !sconf.SendSingle {
  309. return doCollectData(ctx, sink, outs, sendManager, stats)
  310. } else {
  311. var err error
  312. for _, d := range outs {
  313. if sconf.Omitempty && (d == nil || len(d) == 0) {
  314. ctx.GetLogger().Debugf("receive empty in sink")
  315. continue
  316. }
  317. newErr := doCollectData(ctx, sink, d, sendManager, stats)
  318. if newErr != nil {
  319. err = newErr
  320. }
  321. }
  322. return err
  323. }
  324. }
  325. func itemToMap(item interface{}) []map[string]interface{} {
  326. var outs []map[string]interface{}
  327. switch val := item.(type) {
  328. case error:
  329. outs = []map[string]interface{}{
  330. {"error": val.Error()},
  331. }
  332. break
  333. case xsql.Collection: // The order is important here, because some element is both a collection and a row, such as WindowTuples, JoinTuples, etc.
  334. outs = val.ToMaps()
  335. break
  336. case xsql.Row:
  337. outs = []map[string]interface{}{
  338. val.ToMap(),
  339. }
  340. break
  341. case []map[string]interface{}: // for test only
  342. outs = val
  343. break
  344. default:
  345. outs = []map[string]interface{}{
  346. {"error": fmt.Sprintf("result is not a map slice but found %#v", val)},
  347. }
  348. }
  349. return outs
  350. }
  351. // doCollectData outData must be map or []map
  352. func doCollectData(ctx api.StreamContext, sink api.Sink, outData interface{}, sendManager *sinkUtil.SendManager, stats metric.StatManager) error {
  353. if sendManager != nil {
  354. switch v := outData.(type) {
  355. case map[string]interface{}:
  356. sendManager.RecvData(v)
  357. case []map[string]interface{}:
  358. for _, d := range v {
  359. sendManager.RecvData(d)
  360. }
  361. }
  362. return nil
  363. }
  364. select {
  365. case <-ctx.Done():
  366. ctx.GetLogger().Infof("sink node %s instance %d stops data resending", ctx.GetOpId(), ctx.GetInstanceId())
  367. return nil
  368. default:
  369. return sendDataToSink(ctx, sink, outData, stats)
  370. }
  371. }
  372. func doCollectDataInBatch(ctx api.StreamContext, sink api.Sink, sendManager *sinkUtil.SendManager, stats metric.StatManager) {
  373. for {
  374. select {
  375. case <-ctx.Done():
  376. ctx.GetLogger().Infof("sink node %s instance %d stops data batch collecting", ctx.GetOpId(), ctx.GetInstanceId())
  377. return
  378. case outData := <-sendManager.GetOutputChan():
  379. if err := sendDataToSink(ctx, sink, outData, stats); err != nil {
  380. ctx.GetLogger().Warnf("sink node %s instance %d publish %s error: %v", ctx.GetOpId(), ctx.GetInstanceId(), outData, err)
  381. }
  382. }
  383. }
  384. }
  385. func sendDataToSink(ctx api.StreamContext, sink api.Sink, outData interface{}, stats metric.StatManager) error {
  386. if err := sink.Collect(ctx, outData); err != nil {
  387. stats.IncTotalExceptions(err.Error())
  388. return err
  389. } else {
  390. ctx.GetLogger().Debugf("success")
  391. stats.IncTotalRecordsOut()
  392. return nil
  393. }
  394. }
  395. func getSink(name string, action map[string]interface{}) (api.Sink, error) {
  396. var (
  397. s api.Sink
  398. err error
  399. )
  400. s, err = io.Sink(name)
  401. if s != nil {
  402. newAction := nodeConf.GetSinkConf(name, action)
  403. err = s.Configure(newAction)
  404. if err != nil {
  405. return nil, err
  406. }
  407. return s, nil
  408. } else {
  409. if err != nil {
  410. return nil, err
  411. } else {
  412. return nil, fmt.Errorf("sink %s not found", name)
  413. }
  414. }
  415. }
  416. // AddOutput Override defaultNode
  417. func (m *SinkNode) AddOutput(_ chan<- interface{}, name string) error {
  418. return fmt.Errorf("fail to add output %s, sink %s cannot add output", name, m.name)
  419. }
  420. // Broadcast Override defaultNode
  421. func (m *SinkNode) Broadcast(_ interface{}) error {
  422. return fmt.Errorf("sink %s cannot add broadcast", m.name)
  423. }