sink_node.go 12 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406
  1. // Copyright 2022 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. "github.com/lf-edge/ekuiper/internal/topo/context"
  22. "github.com/lf-edge/ekuiper/internal/topo/node/cache"
  23. nodeConf "github.com/lf-edge/ekuiper/internal/topo/node/conf"
  24. "github.com/lf-edge/ekuiper/internal/topo/node/metric"
  25. "github.com/lf-edge/ekuiper/internal/topo/transform"
  26. "github.com/lf-edge/ekuiper/internal/xsql"
  27. "github.com/lf-edge/ekuiper/pkg/api"
  28. "github.com/lf-edge/ekuiper/pkg/cast"
  29. "github.com/lf-edge/ekuiper/pkg/errorx"
  30. "github.com/lf-edge/ekuiper/pkg/infra"
  31. "github.com/lf-edge/ekuiper/pkg/message"
  32. )
  33. type SinkConf struct {
  34. Concurrency int `json:"concurrency"`
  35. RunAsync bool `json:"runAsync"` // deprecated, will remove in the next release
  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. conf.SinkConf
  44. }
  45. type SinkNode struct {
  46. *defaultSinkNode
  47. //static
  48. sinkType string
  49. mutex sync.RWMutex
  50. //configs (also static for sinks)
  51. options map[string]interface{}
  52. isMock bool
  53. //states varies after restart
  54. sinks []api.Sink
  55. }
  56. func NewSinkNode(name string, sinkType string, props map[string]interface{}) *SinkNode {
  57. bufferLength := 1024
  58. if c, ok := props["bufferLength"]; ok {
  59. if t, err := cast.ToInt(c, cast.STRICT); err != nil || t <= 0 {
  60. //invalid property bufferLength
  61. } else {
  62. bufferLength = t
  63. }
  64. }
  65. return &SinkNode{
  66. defaultSinkNode: &defaultSinkNode{
  67. input: make(chan interface{}, bufferLength),
  68. defaultNode: &defaultNode{
  69. name: name,
  70. concurrency: 1,
  71. ctx: nil,
  72. },
  73. },
  74. sinkType: sinkType,
  75. options: props,
  76. }
  77. }
  78. // NewSinkNodeWithSink Only for mock source, do not use it in production
  79. func NewSinkNodeWithSink(name string, sink api.Sink, props map[string]interface{}) *SinkNode {
  80. return &SinkNode{
  81. defaultSinkNode: &defaultSinkNode{
  82. input: make(chan interface{}, 1024),
  83. defaultNode: &defaultNode{
  84. name: name,
  85. concurrency: 1,
  86. ctx: nil,
  87. },
  88. },
  89. sinks: []api.Sink{sink},
  90. options: props,
  91. isMock: true,
  92. }
  93. }
  94. func (m *SinkNode) Open(ctx api.StreamContext, result chan<- error) {
  95. m.ctx = ctx
  96. logger := ctx.GetLogger()
  97. logger.Debugf("open sink node %s", m.name)
  98. go func() {
  99. err := infra.SafeRun(func() error {
  100. sconf, err := m.parseConf(logger)
  101. if err != nil {
  102. return err
  103. }
  104. tf, err := transform.GenTransform(sconf.DataTemplate, sconf.Format, sconf.SchemaId, sconf.Delimiter)
  105. if err != nil {
  106. msg := fmt.Sprintf("property dataTemplate %v is invalid: %v", sconf.DataTemplate, err)
  107. logger.Warnf(msg)
  108. return fmt.Errorf(msg)
  109. }
  110. ctx = context.WithValue(ctx.(*context.DefaultContext), context.TransKey, tf)
  111. m.reset()
  112. logger.Infof("open sink node %d instances", m.concurrency)
  113. for i := 0; i < m.concurrency; i++ { // workers
  114. go func(instance int) {
  115. panicOrError := infra.SafeRun(func() error {
  116. var (
  117. sink api.Sink
  118. err error
  119. )
  120. if !m.isMock {
  121. logger.Debugf("Trying to get sink for rule %s with options %v\n", ctx.GetRuleId(), m.options)
  122. sink, err = getSink(m.sinkType, m.options)
  123. if err != nil {
  124. return err
  125. }
  126. logger.Debugf("Successfully get the sink %s", m.sinkType)
  127. m.mutex.Lock()
  128. m.sinks = append(m.sinks, sink)
  129. m.mutex.Unlock()
  130. logger.Debugf("Now is to open sink for rule %s.\n", ctx.GetRuleId())
  131. if err := sink.Open(ctx); err != nil {
  132. return err
  133. }
  134. logger.Debugf("Successfully open sink for rule %s.\n", ctx.GetRuleId())
  135. } else {
  136. sink = m.sinks[instance]
  137. }
  138. stats, err := metric.NewStatManager(ctx, "sink")
  139. if err != nil {
  140. return err
  141. }
  142. m.mutex.Lock()
  143. m.statManagers = append(m.statManagers, stats)
  144. m.mutex.Unlock()
  145. if !sconf.EnableCache {
  146. for {
  147. select {
  148. case data := <-m.input:
  149. if temp, processed := m.preprocess(data); !processed {
  150. data = temp
  151. } else {
  152. break
  153. }
  154. stats.SetBufferLength(int64(len(m.input)))
  155. stats.IncTotalRecordsIn()
  156. if sconf.RunAsync {
  157. conf.Log.Warnf("RunAsync is deprecated and ignored.")
  158. }
  159. err := doCollect(ctx, sink, data, stats, sconf)
  160. if err != nil {
  161. logger.Warnf("sink collect error: %v", err)
  162. }
  163. case <-ctx.Done():
  164. logger.Infof("sink node %s instance %d done", m.name, instance)
  165. if err := sink.Close(ctx); err != nil {
  166. logger.Warnf("close sink node %s instance %d fails: %v", m.name, instance, err)
  167. }
  168. return nil
  169. }
  170. }
  171. } else {
  172. logger.Infof("Creating sink cache")
  173. if sconf.RunAsync { // async mode, the ack must have an id
  174. // is not supported and validated in the configure, should not go here
  175. return fmt.Errorf("async mode is not supported for cache sink")
  176. } else { // sync mode, the ack is already in order
  177. dataCh := make(chan []map[string]interface{}, sconf.BufferLength)
  178. c := cache.NewSyncCache(ctx, dataCh, result, stats, &sconf.SinkConf, sconf.BufferLength)
  179. for {
  180. select {
  181. case data := <-m.input:
  182. if temp, processed := m.preprocess(data); !processed {
  183. data = temp
  184. } else {
  185. break
  186. }
  187. stats.IncTotalRecordsIn()
  188. outs := itemToMap(data)
  189. if sconf.Omitempty && (data == nil || len(outs) == 0) {
  190. ctx.GetLogger().Debugf("receive empty in sink")
  191. return nil
  192. }
  193. select {
  194. case dataCh <- outs:
  195. case <-ctx.Done():
  196. }
  197. case data := <-c.Out:
  198. stats.ProcessTimeStart()
  199. ack := true
  200. err := doCollectMaps(ctx, sink, sconf, data, stats)
  201. // Only recoverable error should be cached
  202. if err != nil {
  203. if strings.HasPrefix(err.Error(), errorx.IOErr) { // do not log to prevent a lot of logs!
  204. ack = false
  205. } else {
  206. ctx.GetLogger().Warnf("sink node %s instance %d publish %s error: %v", ctx.GetOpId(), ctx.GetInstanceId(), data, err)
  207. }
  208. } else {
  209. ctx.GetLogger().Debugf("sent data to MQTT: %v", data)
  210. }
  211. select {
  212. case c.Ack <- ack:
  213. case <-ctx.Done():
  214. }
  215. stats.ProcessTimeEnd()
  216. case <-ctx.Done():
  217. logger.Infof("sink node %s instance %d done", m.name, instance)
  218. if err := sink.Close(ctx); err != nil {
  219. logger.Warnf("close sink node %s instance %d fails: %v", m.name, instance, err)
  220. }
  221. return nil
  222. }
  223. }
  224. }
  225. }
  226. })
  227. if panicOrError != nil {
  228. infra.DrainError(ctx, panicOrError, result)
  229. }
  230. }(i)
  231. }
  232. return nil
  233. })
  234. if err != nil {
  235. infra.DrainError(ctx, err, result)
  236. }
  237. }()
  238. }
  239. func (m *SinkNode) parseConf(logger api.Logger) (*SinkConf, error) {
  240. sconf := &SinkConf{
  241. Concurrency: 1,
  242. RunAsync: false,
  243. Omitempty: false,
  244. SendSingle: false,
  245. DataTemplate: "",
  246. SinkConf: *conf.Config.Sink,
  247. BufferLength: 1024,
  248. }
  249. err := cast.MapToStruct(m.options, sconf)
  250. if err != nil {
  251. return nil, fmt.Errorf("read properties %v fail with error: %v", m.options, err)
  252. }
  253. if sconf.Concurrency <= 0 {
  254. logger.Warnf("invalid type for concurrency property, should be positive integer but found %t", sconf.Concurrency)
  255. sconf.Concurrency = 1
  256. }
  257. m.concurrency = sconf.Concurrency
  258. if sconf.Format == "" {
  259. sconf.Format = "json"
  260. } else if sconf.Format != message.FormatJson && sconf.Format != message.FormatProtobuf && sconf.Format != message.FormatBinary && sconf.Format != message.FormatCustom && sconf.Format != message.FormatDelimited {
  261. logger.Warnf("invalid type for format property, should be json protobuf or binary but found %s", sconf.Format)
  262. sconf.Format = "json"
  263. }
  264. err = cast.MapToStruct(m.options, &sconf.SinkConf)
  265. if err != nil {
  266. return nil, fmt.Errorf("read properties %v to cache conf fail with error: %v", m.options, err)
  267. }
  268. if sconf.SinkConf.EnableCache && sconf.RunAsync {
  269. conf.Log.Warnf("RunAsync is deprecated and ignored.")
  270. return nil, fmt.Errorf("cache is not supported for async sink, do not use enableCache and runAsync properties together")
  271. }
  272. err = sconf.SinkConf.Validate()
  273. if err != nil {
  274. return nil, fmt.Errorf("invalid cache properties: %v", err)
  275. }
  276. return sconf, err
  277. }
  278. func (m *SinkNode) reset() {
  279. if !m.isMock {
  280. m.sinks = nil
  281. }
  282. m.statManagers = nil
  283. }
  284. func doCollect(ctx api.StreamContext, sink api.Sink, item interface{}, stats metric.StatManager, sconf *SinkConf) error {
  285. stats.ProcessTimeStart()
  286. defer stats.ProcessTimeEnd()
  287. outs := itemToMap(item)
  288. if sconf.Omitempty && (item == nil || len(outs) == 0) {
  289. ctx.GetLogger().Debugf("receive empty in sink")
  290. return nil
  291. }
  292. return doCollectMaps(ctx, sink, sconf, outs, stats)
  293. }
  294. func doCollectMaps(ctx api.StreamContext, sink api.Sink, sconf *SinkConf, outs []map[string]interface{}, stats metric.StatManager) error {
  295. if !sconf.SendSingle {
  296. return doCollectData(ctx, sink, outs, stats)
  297. } else {
  298. var err error
  299. for _, d := range outs {
  300. if sconf.Omitempty && (d == nil || len(d) == 0) {
  301. ctx.GetLogger().Debugf("receive empty in sink")
  302. continue
  303. }
  304. newErr := doCollectData(ctx, sink, d, stats)
  305. if newErr != nil {
  306. err = newErr
  307. }
  308. }
  309. return err
  310. }
  311. }
  312. func itemToMap(item interface{}) []map[string]interface{} {
  313. var outs []map[string]interface{}
  314. switch val := item.(type) {
  315. case error:
  316. outs = []map[string]interface{}{
  317. {"error": val.Error()},
  318. }
  319. break
  320. case xsql.Collection: // The order is important here, because some element is both a collection and a row, such as WindowTuples, JoinTuples, etc.
  321. outs = val.ToMaps()
  322. break
  323. case xsql.Row:
  324. outs = []map[string]interface{}{
  325. val.ToMap(),
  326. }
  327. break
  328. case []map[string]interface{}: // for test only
  329. outs = val
  330. break
  331. default:
  332. outs = []map[string]interface{}{
  333. {"error": fmt.Sprintf("result is not a map slice but found %#v", val)},
  334. }
  335. }
  336. return outs
  337. }
  338. // doCollectData outData must be map or []map
  339. func doCollectData(ctx api.StreamContext, sink api.Sink, outData interface{}, stats metric.StatManager) error {
  340. select {
  341. case <-ctx.Done():
  342. ctx.GetLogger().Infof("sink node %s instance %d stops data resending", ctx.GetOpId(), ctx.GetInstanceId())
  343. return nil
  344. default:
  345. if err := sink.Collect(ctx, outData); err != nil {
  346. stats.IncTotalExceptions(err.Error())
  347. return err
  348. } else {
  349. ctx.GetLogger().Debugf("success")
  350. stats.IncTotalRecordsOut()
  351. return nil
  352. }
  353. }
  354. }
  355. func getSink(name string, action map[string]interface{}) (api.Sink, error) {
  356. var (
  357. s api.Sink
  358. err error
  359. )
  360. s, err = io.Sink(name)
  361. if s != nil {
  362. newAction := nodeConf.GetSinkConf(name, action)
  363. err = s.Configure(newAction)
  364. if err != nil {
  365. return nil, err
  366. }
  367. return s, nil
  368. } else {
  369. if err != nil {
  370. return nil, err
  371. } else {
  372. return nil, fmt.Errorf("sink %s not found", name)
  373. }
  374. }
  375. }
  376. // AddOutput Override defaultNode
  377. func (m *SinkNode) AddOutput(_ chan<- interface{}, name string) error {
  378. return fmt.Errorf("fail to add output %s, sink %s cannot add output", name, m.name)
  379. }
  380. // Broadcast Override defaultNode
  381. func (m *SinkNode) Broadcast(_ interface{}) error {
  382. return fmt.Errorf("sink %s cannot add broadcast", m.name)
  383. }