planner_test.go 87 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647648649650651652653654655656657658659660661662663664665666667668669670671672673674675676677678679680681682683684685686687688689690691692693694695696697698699700701702703704705706707708709710711712713714715716717718719720721722723724725726727728729730731732733734735736737738739740741742743744745746747748749750751752753754755756757758759760761762763764765766767768769770771772773774775776777778779780781782783784785786787788789790791792793794795796797798799800801802803804805806807808809810811812813814815816817818819820821822823824825826827828829830831832833834835836837838839840841842843844845846847848849850851852853854855856857858859860861862863864865866867868869870871872873874875876877878879880881882883884885886887888889890891892893894895896897898899900901902903904905906907908909910911912913914915916917918919920921922923924925926927928929930931932933934935936937938939940941942943944945946947948949950951952953954955956957958959960961962963964965966967968969970971972973974975976977978979980981982983984985986987988989990991992993994995996997998999100010011002100310041005100610071008100910101011101210131014101510161017101810191020102110221023102410251026102710281029103010311032103310341035103610371038103910401041104210431044104510461047104810491050105110521053105410551056105710581059106010611062106310641065106610671068106910701071107210731074107510761077107810791080108110821083108410851086108710881089109010911092109310941095109610971098109911001101110211031104110511061107110811091110111111121113111411151116111711181119112011211122112311241125112611271128112911301131113211331134113511361137113811391140114111421143114411451146114711481149115011511152115311541155115611571158115911601161116211631164116511661167116811691170117111721173117411751176117711781179118011811182118311841185118611871188118911901191119211931194119511961197119811991200120112021203120412051206120712081209121012111212121312141215121612171218121912201221122212231224122512261227122812291230123112321233123412351236123712381239124012411242124312441245124612471248124912501251125212531254125512561257125812591260126112621263126412651266126712681269127012711272127312741275127612771278127912801281128212831284128512861287128812891290129112921293129412951296129712981299130013011302130313041305130613071308130913101311131213131314131513161317131813191320132113221323132413251326132713281329133013311332133313341335133613371338133913401341134213431344134513461347134813491350135113521353135413551356135713581359136013611362136313641365136613671368136913701371137213731374137513761377137813791380138113821383138413851386138713881389139013911392139313941395139613971398139914001401140214031404140514061407140814091410141114121413141414151416141714181419142014211422142314241425142614271428142914301431143214331434143514361437143814391440144114421443144414451446144714481449145014511452145314541455145614571458145914601461146214631464146514661467146814691470147114721473147414751476147714781479148014811482148314841485148614871488148914901491149214931494149514961497149814991500150115021503150415051506150715081509151015111512151315141515151615171518151915201521152215231524152515261527152815291530153115321533153415351536153715381539154015411542154315441545154615471548154915501551155215531554155515561557155815591560156115621563156415651566156715681569157015711572157315741575157615771578157915801581158215831584158515861587158815891590159115921593159415951596159715981599160016011602160316041605160616071608160916101611161216131614161516161617161816191620162116221623162416251626162716281629163016311632163316341635163616371638163916401641164216431644164516461647164816491650165116521653165416551656165716581659166016611662166316641665166616671668166916701671167216731674167516761677167816791680168116821683168416851686168716881689169016911692169316941695169616971698169917001701170217031704170517061707170817091710171117121713171417151716171717181719172017211722172317241725172617271728172917301731173217331734173517361737173817391740174117421743174417451746174717481749175017511752175317541755175617571758175917601761176217631764176517661767176817691770177117721773177417751776177717781779178017811782178317841785178617871788178917901791179217931794179517961797179817991800180118021803180418051806180718081809181018111812181318141815181618171818181918201821182218231824182518261827182818291830183118321833183418351836183718381839184018411842184318441845184618471848184918501851185218531854185518561857185818591860186118621863186418651866186718681869187018711872187318741875187618771878187918801881188218831884188518861887188818891890189118921893189418951896189718981899190019011902190319041905190619071908190919101911191219131914191519161917191819191920192119221923192419251926192719281929193019311932193319341935193619371938193919401941194219431944194519461947194819491950195119521953195419551956195719581959196019611962196319641965196619671968196919701971197219731974197519761977197819791980198119821983198419851986198719881989199019911992199319941995199619971998199920002001200220032004200520062007200820092010201120122013201420152016201720182019202020212022202320242025202620272028202920302031203220332034203520362037203820392040204120422043204420452046204720482049205020512052205320542055205620572058205920602061206220632064206520662067206820692070207120722073207420752076207720782079208020812082208320842085208620872088208920902091209220932094209520962097209820992100210121022103210421052106210721082109211021112112211321142115211621172118211921202121212221232124212521262127212821292130213121322133213421352136213721382139214021412142214321442145214621472148214921502151215221532154215521562157215821592160216121622163216421652166216721682169217021712172217321742175217621772178217921802181218221832184218521862187218821892190219121922193219421952196219721982199220022012202220322042205220622072208220922102211221222132214221522162217221822192220222122222223222422252226222722282229223022312232223322342235223622372238223922402241224222432244224522462247224822492250225122522253225422552256225722582259226022612262226322642265226622672268226922702271227222732274227522762277227822792280228122822283228422852286228722882289229022912292229322942295229622972298229923002301230223032304230523062307230823092310231123122313231423152316231723182319232023212322232323242325232623272328232923302331233223332334233523362337233823392340234123422343234423452346234723482349235023512352235323542355235623572358235923602361236223632364236523662367236823692370237123722373237423752376237723782379238023812382238323842385238623872388238923902391239223932394239523962397239823992400240124022403240424052406240724082409241024112412241324142415241624172418241924202421242224232424242524262427242824292430243124322433243424352436243724382439244024412442244324442445244624472448244924502451245224532454245524562457245824592460246124622463246424652466246724682469247024712472247324742475247624772478247924802481248224832484248524862487248824892490249124922493249424952496249724982499250025012502250325042505250625072508250925102511251225132514251525162517251825192520252125222523252425252526252725282529253025312532253325342535253625372538253925402541254225432544254525462547254825492550255125522553255425552556255725582559256025612562256325642565256625672568256925702571257225732574257525762577257825792580258125822583258425852586258725882589259025912592259325942595259625972598259926002601260226032604260526062607260826092610261126122613261426152616261726182619262026212622262326242625262626272628262926302631263226332634263526362637263826392640264126422643264426452646264726482649265026512652265326542655265626572658265926602661266226632664266526662667266826692670267126722673267426752676267726782679268026812682268326842685268626872688268926902691269226932694269526962697269826992700270127022703270427052706270727082709271027112712271327142715271627172718271927202721272227232724272527262727272827292730273127322733273427352736273727382739274027412742274327442745274627472748274927502751275227532754275527562757275827592760276127622763276427652766276727682769277027712772277327742775277627772778277927802781278227832784278527862787278827892790279127922793279427952796279727982799280028012802280328042805280628072808280928102811281228132814281528162817281828192820282128222823282428252826282728282829283028312832283328342835283628372838283928402841284228432844284528462847284828492850285128522853285428552856285728582859286028612862286328642865286628672868286928702871287228732874287528762877287828792880288128822883288428852886288728882889289028912892289328942895289628972898289929002901290229032904290529062907290829092910291129122913291429152916291729182919292029212922292329242925292629272928292929302931293229332934293529362937293829392940294129422943294429452946294729482949295029512952295329542955295629572958295929602961296229632964296529662967296829692970297129722973297429752976297729782979298029812982298329842985298629872988298929902991299229932994299529962997299829993000300130023003300430053006300730083009301030113012301330143015301630173018301930203021302230233024302530263027302830293030303130323033303430353036303730383039304030413042304330443045304630473048304930503051305230533054305530563057305830593060306130623063306430653066306730683069307030713072307330743075307630773078307930803081308230833084308530863087308830893090309130923093309430953096309730983099310031013102310331043105310631073108310931103111311231133114311531163117311831193120312131223123312431253126312731283129313031313132313331343135313631373138313931403141314231433144314531463147
  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 planner
  15. import (
  16. "encoding/json"
  17. "fmt"
  18. "reflect"
  19. "strings"
  20. "testing"
  21. "github.com/gdexlab/go-render/render"
  22. "github.com/lf-edge/ekuiper/internal/pkg/store"
  23. "github.com/lf-edge/ekuiper/internal/testx"
  24. "github.com/lf-edge/ekuiper/internal/topo/node"
  25. "github.com/lf-edge/ekuiper/internal/xsql"
  26. "github.com/lf-edge/ekuiper/pkg/api"
  27. "github.com/lf-edge/ekuiper/pkg/ast"
  28. )
  29. func init() {
  30. testx.InitEnv()
  31. }
  32. func Test_createLogicalPlan(t *testing.T) {
  33. store, err := store.GetKV("stream")
  34. if err != nil {
  35. t.Error(err)
  36. return
  37. }
  38. streamSqls := map[string]string{
  39. "src1": `CREATE STREAM src1 (
  40. id1 BIGINT,
  41. temp BIGINT,
  42. name string,
  43. myarray array(string)
  44. ) WITH (DATASOURCE="src1", FORMAT="json", KEY="ts");`,
  45. "src2": `CREATE STREAM src2 (
  46. id2 BIGINT,
  47. hum BIGINT
  48. ) WITH (DATASOURCE="src2", FORMAT="json", KEY="ts", TIMESTAMP_FORMAT="YYYY-MM-dd HH:mm:ss");`,
  49. "tableInPlanner": `CREATE TABLE tableInPlanner (
  50. id BIGINT,
  51. name STRING,
  52. value STRING,
  53. hum BIGINT
  54. ) WITH (TYPE="file");`,
  55. }
  56. types := map[string]ast.StreamType{
  57. "src1": ast.TypeStream,
  58. "src2": ast.TypeStream,
  59. "tableInPlanner": ast.TypeTable,
  60. }
  61. for name, sql := range streamSqls {
  62. s, err := json.Marshal(&xsql.StreamInfo{
  63. StreamType: types[name],
  64. Statement: sql,
  65. })
  66. if err != nil {
  67. t.Error(err)
  68. t.Fail()
  69. }
  70. err = store.Set(name, string(s))
  71. if err != nil {
  72. t.Error(err)
  73. t.Fail()
  74. }
  75. }
  76. streams := make(map[string]*ast.StreamStmt)
  77. for n := range streamSqls {
  78. streamStmt, err := xsql.GetDataSource(store, n)
  79. if err != nil {
  80. t.Errorf("fail to get stream %s, please check if stream is created", n)
  81. return
  82. }
  83. streams[n] = streamStmt
  84. }
  85. // boolTrue = true
  86. boolFalse := false
  87. tests := []struct {
  88. sql string
  89. p LogicalPlan
  90. err string
  91. }{
  92. {
  93. sql: "select unnest(myarray) as col from src1",
  94. p: ProjectSetPlan{
  95. SrfMapping: map[string]struct{}{
  96. "col": {},
  97. },
  98. baseLogicalPlan: baseLogicalPlan{
  99. children: []LogicalPlan{
  100. ProjectPlan{
  101. baseLogicalPlan: baseLogicalPlan{
  102. children: []LogicalPlan{
  103. DataSourcePlan{
  104. baseLogicalPlan: baseLogicalPlan{},
  105. name: "src1",
  106. streamFields: map[string]*ast.JsonStreamField{
  107. "myarray": {
  108. Type: "array",
  109. Items: &ast.JsonStreamField{
  110. Type: "string",
  111. },
  112. },
  113. },
  114. streamStmt: streams["src1"],
  115. metaFields: []string{},
  116. }.Init(),
  117. },
  118. },
  119. fields: []ast.Field{
  120. {
  121. Name: "unnest",
  122. AName: "col",
  123. Expr: func() *ast.FieldRef {
  124. fr := &ast.FieldRef{
  125. StreamName: ast.AliasStream,
  126. Name: "col",
  127. AliasRef: &ast.AliasRef{
  128. Expression: &ast.Call{
  129. Name: "unnest",
  130. FuncType: ast.FuncTypeSrf,
  131. Args: []ast.Expr{
  132. &ast.FieldRef{
  133. StreamName: "src1",
  134. Name: "myarray",
  135. },
  136. },
  137. },
  138. },
  139. }
  140. fr.SetRefSource([]ast.StreamName{"src1"})
  141. return fr
  142. }(),
  143. },
  144. },
  145. }.Init(),
  146. },
  147. },
  148. }.Init(),
  149. },
  150. { // 0
  151. sql: "SELECT unnest(myarray), name from src1",
  152. p: ProjectSetPlan{
  153. SrfMapping: map[string]struct{}{
  154. "unnest": {},
  155. },
  156. baseLogicalPlan: baseLogicalPlan{
  157. children: []LogicalPlan{
  158. ProjectPlan{
  159. baseLogicalPlan: baseLogicalPlan{
  160. children: []LogicalPlan{
  161. DataSourcePlan{
  162. baseLogicalPlan: baseLogicalPlan{},
  163. name: "src1",
  164. streamFields: map[string]*ast.JsonStreamField{
  165. "myarray": {
  166. Type: "array",
  167. Items: &ast.JsonStreamField{
  168. Type: "string",
  169. },
  170. },
  171. "name": {
  172. Type: "string",
  173. },
  174. },
  175. streamStmt: streams["src1"],
  176. metaFields: []string{},
  177. }.Init(),
  178. },
  179. },
  180. fields: []ast.Field{
  181. {
  182. Expr: &ast.Call{
  183. Name: "unnest",
  184. FuncType: ast.FuncTypeSrf,
  185. Args: []ast.Expr{
  186. &ast.FieldRef{
  187. StreamName: "src1",
  188. Name: "myarray",
  189. },
  190. },
  191. },
  192. Name: "unnest",
  193. },
  194. {
  195. Name: "name",
  196. Expr: &ast.FieldRef{
  197. StreamName: "src1",
  198. Name: "name",
  199. },
  200. },
  201. },
  202. }.Init(),
  203. },
  204. },
  205. }.Init(),
  206. },
  207. { // 0
  208. sql: `SELECT myarray[temp] FROM src1`,
  209. p: ProjectPlan{
  210. baseLogicalPlan: baseLogicalPlan{
  211. children: []LogicalPlan{
  212. DataSourcePlan{
  213. baseLogicalPlan: baseLogicalPlan{},
  214. name: "src1",
  215. streamFields: map[string]*ast.JsonStreamField{
  216. "myarray": {
  217. Type: "array",
  218. Items: &ast.JsonStreamField{
  219. Type: "string",
  220. },
  221. },
  222. "temp": {
  223. Type: "bigint",
  224. },
  225. },
  226. streamStmt: streams["src1"],
  227. metaFields: []string{},
  228. }.Init(),
  229. },
  230. },
  231. fields: []ast.Field{
  232. {
  233. Expr: &ast.BinaryExpr{
  234. OP: ast.SUBSET,
  235. LHS: &ast.FieldRef{
  236. StreamName: "src1",
  237. Name: "myarray",
  238. },
  239. RHS: &ast.IndexExpr{Index: &ast.FieldRef{
  240. StreamName: "src1",
  241. Name: "temp",
  242. }},
  243. },
  244. Name: "kuiper_field_0",
  245. AName: "",
  246. },
  247. },
  248. isAggregate: false,
  249. sendMeta: false,
  250. }.Init(),
  251. },
  252. { // 1 optimize where to data source
  253. sql: `SELECT temp FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  254. p: ProjectPlan{
  255. baseLogicalPlan: baseLogicalPlan{
  256. children: []LogicalPlan{
  257. WindowPlan{
  258. baseLogicalPlan: baseLogicalPlan{
  259. children: []LogicalPlan{
  260. FilterPlan{
  261. baseLogicalPlan: baseLogicalPlan{
  262. children: []LogicalPlan{
  263. DataSourcePlan{
  264. name: "src1",
  265. streamFields: map[string]*ast.JsonStreamField{
  266. "name": {
  267. Type: "string",
  268. },
  269. "temp": {
  270. Type: "bigint",
  271. },
  272. },
  273. streamStmt: streams["src1"],
  274. metaFields: []string{},
  275. }.Init(),
  276. },
  277. },
  278. condition: &ast.BinaryExpr{
  279. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  280. OP: ast.EQ,
  281. RHS: &ast.StringLiteral{Val: "v1"},
  282. },
  283. }.Init(),
  284. },
  285. },
  286. condition: nil,
  287. wtype: ast.TUMBLING_WINDOW,
  288. length: 10000,
  289. interval: 0,
  290. limit: 0,
  291. }.Init(),
  292. },
  293. },
  294. fields: []ast.Field{
  295. {
  296. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  297. Name: "temp",
  298. AName: "",
  299. },
  300. },
  301. isAggregate: false,
  302. sendMeta: false,
  303. }.Init(),
  304. },
  305. { // 2 condition that cannot be optimized
  306. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 WHERE src1.temp > 20 OR src2.hum > 60 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  307. p: ProjectPlan{
  308. baseLogicalPlan: baseLogicalPlan{
  309. children: []LogicalPlan{
  310. JoinPlan{
  311. baseLogicalPlan: baseLogicalPlan{
  312. children: []LogicalPlan{
  313. WindowPlan{
  314. baseLogicalPlan: baseLogicalPlan{
  315. children: []LogicalPlan{
  316. DataSourcePlan{
  317. name: "src1",
  318. streamFields: map[string]*ast.JsonStreamField{
  319. "id1": {
  320. Type: "bigint",
  321. },
  322. "temp": {
  323. Type: "bigint",
  324. },
  325. },
  326. streamStmt: streams["src1"],
  327. metaFields: []string{},
  328. }.Init(),
  329. DataSourcePlan{
  330. name: "src2",
  331. streamFields: map[string]*ast.JsonStreamField{
  332. "hum": {
  333. Type: "bigint",
  334. },
  335. "id2": {
  336. Type: "bigint",
  337. },
  338. },
  339. streamStmt: streams["src2"],
  340. metaFields: []string{},
  341. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  342. }.Init(),
  343. },
  344. },
  345. condition: nil,
  346. wtype: ast.TUMBLING_WINDOW,
  347. length: 10000,
  348. interval: 0,
  349. limit: 0,
  350. }.Init(),
  351. },
  352. },
  353. from: &ast.Table{Name: "src1"},
  354. joins: ast.Joins{ast.Join{
  355. Name: "src2",
  356. JoinType: ast.INNER_JOIN,
  357. Expr: &ast.BinaryExpr{
  358. OP: ast.AND,
  359. LHS: &ast.BinaryExpr{
  360. LHS: &ast.BinaryExpr{
  361. OP: ast.GT,
  362. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  363. RHS: &ast.IntegerLiteral{Val: 20},
  364. },
  365. OP: ast.OR,
  366. RHS: &ast.BinaryExpr{
  367. OP: ast.GT,
  368. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  369. RHS: &ast.IntegerLiteral{Val: 60},
  370. },
  371. },
  372. RHS: &ast.BinaryExpr{
  373. OP: ast.EQ,
  374. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  375. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  376. },
  377. },
  378. }},
  379. }.Init(),
  380. },
  381. },
  382. fields: []ast.Field{
  383. {
  384. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  385. Name: "id1",
  386. AName: "",
  387. },
  388. },
  389. isAggregate: false,
  390. sendMeta: false,
  391. }.Init(),
  392. },
  393. { // 3 optimize window filter
  394. sql: `SELECT id1 FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10) FILTER( WHERE temp > 2)`,
  395. p: ProjectPlan{
  396. baseLogicalPlan: baseLogicalPlan{
  397. children: []LogicalPlan{
  398. WindowPlan{
  399. baseLogicalPlan: baseLogicalPlan{
  400. children: []LogicalPlan{
  401. FilterPlan{
  402. baseLogicalPlan: baseLogicalPlan{
  403. children: []LogicalPlan{
  404. DataSourcePlan{
  405. name: "src1",
  406. streamFields: map[string]*ast.JsonStreamField{
  407. "id1": {
  408. Type: "bigint",
  409. },
  410. "name": {
  411. Type: "string",
  412. },
  413. "temp": {
  414. Type: "bigint",
  415. },
  416. },
  417. streamStmt: streams["src1"],
  418. metaFields: []string{},
  419. }.Init(),
  420. },
  421. },
  422. condition: &ast.BinaryExpr{
  423. OP: ast.AND,
  424. LHS: &ast.BinaryExpr{
  425. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  426. OP: ast.EQ,
  427. RHS: &ast.StringLiteral{Val: "v1"},
  428. },
  429. RHS: &ast.BinaryExpr{
  430. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  431. OP: ast.GT,
  432. RHS: &ast.IntegerLiteral{Val: 2},
  433. },
  434. },
  435. }.Init(),
  436. },
  437. },
  438. condition: nil,
  439. wtype: ast.TUMBLING_WINDOW,
  440. length: 10000,
  441. interval: 0,
  442. limit: 0,
  443. }.Init(),
  444. },
  445. },
  446. fields: []ast.Field{
  447. {
  448. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  449. Name: "id1",
  450. AName: "",
  451. },
  452. },
  453. isAggregate: false,
  454. sendMeta: false,
  455. }.Init(),
  456. },
  457. { // 4. do not optimize count window
  458. sql: `SELECT * FROM src1 WHERE temp > 20 GROUP BY COUNTWINDOW(5,1) HAVING COUNT(*) > 2`,
  459. p: ProjectPlan{
  460. baseLogicalPlan: baseLogicalPlan{
  461. children: []LogicalPlan{
  462. HavingPlan{
  463. baseLogicalPlan: baseLogicalPlan{
  464. children: []LogicalPlan{
  465. FilterPlan{
  466. baseLogicalPlan: baseLogicalPlan{
  467. children: []LogicalPlan{
  468. WindowPlan{
  469. baseLogicalPlan: baseLogicalPlan{
  470. children: []LogicalPlan{
  471. DataSourcePlan{
  472. name: "src1",
  473. isWildCard: true,
  474. streamFields: map[string]*ast.JsonStreamField{
  475. "id1": {
  476. Type: "bigint",
  477. },
  478. "temp": {
  479. Type: "bigint",
  480. },
  481. "name": {
  482. Type: "string",
  483. },
  484. "myarray": {
  485. Type: "array",
  486. Items: &ast.JsonStreamField{
  487. Type: "string",
  488. },
  489. },
  490. },
  491. streamStmt: streams["src1"],
  492. metaFields: []string{},
  493. }.Init(),
  494. },
  495. },
  496. condition: nil,
  497. wtype: ast.COUNT_WINDOW,
  498. length: 5,
  499. interval: 1,
  500. limit: 0,
  501. }.Init(),
  502. },
  503. },
  504. condition: &ast.BinaryExpr{
  505. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  506. OP: ast.GT,
  507. RHS: &ast.IntegerLiteral{Val: 20},
  508. },
  509. }.Init(),
  510. },
  511. },
  512. condition: &ast.BinaryExpr{
  513. LHS: &ast.Call{Name: "count", FuncId: 0, Args: []ast.Expr{&ast.Wildcard{
  514. Token: ast.ASTERISK,
  515. }}, FuncType: ast.FuncTypeAgg},
  516. OP: ast.GT,
  517. RHS: &ast.IntegerLiteral{Val: 2},
  518. },
  519. }.Init(),
  520. },
  521. },
  522. fields: []ast.Field{
  523. {
  524. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  525. Name: "*",
  526. AName: "",
  527. },
  528. },
  529. isAggregate: false,
  530. sendMeta: false,
  531. }.Init(),
  532. },
  533. { // 5. optimize join on
  534. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  535. p: ProjectPlan{
  536. baseLogicalPlan: baseLogicalPlan{
  537. children: []LogicalPlan{
  538. JoinPlan{
  539. baseLogicalPlan: baseLogicalPlan{
  540. children: []LogicalPlan{
  541. WindowPlan{
  542. baseLogicalPlan: baseLogicalPlan{
  543. children: []LogicalPlan{
  544. FilterPlan{
  545. baseLogicalPlan: baseLogicalPlan{
  546. children: []LogicalPlan{
  547. DataSourcePlan{
  548. name: "src1",
  549. streamFields: map[string]*ast.JsonStreamField{
  550. "id1": {
  551. Type: "bigint",
  552. },
  553. "temp": {
  554. Type: "bigint",
  555. },
  556. },
  557. streamStmt: streams["src1"],
  558. metaFields: []string{},
  559. }.Init(),
  560. },
  561. },
  562. condition: &ast.BinaryExpr{
  563. RHS: &ast.BinaryExpr{
  564. OP: ast.GT,
  565. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  566. RHS: &ast.IntegerLiteral{Val: 20},
  567. },
  568. OP: ast.AND,
  569. LHS: &ast.BinaryExpr{
  570. OP: ast.GT,
  571. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  572. RHS: &ast.IntegerLiteral{Val: 111},
  573. },
  574. },
  575. }.Init(),
  576. FilterPlan{
  577. baseLogicalPlan: baseLogicalPlan{
  578. children: []LogicalPlan{
  579. DataSourcePlan{
  580. name: "src2",
  581. streamFields: map[string]*ast.JsonStreamField{
  582. "hum": {
  583. Type: "bigint",
  584. },
  585. "id2": {
  586. Type: "bigint",
  587. },
  588. },
  589. streamStmt: streams["src2"],
  590. metaFields: []string{},
  591. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  592. }.Init(),
  593. },
  594. },
  595. condition: &ast.BinaryExpr{
  596. OP: ast.LT,
  597. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  598. RHS: &ast.IntegerLiteral{Val: 60},
  599. },
  600. }.Init(),
  601. },
  602. },
  603. condition: nil,
  604. wtype: ast.TUMBLING_WINDOW,
  605. length: 10000,
  606. interval: 0,
  607. limit: 0,
  608. }.Init(),
  609. },
  610. },
  611. from: &ast.Table{
  612. Name: "src1",
  613. },
  614. joins: []ast.Join{
  615. {
  616. Name: "src2",
  617. Alias: "",
  618. JoinType: ast.INNER_JOIN,
  619. Expr: &ast.BinaryExpr{
  620. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  621. OP: ast.EQ,
  622. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  623. },
  624. },
  625. },
  626. }.Init(),
  627. },
  628. },
  629. fields: []ast.Field{
  630. {
  631. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  632. Name: "id1",
  633. AName: "",
  634. },
  635. },
  636. isAggregate: false,
  637. sendMeta: false,
  638. }.Init(),
  639. },
  640. { // 6. optimize outter join on
  641. sql: `SELECT id1 FROM src1 FULL JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  642. p: ProjectPlan{
  643. baseLogicalPlan: baseLogicalPlan{
  644. children: []LogicalPlan{
  645. JoinPlan{
  646. baseLogicalPlan: baseLogicalPlan{
  647. children: []LogicalPlan{
  648. WindowPlan{
  649. baseLogicalPlan: baseLogicalPlan{
  650. children: []LogicalPlan{
  651. FilterPlan{
  652. baseLogicalPlan: baseLogicalPlan{
  653. children: []LogicalPlan{
  654. DataSourcePlan{
  655. name: "src1",
  656. streamFields: map[string]*ast.JsonStreamField{
  657. "id1": {
  658. Type: "bigint",
  659. },
  660. "temp": {
  661. Type: "bigint",
  662. },
  663. },
  664. streamStmt: streams["src1"],
  665. metaFields: []string{},
  666. }.Init(),
  667. },
  668. },
  669. condition: &ast.BinaryExpr{
  670. OP: ast.GT,
  671. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  672. RHS: &ast.IntegerLiteral{Val: 111},
  673. },
  674. }.Init(),
  675. DataSourcePlan{
  676. name: "src2",
  677. streamFields: map[string]*ast.JsonStreamField{
  678. "hum": {
  679. Type: "bigint",
  680. },
  681. "id2": {
  682. Type: "bigint",
  683. },
  684. },
  685. streamStmt: streams["src2"],
  686. metaFields: []string{},
  687. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  688. }.Init(),
  689. },
  690. },
  691. condition: nil,
  692. wtype: ast.TUMBLING_WINDOW,
  693. length: 10000,
  694. interval: 0,
  695. limit: 0,
  696. }.Init(),
  697. },
  698. },
  699. from: &ast.Table{
  700. Name: "src1",
  701. },
  702. joins: []ast.Join{
  703. {
  704. Name: "src2",
  705. Alias: "",
  706. JoinType: ast.FULL_JOIN,
  707. Expr: &ast.BinaryExpr{
  708. OP: ast.AND,
  709. LHS: &ast.BinaryExpr{
  710. OP: ast.AND,
  711. LHS: &ast.BinaryExpr{
  712. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  713. OP: ast.EQ,
  714. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  715. },
  716. RHS: &ast.BinaryExpr{
  717. OP: ast.GT,
  718. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  719. RHS: &ast.IntegerLiteral{Val: 20},
  720. },
  721. },
  722. RHS: &ast.BinaryExpr{
  723. OP: ast.LT,
  724. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  725. RHS: &ast.IntegerLiteral{Val: 60},
  726. },
  727. },
  728. },
  729. },
  730. }.Init(),
  731. },
  732. },
  733. fields: []ast.Field{
  734. {
  735. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  736. Name: "id1",
  737. AName: "",
  738. },
  739. },
  740. isAggregate: false,
  741. sendMeta: false,
  742. }.Init(),
  743. },
  744. { // 7 window error for table
  745. sql: `SELECT value FROM tableInPlanner WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  746. p: nil,
  747. err: "cannot run window for TABLE sources",
  748. },
  749. { // 8 join table without window
  750. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and hum < 60 WHERE src1.id1 > 111`,
  751. p: ProjectPlan{
  752. baseLogicalPlan: baseLogicalPlan{
  753. children: []LogicalPlan{
  754. JoinPlan{
  755. baseLogicalPlan: baseLogicalPlan{
  756. children: []LogicalPlan{
  757. JoinAlignPlan{
  758. baseLogicalPlan: baseLogicalPlan{
  759. children: []LogicalPlan{
  760. FilterPlan{
  761. baseLogicalPlan: baseLogicalPlan{
  762. children: []LogicalPlan{
  763. DataSourcePlan{
  764. name: "src1",
  765. streamFields: map[string]*ast.JsonStreamField{
  766. "id1": {
  767. Type: "bigint",
  768. },
  769. "temp": {
  770. Type: "bigint",
  771. },
  772. },
  773. streamStmt: streams["src1"],
  774. metaFields: []string{},
  775. }.Init(),
  776. },
  777. },
  778. condition: &ast.BinaryExpr{
  779. RHS: &ast.BinaryExpr{
  780. OP: ast.GT,
  781. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  782. RHS: &ast.IntegerLiteral{Val: 20},
  783. },
  784. OP: ast.AND,
  785. LHS: &ast.BinaryExpr{
  786. OP: ast.GT,
  787. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  788. RHS: &ast.IntegerLiteral{Val: 111},
  789. },
  790. },
  791. }.Init(),
  792. DataSourcePlan{
  793. name: "tableInPlanner",
  794. streamFields: map[string]*ast.JsonStreamField{
  795. "hum": {
  796. Type: "bigint",
  797. },
  798. "id": {
  799. Type: "bigint",
  800. },
  801. },
  802. streamStmt: streams["tableInPlanner"],
  803. metaFields: []string{},
  804. }.Init(),
  805. },
  806. },
  807. Emitters: []string{"tableInPlanner"},
  808. }.Init(),
  809. },
  810. },
  811. from: &ast.Table{
  812. Name: "src1",
  813. },
  814. joins: []ast.Join{
  815. {
  816. Name: "tableInPlanner",
  817. Alias: "",
  818. JoinType: ast.INNER_JOIN,
  819. Expr: &ast.BinaryExpr{
  820. OP: ast.AND,
  821. LHS: &ast.BinaryExpr{
  822. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  823. OP: ast.EQ,
  824. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  825. },
  826. RHS: &ast.BinaryExpr{
  827. OP: ast.LT,
  828. LHS: &ast.FieldRef{Name: "hum", StreamName: "tableInPlanner"},
  829. RHS: &ast.IntegerLiteral{Val: 60},
  830. },
  831. },
  832. },
  833. },
  834. }.Init(),
  835. },
  836. },
  837. fields: []ast.Field{
  838. {
  839. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  840. Name: "id1",
  841. AName: "",
  842. },
  843. },
  844. isAggregate: false,
  845. sendMeta: false,
  846. }.Init(),
  847. },
  848. { // 9 join table with window
  849. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and tableInPlanner.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  850. p: ProjectPlan{
  851. baseLogicalPlan: baseLogicalPlan{
  852. children: []LogicalPlan{
  853. JoinPlan{
  854. baseLogicalPlan: baseLogicalPlan{
  855. children: []LogicalPlan{
  856. JoinAlignPlan{
  857. baseLogicalPlan: baseLogicalPlan{
  858. children: []LogicalPlan{
  859. WindowPlan{
  860. baseLogicalPlan: baseLogicalPlan{
  861. children: []LogicalPlan{
  862. DataSourcePlan{
  863. name: "src1",
  864. streamFields: map[string]*ast.JsonStreamField{
  865. "id1": {
  866. Type: "bigint",
  867. },
  868. "temp": {
  869. Type: "bigint",
  870. },
  871. },
  872. streamStmt: streams["src1"],
  873. metaFields: []string{},
  874. }.Init(),
  875. },
  876. },
  877. condition: nil,
  878. wtype: ast.TUMBLING_WINDOW,
  879. length: 10000,
  880. interval: 0,
  881. limit: 0,
  882. }.Init(),
  883. DataSourcePlan{
  884. name: "tableInPlanner",
  885. streamFields: map[string]*ast.JsonStreamField{
  886. "hum": {
  887. Type: "bigint",
  888. },
  889. "id": {
  890. Type: "bigint",
  891. },
  892. },
  893. streamStmt: streams["tableInPlanner"],
  894. metaFields: []string{},
  895. }.Init(),
  896. },
  897. },
  898. Emitters: []string{"tableInPlanner"},
  899. }.Init(),
  900. },
  901. },
  902. from: &ast.Table{
  903. Name: "src1",
  904. },
  905. joins: []ast.Join{
  906. {
  907. Name: "tableInPlanner",
  908. Alias: "",
  909. JoinType: ast.INNER_JOIN,
  910. Expr: &ast.BinaryExpr{
  911. OP: ast.AND,
  912. LHS: &ast.BinaryExpr{
  913. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  914. OP: ast.EQ,
  915. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  916. },
  917. RHS: &ast.BinaryExpr{
  918. RHS: &ast.BinaryExpr{
  919. OP: ast.AND,
  920. LHS: &ast.BinaryExpr{
  921. OP: ast.GT,
  922. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  923. RHS: &ast.IntegerLiteral{Val: 20},
  924. },
  925. RHS: &ast.BinaryExpr{
  926. OP: ast.LT,
  927. LHS: &ast.FieldRef{Name: "hum", StreamName: "tableInPlanner"},
  928. RHS: &ast.IntegerLiteral{Val: 60},
  929. },
  930. },
  931. OP: ast.AND,
  932. LHS: &ast.BinaryExpr{
  933. OP: ast.GT,
  934. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  935. RHS: &ast.IntegerLiteral{Val: 111},
  936. },
  937. },
  938. },
  939. },
  940. },
  941. }.Init(),
  942. },
  943. },
  944. fields: []ast.Field{
  945. {
  946. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  947. Name: "id1",
  948. AName: "",
  949. },
  950. },
  951. isAggregate: false,
  952. sendMeta: false,
  953. }.Init(),
  954. },
  955. { // 10 meta
  956. sql: `SELECT temp, meta(id) AS eid,meta(Humidity->Device) AS hdevice FROM src1 WHERE meta(device)="demo2"`,
  957. p: ProjectPlan{
  958. baseLogicalPlan: baseLogicalPlan{
  959. children: []LogicalPlan{
  960. FilterPlan{
  961. baseLogicalPlan: baseLogicalPlan{
  962. children: []LogicalPlan{
  963. DataSourcePlan{
  964. name: "src1",
  965. streamFields: map[string]*ast.JsonStreamField{
  966. "temp": {
  967. Type: "bigint",
  968. },
  969. },
  970. streamStmt: streams["src1"],
  971. metaFields: []string{"Humidity", "device", "id"},
  972. }.Init(),
  973. },
  974. },
  975. condition: &ast.BinaryExpr{
  976. LHS: &ast.Call{
  977. Name: "meta",
  978. FuncId: 2,
  979. Args: []ast.Expr{&ast.MetaRef{
  980. Name: "device",
  981. StreamName: ast.DefaultStream,
  982. }},
  983. },
  984. OP: ast.EQ,
  985. RHS: &ast.StringLiteral{
  986. Val: "demo2",
  987. },
  988. },
  989. }.Init(),
  990. },
  991. },
  992. fields: []ast.Field{
  993. {
  994. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  995. Name: "temp",
  996. AName: "",
  997. }, {
  998. Expr: &ast.FieldRef{Name: "eid", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  999. &ast.Call{Name: "meta", FuncId: 0, Args: []ast.Expr{&ast.MetaRef{
  1000. Name: "id",
  1001. StreamName: ast.DefaultStream,
  1002. }}},
  1003. []ast.StreamName{},
  1004. nil,
  1005. )},
  1006. Name: "meta",
  1007. AName: "eid",
  1008. }, {
  1009. Expr: &ast.FieldRef{Name: "hdevice", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1010. &ast.Call{Name: "meta", FuncId: 1, Args: []ast.Expr{
  1011. &ast.BinaryExpr{
  1012. OP: ast.ARROW,
  1013. LHS: &ast.MetaRef{Name: "Humidity", StreamName: ast.DefaultStream},
  1014. RHS: &ast.JsonFieldRef{Name: "Device"},
  1015. },
  1016. }},
  1017. []ast.StreamName{},
  1018. nil,
  1019. )},
  1020. Name: "meta",
  1021. AName: "hdevice",
  1022. },
  1023. },
  1024. isAggregate: false,
  1025. sendMeta: false,
  1026. }.Init(),
  1027. },
  1028. { // 11 join with same name field and aliased
  1029. sql: `SELECT src2.hum AS hum1, tableInPlanner.hum AS hum2 FROM src2 INNER JOIN tableInPlanner on id2 = id WHERE hum1 > hum2`,
  1030. p: ProjectPlan{
  1031. baseLogicalPlan: baseLogicalPlan{
  1032. children: []LogicalPlan{
  1033. JoinPlan{
  1034. baseLogicalPlan: baseLogicalPlan{
  1035. children: []LogicalPlan{
  1036. JoinAlignPlan{
  1037. baseLogicalPlan: baseLogicalPlan{
  1038. children: []LogicalPlan{
  1039. DataSourcePlan{
  1040. name: "src2",
  1041. streamFields: map[string]*ast.JsonStreamField{
  1042. "hum": {
  1043. Type: "bigint",
  1044. },
  1045. "id2": {
  1046. Type: "bigint",
  1047. },
  1048. },
  1049. streamStmt: streams["src2"],
  1050. metaFields: []string{},
  1051. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  1052. }.Init(),
  1053. DataSourcePlan{
  1054. name: "tableInPlanner",
  1055. streamFields: map[string]*ast.JsonStreamField{
  1056. "hum": {
  1057. Type: "bigint",
  1058. },
  1059. "id": {
  1060. Type: "bigint",
  1061. },
  1062. },
  1063. streamStmt: streams["tableInPlanner"],
  1064. metaFields: []string{},
  1065. }.Init(),
  1066. },
  1067. },
  1068. Emitters: []string{"tableInPlanner"},
  1069. }.Init(),
  1070. },
  1071. },
  1072. from: &ast.Table{
  1073. Name: "src2",
  1074. },
  1075. joins: []ast.Join{
  1076. {
  1077. Name: "tableInPlanner",
  1078. Alias: "",
  1079. JoinType: ast.INNER_JOIN,
  1080. Expr: &ast.BinaryExpr{
  1081. RHS: &ast.BinaryExpr{
  1082. OP: ast.EQ,
  1083. LHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  1084. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  1085. },
  1086. OP: ast.AND,
  1087. LHS: &ast.BinaryExpr{
  1088. OP: ast.GT,
  1089. LHS: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1090. &ast.FieldRef{
  1091. Name: "hum",
  1092. StreamName: "src2",
  1093. },
  1094. []ast.StreamName{"src2"},
  1095. &boolFalse,
  1096. )},
  1097. RHS: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1098. &ast.FieldRef{
  1099. Name: "hum",
  1100. StreamName: "tableInPlanner",
  1101. },
  1102. []ast.StreamName{"tableInPlanner"},
  1103. &boolFalse,
  1104. )},
  1105. },
  1106. },
  1107. },
  1108. },
  1109. }.Init(),
  1110. },
  1111. },
  1112. fields: []ast.Field{
  1113. {
  1114. Expr: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1115. &ast.FieldRef{
  1116. Name: "hum",
  1117. StreamName: "src2",
  1118. },
  1119. []ast.StreamName{"src2"},
  1120. &boolFalse,
  1121. )},
  1122. Name: "hum",
  1123. AName: "hum1",
  1124. }, {
  1125. Expr: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1126. &ast.FieldRef{
  1127. Name: "hum",
  1128. StreamName: "tableInPlanner",
  1129. },
  1130. []ast.StreamName{"tableInPlanner"},
  1131. &boolFalse,
  1132. )},
  1133. Name: "hum",
  1134. AName: "hum2",
  1135. },
  1136. },
  1137. isAggregate: false,
  1138. sendMeta: false,
  1139. }.Init(),
  1140. },
  1141. { // 12 meta with more fields
  1142. sql: `SELECT temp, meta(*) as m FROM src1 WHERE meta(device)="demo2"`,
  1143. p: ProjectPlan{
  1144. baseLogicalPlan: baseLogicalPlan{
  1145. children: []LogicalPlan{
  1146. FilterPlan{
  1147. baseLogicalPlan: baseLogicalPlan{
  1148. children: []LogicalPlan{
  1149. DataSourcePlan{
  1150. name: "src1",
  1151. streamFields: map[string]*ast.JsonStreamField{
  1152. "temp": {
  1153. Type: "bigint",
  1154. },
  1155. },
  1156. streamStmt: streams["src1"],
  1157. metaFields: []string{},
  1158. allMeta: true,
  1159. }.Init(),
  1160. },
  1161. },
  1162. condition: &ast.BinaryExpr{
  1163. LHS: &ast.Call{
  1164. Name: "meta",
  1165. FuncId: 1,
  1166. Args: []ast.Expr{&ast.MetaRef{
  1167. Name: "device",
  1168. StreamName: ast.DefaultStream,
  1169. }},
  1170. },
  1171. OP: ast.EQ,
  1172. RHS: &ast.StringLiteral{
  1173. Val: "demo2",
  1174. },
  1175. },
  1176. }.Init(),
  1177. },
  1178. },
  1179. fields: []ast.Field{
  1180. {
  1181. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1182. Name: "temp",
  1183. AName: "",
  1184. }, {
  1185. Expr: &ast.FieldRef{Name: "m", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1186. &ast.Call{Name: "meta", FuncId: 0, Args: []ast.Expr{&ast.MetaRef{
  1187. Name: "*",
  1188. StreamName: ast.DefaultStream,
  1189. }}},
  1190. []ast.StreamName{},
  1191. nil,
  1192. )},
  1193. Name: "meta",
  1194. AName: "m",
  1195. },
  1196. },
  1197. isAggregate: false,
  1198. sendMeta: false,
  1199. }.Init(),
  1200. },
  1201. { // 13 analytic function plan
  1202. sql: `SELECT latest(lag(name)), id1 FROM src1 WHERE lag(temp) > temp`,
  1203. p: ProjectPlan{
  1204. baseLogicalPlan: baseLogicalPlan{
  1205. children: []LogicalPlan{
  1206. FilterPlan{
  1207. baseLogicalPlan: baseLogicalPlan{
  1208. children: []LogicalPlan{
  1209. AnalyticFuncsPlan{
  1210. baseLogicalPlan: baseLogicalPlan{
  1211. children: []LogicalPlan{
  1212. DataSourcePlan{
  1213. name: "src1",
  1214. streamFields: map[string]*ast.JsonStreamField{
  1215. "id1": {
  1216. Type: "bigint",
  1217. },
  1218. "name": {
  1219. Type: "string",
  1220. },
  1221. "temp": {
  1222. Type: "bigint",
  1223. },
  1224. },
  1225. streamStmt: streams["src1"],
  1226. metaFields: []string{},
  1227. }.Init(),
  1228. },
  1229. },
  1230. funcs: []*ast.Call{
  1231. {
  1232. Name: "lag",
  1233. FuncId: 2,
  1234. CachedField: "$$a_lag_2",
  1235. Args: []ast.Expr{&ast.FieldRef{
  1236. Name: "temp",
  1237. StreamName: "src1",
  1238. }},
  1239. },
  1240. {
  1241. Name: "latest", FuncId: 1, CachedField: "$$a_latest_1", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1242. },
  1243. {
  1244. Name: "lag", FuncId: 0, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}},
  1245. },
  1246. },
  1247. }.Init(),
  1248. },
  1249. },
  1250. condition: &ast.BinaryExpr{
  1251. LHS: &ast.Call{
  1252. Name: "lag",
  1253. FuncId: 2,
  1254. Args: []ast.Expr{&ast.FieldRef{
  1255. Name: "temp",
  1256. StreamName: "src1",
  1257. }},
  1258. CachedField: "$$a_lag_2",
  1259. Cached: true,
  1260. },
  1261. OP: ast.GT,
  1262. RHS: &ast.FieldRef{
  1263. Name: "temp",
  1264. StreamName: "src1",
  1265. },
  1266. },
  1267. }.Init(),
  1268. },
  1269. },
  1270. fields: []ast.Field{
  1271. {
  1272. Expr: &ast.Call{
  1273. Name: "latest",
  1274. FuncId: 1,
  1275. Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1276. CachedField: "$$a_latest_1",
  1277. Cached: true,
  1278. },
  1279. Name: "latest",
  1280. }, {
  1281. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1282. Name: "id1",
  1283. },
  1284. },
  1285. isAggregate: false,
  1286. sendMeta: false,
  1287. }.Init(),
  1288. },
  1289. { // 14
  1290. sql: `SELECT name, *, meta(device) FROM src1`,
  1291. p: ProjectPlan{
  1292. baseLogicalPlan: baseLogicalPlan{
  1293. children: []LogicalPlan{
  1294. DataSourcePlan{
  1295. baseLogicalPlan: baseLogicalPlan{},
  1296. name: "src1",
  1297. streamFields: map[string]*ast.JsonStreamField{
  1298. "id1": {
  1299. Type: "bigint",
  1300. },
  1301. "temp": {
  1302. Type: "bigint",
  1303. },
  1304. "name": {
  1305. Type: "string",
  1306. },
  1307. "myarray": {
  1308. Type: "array",
  1309. Items: &ast.JsonStreamField{
  1310. Type: "string",
  1311. },
  1312. },
  1313. },
  1314. streamStmt: streams["src1"],
  1315. metaFields: []string{"device"},
  1316. isWildCard: true,
  1317. }.Init(),
  1318. },
  1319. },
  1320. fields: []ast.Field{
  1321. {
  1322. Expr: &ast.FieldRef{Name: "name", StreamName: "src1"},
  1323. Name: "name",
  1324. AName: "",
  1325. },
  1326. {
  1327. Name: "*",
  1328. Expr: &ast.Wildcard{
  1329. Token: ast.ASTERISK,
  1330. },
  1331. },
  1332. {
  1333. Name: "meta",
  1334. Expr: &ast.Call{
  1335. Name: "meta",
  1336. Args: []ast.Expr{
  1337. &ast.MetaRef{
  1338. StreamName: ast.DefaultStream,
  1339. Name: "device",
  1340. },
  1341. },
  1342. },
  1343. },
  1344. },
  1345. isAggregate: false,
  1346. allWildcard: true,
  1347. sendMeta: false,
  1348. }.Init(),
  1349. },
  1350. { // 15 analytic function over partition plan
  1351. sql: `SELECT latest(lag(name)) OVER (PARTITION BY temp), id1 FROM src1 WHERE lag(temp) > temp`,
  1352. p: ProjectPlan{
  1353. baseLogicalPlan: baseLogicalPlan{
  1354. children: []LogicalPlan{
  1355. FilterPlan{
  1356. baseLogicalPlan: baseLogicalPlan{
  1357. children: []LogicalPlan{
  1358. AnalyticFuncsPlan{
  1359. baseLogicalPlan: baseLogicalPlan{
  1360. children: []LogicalPlan{
  1361. DataSourcePlan{
  1362. name: "src1",
  1363. streamFields: map[string]*ast.JsonStreamField{
  1364. "id1": {
  1365. Type: "bigint",
  1366. },
  1367. "name": {
  1368. Type: "string",
  1369. },
  1370. "temp": {
  1371. Type: "bigint",
  1372. },
  1373. },
  1374. streamStmt: streams["src1"],
  1375. metaFields: []string{},
  1376. }.Init(),
  1377. },
  1378. },
  1379. funcs: []*ast.Call{
  1380. {
  1381. Name: "lag",
  1382. FuncId: 2,
  1383. CachedField: "$$a_lag_2",
  1384. Args: []ast.Expr{&ast.FieldRef{
  1385. Name: "temp",
  1386. StreamName: "src1",
  1387. }},
  1388. },
  1389. {
  1390. Name: "latest", FuncId: 1, CachedField: "$$a_latest_1", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}}, Partition: &ast.PartitionExpr{Exprs: []ast.Expr{&ast.FieldRef{Name: "temp", StreamName: "src1"}}},
  1391. },
  1392. {
  1393. Name: "lag", FuncId: 0, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}},
  1394. },
  1395. },
  1396. }.Init(),
  1397. },
  1398. },
  1399. condition: &ast.BinaryExpr{
  1400. LHS: &ast.Call{
  1401. Name: "lag",
  1402. FuncId: 2,
  1403. Args: []ast.Expr{&ast.FieldRef{
  1404. Name: "temp",
  1405. StreamName: "src1",
  1406. }},
  1407. CachedField: "$$a_lag_2",
  1408. Cached: true,
  1409. },
  1410. OP: ast.GT,
  1411. RHS: &ast.FieldRef{
  1412. Name: "temp",
  1413. StreamName: "src1",
  1414. },
  1415. },
  1416. }.Init(),
  1417. },
  1418. },
  1419. fields: []ast.Field{
  1420. {
  1421. Expr: &ast.Call{
  1422. Name: "latest",
  1423. FuncId: 1,
  1424. Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1425. CachedField: "$$a_latest_1",
  1426. Cached: true,
  1427. Partition: &ast.PartitionExpr{
  1428. Exprs: []ast.Expr{
  1429. &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1430. },
  1431. },
  1432. },
  1433. Name: "latest",
  1434. }, {
  1435. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1436. Name: "id1",
  1437. },
  1438. },
  1439. isAggregate: false,
  1440. sendMeta: false,
  1441. }.Init(),
  1442. },
  1443. { // 16 analytic function over partition when plan
  1444. sql: `SELECT latest(lag(name)) OVER (PARTITION BY temp WHEN temp > 12), id1 FROM src1 WHERE lag(temp) > temp`,
  1445. p: ProjectPlan{
  1446. baseLogicalPlan: baseLogicalPlan{
  1447. children: []LogicalPlan{
  1448. FilterPlan{
  1449. baseLogicalPlan: baseLogicalPlan{
  1450. children: []LogicalPlan{
  1451. AnalyticFuncsPlan{
  1452. baseLogicalPlan: baseLogicalPlan{
  1453. children: []LogicalPlan{
  1454. DataSourcePlan{
  1455. name: "src1",
  1456. streamFields: map[string]*ast.JsonStreamField{
  1457. "id1": {
  1458. Type: "bigint",
  1459. },
  1460. "name": {
  1461. Type: "string",
  1462. },
  1463. "temp": {
  1464. Type: "bigint",
  1465. },
  1466. },
  1467. streamStmt: streams["src1"],
  1468. metaFields: []string{},
  1469. }.Init(),
  1470. },
  1471. },
  1472. funcs: []*ast.Call{
  1473. {
  1474. Name: "lag",
  1475. FuncId: 2,
  1476. CachedField: "$$a_lag_2",
  1477. Args: []ast.Expr{&ast.FieldRef{
  1478. Name: "temp",
  1479. StreamName: "src1",
  1480. }},
  1481. },
  1482. {
  1483. Name: "latest", FuncId: 1, CachedField: "$$a_latest_1", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}}, Partition: &ast.PartitionExpr{Exprs: []ast.Expr{&ast.FieldRef{Name: "temp", StreamName: "src1"}}}, WhenExpr: &ast.BinaryExpr{LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"}, OP: ast.GT, RHS: &ast.IntegerLiteral{Val: 12}},
  1484. },
  1485. {
  1486. Name: "lag", FuncId: 0, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}},
  1487. },
  1488. },
  1489. }.Init(),
  1490. },
  1491. },
  1492. condition: &ast.BinaryExpr{
  1493. LHS: &ast.Call{
  1494. Name: "lag",
  1495. FuncId: 2,
  1496. Args: []ast.Expr{&ast.FieldRef{
  1497. Name: "temp",
  1498. StreamName: "src1",
  1499. }},
  1500. CachedField: "$$a_lag_2",
  1501. Cached: true,
  1502. },
  1503. OP: ast.GT,
  1504. RHS: &ast.FieldRef{
  1505. Name: "temp",
  1506. StreamName: "src1",
  1507. },
  1508. },
  1509. }.Init(),
  1510. },
  1511. },
  1512. fields: []ast.Field{
  1513. {
  1514. Expr: &ast.Call{
  1515. Name: "latest",
  1516. FuncId: 1,
  1517. Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1518. CachedField: "$$a_latest_1",
  1519. Cached: true,
  1520. Partition: &ast.PartitionExpr{
  1521. Exprs: []ast.Expr{
  1522. &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1523. },
  1524. },
  1525. WhenExpr: &ast.BinaryExpr{
  1526. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1527. OP: ast.GT,
  1528. RHS: &ast.IntegerLiteral{Val: 12},
  1529. },
  1530. },
  1531. Name: "latest",
  1532. }, {
  1533. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1534. Name: "id1",
  1535. },
  1536. },
  1537. isAggregate: false,
  1538. sendMeta: false,
  1539. }.Init(),
  1540. },
  1541. { // 17. do not optimize sliding window
  1542. sql: `SELECT * FROM src1 WHERE temp > 20 GROUP BY SLIDINGWINDOW(ss, 10) HAVING COUNT(*) > 2`,
  1543. p: ProjectPlan{
  1544. baseLogicalPlan: baseLogicalPlan{
  1545. children: []LogicalPlan{
  1546. HavingPlan{
  1547. baseLogicalPlan: baseLogicalPlan{
  1548. children: []LogicalPlan{
  1549. FilterPlan{
  1550. baseLogicalPlan: baseLogicalPlan{
  1551. children: []LogicalPlan{
  1552. WindowPlan{
  1553. baseLogicalPlan: baseLogicalPlan{
  1554. children: []LogicalPlan{
  1555. DataSourcePlan{
  1556. name: "src1",
  1557. isWildCard: true,
  1558. streamFields: map[string]*ast.JsonStreamField{
  1559. "id1": {
  1560. Type: "bigint",
  1561. },
  1562. "temp": {
  1563. Type: "bigint",
  1564. },
  1565. "name": {
  1566. Type: "string",
  1567. },
  1568. "myarray": {
  1569. Type: "array",
  1570. Items: &ast.JsonStreamField{
  1571. Type: "string",
  1572. },
  1573. },
  1574. },
  1575. streamStmt: streams["src1"],
  1576. metaFields: []string{},
  1577. }.Init(),
  1578. },
  1579. },
  1580. condition: nil,
  1581. wtype: ast.SLIDING_WINDOW,
  1582. length: 10000,
  1583. interval: 0,
  1584. limit: 0,
  1585. }.Init(),
  1586. },
  1587. },
  1588. condition: &ast.BinaryExpr{
  1589. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1590. OP: ast.GT,
  1591. RHS: &ast.IntegerLiteral{Val: 20},
  1592. },
  1593. }.Init(),
  1594. },
  1595. },
  1596. condition: &ast.BinaryExpr{
  1597. LHS: &ast.Call{Name: "count", FuncId: 0, Args: []ast.Expr{&ast.Wildcard{
  1598. Token: ast.ASTERISK,
  1599. }}, FuncType: ast.FuncTypeAgg},
  1600. OP: ast.GT,
  1601. RHS: &ast.IntegerLiteral{Val: 2},
  1602. },
  1603. }.Init(),
  1604. },
  1605. },
  1606. fields: []ast.Field{
  1607. {
  1608. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  1609. Name: "*",
  1610. AName: "",
  1611. },
  1612. },
  1613. isAggregate: false,
  1614. sendMeta: false,
  1615. }.Init(),
  1616. },
  1617. }
  1618. fmt.Printf("The test bucket size is %d.\n\n", len(tests))
  1619. for i, tt := range tests {
  1620. stmt, err := xsql.NewParser(strings.NewReader(tt.sql)).Parse()
  1621. if err != nil {
  1622. t.Errorf("%d. %q: error compile sql: %s\n", i, tt.sql, err)
  1623. continue
  1624. }
  1625. p, err := createLogicalPlan(stmt, &api.RuleOption{
  1626. IsEventTime: false,
  1627. LateTol: 0,
  1628. Concurrency: 0,
  1629. BufferLength: 0,
  1630. SendMetaToSink: false,
  1631. Qos: 0,
  1632. CheckpointInterval: 0,
  1633. SendError: true,
  1634. }, store)
  1635. if !reflect.DeepEqual(tt.err, testx.Errstring(err)) {
  1636. t.Errorf("%d. %q: error mismatch:\n exp=%s\n got=%s\n\n", i, tt.sql, tt.err, err)
  1637. } else if !reflect.DeepEqual(tt.p, p) {
  1638. t.Errorf("%d. %q\n\nstmt mismatch:\n\nexp=%#v\n\ngot=%#v\n\n", i, tt.sql, render.AsCode(tt.p), render.AsCode(p))
  1639. }
  1640. }
  1641. }
  1642. func Test_createLogicalPlanSchemaless(t *testing.T) {
  1643. store, err := store.GetKV("stream")
  1644. if err != nil {
  1645. t.Error(err)
  1646. return
  1647. }
  1648. streamSqls := map[string]string{
  1649. "src1": `CREATE STREAM src1 (
  1650. ) WITH (DATASOURCE="src1", FORMAT="json", KEY="ts");`,
  1651. "src2": `CREATE STREAM src2 (
  1652. ) WITH (DATASOURCE="src2", FORMAT="json", KEY="ts");`,
  1653. "tableInPlanner": `CREATE TABLE tableInPlanner (
  1654. id BIGINT,
  1655. name STRING,
  1656. value STRING,
  1657. hum BIGINT
  1658. ) WITH (TYPE="file");`,
  1659. }
  1660. types := map[string]ast.StreamType{
  1661. "src1": ast.TypeStream,
  1662. "src2": ast.TypeStream,
  1663. "tableInPlanner": ast.TypeTable,
  1664. }
  1665. for name, sql := range streamSqls {
  1666. s, err := json.Marshal(&xsql.StreamInfo{
  1667. StreamType: types[name],
  1668. Statement: sql,
  1669. })
  1670. if err != nil {
  1671. t.Error(err)
  1672. t.Fail()
  1673. }
  1674. err = store.Set(name, string(s))
  1675. if err != nil {
  1676. t.Error(err)
  1677. t.Fail()
  1678. }
  1679. }
  1680. streams := make(map[string]*ast.StreamStmt)
  1681. for n := range streamSqls {
  1682. streamStmt, err := xsql.GetDataSource(store, n)
  1683. if err != nil {
  1684. t.Errorf("fail to get stream %s, please check if stream is created", n)
  1685. return
  1686. }
  1687. streams[n] = streamStmt
  1688. }
  1689. // boolTrue = true
  1690. boolFalse := false
  1691. tests := []struct {
  1692. sql string
  1693. p LogicalPlan
  1694. err string
  1695. }{
  1696. { // 0
  1697. sql: `SELECT name FROM src1`,
  1698. p: ProjectPlan{
  1699. baseLogicalPlan: baseLogicalPlan{
  1700. children: []LogicalPlan{
  1701. DataSourcePlan{
  1702. baseLogicalPlan: baseLogicalPlan{},
  1703. name: "src1",
  1704. streamFields: map[string]*ast.JsonStreamField{
  1705. "name": nil,
  1706. },
  1707. streamStmt: streams["src1"],
  1708. isSchemaless: true,
  1709. metaFields: []string{},
  1710. }.Init(),
  1711. },
  1712. },
  1713. fields: []ast.Field{
  1714. {
  1715. Expr: &ast.FieldRef{Name: "name", StreamName: "src1"},
  1716. Name: "name",
  1717. AName: "",
  1718. },
  1719. },
  1720. isAggregate: false,
  1721. sendMeta: false,
  1722. }.Init(),
  1723. }, { // 1 optimize where to data source
  1724. sql: `SELECT temp FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  1725. p: ProjectPlan{
  1726. baseLogicalPlan: baseLogicalPlan{
  1727. children: []LogicalPlan{
  1728. WindowPlan{
  1729. baseLogicalPlan: baseLogicalPlan{
  1730. children: []LogicalPlan{
  1731. FilterPlan{
  1732. baseLogicalPlan: baseLogicalPlan{
  1733. children: []LogicalPlan{
  1734. DataSourcePlan{
  1735. name: "src1",
  1736. streamFields: map[string]*ast.JsonStreamField{
  1737. "name": nil,
  1738. "temp": nil,
  1739. },
  1740. streamStmt: streams["src1"],
  1741. metaFields: []string{},
  1742. isSchemaless: true,
  1743. }.Init(),
  1744. },
  1745. },
  1746. condition: &ast.BinaryExpr{
  1747. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  1748. OP: ast.EQ,
  1749. RHS: &ast.StringLiteral{Val: "v1"},
  1750. },
  1751. }.Init(),
  1752. },
  1753. },
  1754. condition: nil,
  1755. wtype: ast.TUMBLING_WINDOW,
  1756. length: 10000,
  1757. interval: 0,
  1758. limit: 0,
  1759. }.Init(),
  1760. },
  1761. },
  1762. fields: []ast.Field{
  1763. {
  1764. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1765. Name: "temp",
  1766. AName: "",
  1767. },
  1768. },
  1769. isAggregate: false,
  1770. sendMeta: false,
  1771. }.Init(),
  1772. }, { // 2 condition that cannot be optimized
  1773. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 WHERE src1.temp > 20 OR src2.hum > 60 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  1774. p: ProjectPlan{
  1775. baseLogicalPlan: baseLogicalPlan{
  1776. children: []LogicalPlan{
  1777. JoinPlan{
  1778. baseLogicalPlan: baseLogicalPlan{
  1779. children: []LogicalPlan{
  1780. WindowPlan{
  1781. baseLogicalPlan: baseLogicalPlan{
  1782. children: []LogicalPlan{
  1783. DataSourcePlan{
  1784. name: "src1",
  1785. streamFields: map[string]*ast.JsonStreamField{
  1786. "id1": nil,
  1787. "temp": nil,
  1788. },
  1789. streamStmt: streams["src1"],
  1790. metaFields: []string{},
  1791. isSchemaless: true,
  1792. }.Init(),
  1793. DataSourcePlan{
  1794. name: "src2",
  1795. streamFields: map[string]*ast.JsonStreamField{ // can't determine where is id1 belonged to
  1796. "hum": nil,
  1797. "id1": nil,
  1798. "id2": nil,
  1799. },
  1800. isSchemaless: true,
  1801. streamStmt: streams["src2"],
  1802. metaFields: []string{},
  1803. }.Init(),
  1804. },
  1805. },
  1806. condition: nil,
  1807. wtype: ast.TUMBLING_WINDOW,
  1808. length: 10000,
  1809. interval: 0,
  1810. limit: 0,
  1811. }.Init(),
  1812. },
  1813. },
  1814. from: &ast.Table{Name: "src1"},
  1815. joins: ast.Joins{ast.Join{
  1816. Name: "src2",
  1817. JoinType: ast.INNER_JOIN,
  1818. Expr: &ast.BinaryExpr{
  1819. OP: ast.AND,
  1820. LHS: &ast.BinaryExpr{
  1821. LHS: &ast.BinaryExpr{
  1822. OP: ast.GT,
  1823. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1824. RHS: &ast.IntegerLiteral{Val: 20},
  1825. },
  1826. OP: ast.OR,
  1827. RHS: &ast.BinaryExpr{
  1828. OP: ast.GT,
  1829. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  1830. RHS: &ast.IntegerLiteral{Val: 60},
  1831. },
  1832. },
  1833. RHS: &ast.BinaryExpr{
  1834. OP: ast.EQ,
  1835. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1836. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  1837. },
  1838. },
  1839. }},
  1840. }.Init(),
  1841. },
  1842. },
  1843. fields: []ast.Field{
  1844. {
  1845. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  1846. Name: "id1",
  1847. AName: "",
  1848. },
  1849. },
  1850. isAggregate: false,
  1851. sendMeta: false,
  1852. }.Init(),
  1853. }, { // 3 optimize window filter
  1854. sql: `SELECT id1 FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10) FILTER( WHERE temp > 2)`,
  1855. p: ProjectPlan{
  1856. baseLogicalPlan: baseLogicalPlan{
  1857. children: []LogicalPlan{
  1858. WindowPlan{
  1859. baseLogicalPlan: baseLogicalPlan{
  1860. children: []LogicalPlan{
  1861. FilterPlan{
  1862. baseLogicalPlan: baseLogicalPlan{
  1863. children: []LogicalPlan{
  1864. DataSourcePlan{
  1865. name: "src1",
  1866. streamFields: map[string]*ast.JsonStreamField{
  1867. "id1": nil,
  1868. "name": nil,
  1869. "temp": nil,
  1870. },
  1871. isSchemaless: true,
  1872. streamStmt: streams["src1"],
  1873. metaFields: []string{},
  1874. }.Init(),
  1875. },
  1876. },
  1877. condition: &ast.BinaryExpr{
  1878. OP: ast.AND,
  1879. LHS: &ast.BinaryExpr{
  1880. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  1881. OP: ast.EQ,
  1882. RHS: &ast.StringLiteral{Val: "v1"},
  1883. },
  1884. RHS: &ast.BinaryExpr{
  1885. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1886. OP: ast.GT,
  1887. RHS: &ast.IntegerLiteral{Val: 2},
  1888. },
  1889. },
  1890. }.Init(),
  1891. },
  1892. },
  1893. condition: nil,
  1894. wtype: ast.TUMBLING_WINDOW,
  1895. length: 10000,
  1896. interval: 0,
  1897. limit: 0,
  1898. }.Init(),
  1899. },
  1900. },
  1901. fields: []ast.Field{
  1902. {
  1903. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1904. Name: "id1",
  1905. AName: "",
  1906. },
  1907. },
  1908. isAggregate: false,
  1909. sendMeta: false,
  1910. }.Init(),
  1911. }, { // 4. do not optimize count window
  1912. sql: `SELECT * FROM src1 WHERE temp > 20 GROUP BY COUNTWINDOW(5,1) HAVING COUNT(*) > 2`,
  1913. p: ProjectPlan{
  1914. baseLogicalPlan: baseLogicalPlan{
  1915. children: []LogicalPlan{
  1916. HavingPlan{
  1917. baseLogicalPlan: baseLogicalPlan{
  1918. children: []LogicalPlan{
  1919. FilterPlan{
  1920. baseLogicalPlan: baseLogicalPlan{
  1921. children: []LogicalPlan{
  1922. WindowPlan{
  1923. baseLogicalPlan: baseLogicalPlan{
  1924. children: []LogicalPlan{
  1925. DataSourcePlan{
  1926. name: "src1",
  1927. isWildCard: true,
  1928. streamFields: map[string]*ast.JsonStreamField{},
  1929. streamStmt: streams["src1"],
  1930. metaFields: []string{},
  1931. isSchemaless: true,
  1932. }.Init(),
  1933. },
  1934. },
  1935. condition: nil,
  1936. wtype: ast.COUNT_WINDOW,
  1937. length: 5,
  1938. interval: 1,
  1939. limit: 0,
  1940. }.Init(),
  1941. },
  1942. },
  1943. condition: &ast.BinaryExpr{
  1944. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1945. OP: ast.GT,
  1946. RHS: &ast.IntegerLiteral{Val: 20},
  1947. },
  1948. }.Init(),
  1949. },
  1950. },
  1951. condition: &ast.BinaryExpr{
  1952. LHS: &ast.Call{Name: "count", FuncId: 0, Args: []ast.Expr{&ast.Wildcard{
  1953. Token: ast.ASTERISK,
  1954. }}, FuncType: ast.FuncTypeAgg},
  1955. OP: ast.GT,
  1956. RHS: &ast.IntegerLiteral{Val: 2},
  1957. },
  1958. }.Init(),
  1959. },
  1960. },
  1961. fields: []ast.Field{
  1962. {
  1963. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  1964. Name: "*",
  1965. AName: "",
  1966. },
  1967. },
  1968. isAggregate: false,
  1969. sendMeta: false,
  1970. }.Init(),
  1971. }, { // 5. optimize join on
  1972. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  1973. p: ProjectPlan{
  1974. baseLogicalPlan: baseLogicalPlan{
  1975. children: []LogicalPlan{
  1976. JoinPlan{
  1977. baseLogicalPlan: baseLogicalPlan{
  1978. children: []LogicalPlan{
  1979. WindowPlan{
  1980. baseLogicalPlan: baseLogicalPlan{
  1981. children: []LogicalPlan{
  1982. FilterPlan{
  1983. baseLogicalPlan: baseLogicalPlan{
  1984. children: []LogicalPlan{
  1985. DataSourcePlan{
  1986. name: "src1",
  1987. streamFields: map[string]*ast.JsonStreamField{
  1988. "id1": nil,
  1989. "temp": nil,
  1990. },
  1991. isSchemaless: true,
  1992. streamStmt: streams["src1"],
  1993. metaFields: []string{},
  1994. }.Init(),
  1995. },
  1996. },
  1997. condition: &ast.BinaryExpr{
  1998. RHS: &ast.BinaryExpr{
  1999. OP: ast.GT,
  2000. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2001. RHS: &ast.IntegerLiteral{Val: 20},
  2002. },
  2003. OP: ast.AND,
  2004. LHS: &ast.BinaryExpr{
  2005. OP: ast.GT,
  2006. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2007. RHS: &ast.IntegerLiteral{Val: 111},
  2008. },
  2009. },
  2010. }.Init(),
  2011. FilterPlan{
  2012. baseLogicalPlan: baseLogicalPlan{
  2013. children: []LogicalPlan{
  2014. DataSourcePlan{
  2015. name: "src2",
  2016. streamFields: map[string]*ast.JsonStreamField{
  2017. "hum": nil,
  2018. "id1": nil,
  2019. "id2": nil,
  2020. },
  2021. isSchemaless: true,
  2022. streamStmt: streams["src2"],
  2023. metaFields: []string{},
  2024. }.Init(),
  2025. },
  2026. },
  2027. condition: &ast.BinaryExpr{
  2028. OP: ast.LT,
  2029. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  2030. RHS: &ast.IntegerLiteral{Val: 60},
  2031. },
  2032. }.Init(),
  2033. },
  2034. },
  2035. condition: nil,
  2036. wtype: ast.TUMBLING_WINDOW,
  2037. length: 10000,
  2038. interval: 0,
  2039. limit: 0,
  2040. }.Init(),
  2041. },
  2042. },
  2043. from: &ast.Table{
  2044. Name: "src1",
  2045. },
  2046. joins: []ast.Join{
  2047. {
  2048. Name: "src2",
  2049. Alias: "",
  2050. JoinType: ast.INNER_JOIN,
  2051. Expr: &ast.BinaryExpr{
  2052. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2053. OP: ast.EQ,
  2054. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  2055. },
  2056. },
  2057. },
  2058. }.Init(),
  2059. },
  2060. },
  2061. fields: []ast.Field{
  2062. {
  2063. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2064. Name: "id1",
  2065. AName: "",
  2066. },
  2067. },
  2068. isAggregate: false,
  2069. sendMeta: false,
  2070. }.Init(),
  2071. }, { // 6. optimize outter join on
  2072. sql: `SELECT id1 FROM src1 FULL JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2073. p: ProjectPlan{
  2074. baseLogicalPlan: baseLogicalPlan{
  2075. children: []LogicalPlan{
  2076. JoinPlan{
  2077. baseLogicalPlan: baseLogicalPlan{
  2078. children: []LogicalPlan{
  2079. WindowPlan{
  2080. baseLogicalPlan: baseLogicalPlan{
  2081. children: []LogicalPlan{
  2082. FilterPlan{
  2083. baseLogicalPlan: baseLogicalPlan{
  2084. children: []LogicalPlan{
  2085. DataSourcePlan{
  2086. name: "src1",
  2087. streamFields: map[string]*ast.JsonStreamField{
  2088. "id1": nil,
  2089. "temp": nil,
  2090. },
  2091. isSchemaless: true,
  2092. streamStmt: streams["src1"],
  2093. metaFields: []string{},
  2094. }.Init(),
  2095. },
  2096. },
  2097. condition: &ast.BinaryExpr{
  2098. OP: ast.GT,
  2099. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2100. RHS: &ast.IntegerLiteral{Val: 111},
  2101. },
  2102. }.Init(),
  2103. DataSourcePlan{
  2104. name: "src2",
  2105. streamFields: map[string]*ast.JsonStreamField{
  2106. "hum": nil,
  2107. "id1": nil,
  2108. "id2": nil,
  2109. },
  2110. isSchemaless: true,
  2111. streamStmt: streams["src2"],
  2112. metaFields: []string{},
  2113. }.Init(),
  2114. },
  2115. },
  2116. condition: nil,
  2117. wtype: ast.TUMBLING_WINDOW,
  2118. length: 10000,
  2119. interval: 0,
  2120. limit: 0,
  2121. }.Init(),
  2122. },
  2123. },
  2124. from: &ast.Table{
  2125. Name: "src1",
  2126. },
  2127. joins: []ast.Join{
  2128. {
  2129. Name: "src2",
  2130. Alias: "",
  2131. JoinType: ast.FULL_JOIN,
  2132. Expr: &ast.BinaryExpr{
  2133. OP: ast.AND,
  2134. LHS: &ast.BinaryExpr{
  2135. OP: ast.AND,
  2136. LHS: &ast.BinaryExpr{
  2137. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2138. OP: ast.EQ,
  2139. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  2140. },
  2141. RHS: &ast.BinaryExpr{
  2142. OP: ast.GT,
  2143. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2144. RHS: &ast.IntegerLiteral{Val: 20},
  2145. },
  2146. },
  2147. RHS: &ast.BinaryExpr{
  2148. OP: ast.LT,
  2149. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  2150. RHS: &ast.IntegerLiteral{Val: 60},
  2151. },
  2152. },
  2153. },
  2154. },
  2155. }.Init(),
  2156. },
  2157. },
  2158. fields: []ast.Field{
  2159. {
  2160. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2161. Name: "id1",
  2162. AName: "",
  2163. },
  2164. },
  2165. isAggregate: false,
  2166. sendMeta: false,
  2167. }.Init(),
  2168. }, { // 7 window error for table
  2169. sql: `SELECT value FROM tableInPlanner WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2170. p: nil,
  2171. err: "cannot run window for TABLE sources",
  2172. }, { // 8 join table without window
  2173. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and hum < 60 WHERE src1.id1 > 111`,
  2174. p: ProjectPlan{
  2175. baseLogicalPlan: baseLogicalPlan{
  2176. children: []LogicalPlan{
  2177. JoinPlan{
  2178. baseLogicalPlan: baseLogicalPlan{
  2179. children: []LogicalPlan{
  2180. JoinAlignPlan{
  2181. baseLogicalPlan: baseLogicalPlan{
  2182. children: []LogicalPlan{
  2183. FilterPlan{
  2184. baseLogicalPlan: baseLogicalPlan{
  2185. children: []LogicalPlan{
  2186. DataSourcePlan{
  2187. name: "src1",
  2188. streamFields: map[string]*ast.JsonStreamField{
  2189. "hum": nil,
  2190. "id1": nil,
  2191. "temp": nil,
  2192. },
  2193. isSchemaless: true,
  2194. streamStmt: streams["src1"],
  2195. metaFields: []string{},
  2196. }.Init(),
  2197. },
  2198. },
  2199. condition: &ast.BinaryExpr{
  2200. RHS: &ast.BinaryExpr{
  2201. OP: ast.GT,
  2202. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2203. RHS: &ast.IntegerLiteral{Val: 20},
  2204. },
  2205. OP: ast.AND,
  2206. LHS: &ast.BinaryExpr{
  2207. OP: ast.GT,
  2208. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2209. RHS: &ast.IntegerLiteral{Val: 111},
  2210. },
  2211. },
  2212. }.Init(),
  2213. DataSourcePlan{
  2214. name: "tableInPlanner",
  2215. streamFields: map[string]*ast.JsonStreamField{
  2216. "hum": {
  2217. Type: "bigint",
  2218. },
  2219. "id": {
  2220. Type: "bigint",
  2221. },
  2222. },
  2223. streamStmt: streams["tableInPlanner"],
  2224. metaFields: []string{},
  2225. }.Init(),
  2226. },
  2227. },
  2228. Emitters: []string{"tableInPlanner"},
  2229. }.Init(),
  2230. },
  2231. },
  2232. from: &ast.Table{
  2233. Name: "src1",
  2234. },
  2235. joins: []ast.Join{
  2236. {
  2237. Name: "tableInPlanner",
  2238. Alias: "",
  2239. JoinType: ast.INNER_JOIN,
  2240. Expr: &ast.BinaryExpr{
  2241. OP: ast.AND,
  2242. LHS: &ast.BinaryExpr{
  2243. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2244. OP: ast.EQ,
  2245. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  2246. },
  2247. RHS: &ast.BinaryExpr{
  2248. OP: ast.LT,
  2249. LHS: &ast.FieldRef{Name: "hum", StreamName: ast.DefaultStream},
  2250. RHS: &ast.IntegerLiteral{Val: 60},
  2251. },
  2252. },
  2253. },
  2254. },
  2255. }.Init(),
  2256. },
  2257. },
  2258. fields: []ast.Field{
  2259. {
  2260. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2261. Name: "id1",
  2262. AName: "",
  2263. },
  2264. },
  2265. isAggregate: false,
  2266. sendMeta: false,
  2267. }.Init(),
  2268. }, { // 9 join table with window
  2269. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and tableInPlanner.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2270. p: ProjectPlan{
  2271. baseLogicalPlan: baseLogicalPlan{
  2272. children: []LogicalPlan{
  2273. JoinPlan{
  2274. baseLogicalPlan: baseLogicalPlan{
  2275. children: []LogicalPlan{
  2276. JoinAlignPlan{
  2277. baseLogicalPlan: baseLogicalPlan{
  2278. children: []LogicalPlan{
  2279. WindowPlan{
  2280. baseLogicalPlan: baseLogicalPlan{
  2281. children: []LogicalPlan{
  2282. DataSourcePlan{
  2283. name: "src1",
  2284. streamFields: map[string]*ast.JsonStreamField{
  2285. "id1": nil,
  2286. "temp": nil,
  2287. },
  2288. isSchemaless: true,
  2289. streamStmt: streams["src1"],
  2290. metaFields: []string{},
  2291. }.Init(),
  2292. },
  2293. },
  2294. condition: nil,
  2295. wtype: ast.TUMBLING_WINDOW,
  2296. length: 10000,
  2297. interval: 0,
  2298. limit: 0,
  2299. }.Init(),
  2300. DataSourcePlan{
  2301. name: "tableInPlanner",
  2302. streamFields: map[string]*ast.JsonStreamField{
  2303. "hum": {
  2304. Type: "bigint",
  2305. },
  2306. "id": {
  2307. Type: "bigint",
  2308. },
  2309. },
  2310. streamStmt: streams["tableInPlanner"],
  2311. metaFields: []string{},
  2312. }.Init(),
  2313. },
  2314. },
  2315. Emitters: []string{"tableInPlanner"},
  2316. }.Init(),
  2317. },
  2318. },
  2319. from: &ast.Table{
  2320. Name: "src1",
  2321. },
  2322. joins: []ast.Join{
  2323. {
  2324. Name: "tableInPlanner",
  2325. Alias: "",
  2326. JoinType: ast.INNER_JOIN,
  2327. Expr: &ast.BinaryExpr{
  2328. OP: ast.AND,
  2329. LHS: &ast.BinaryExpr{
  2330. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2331. OP: ast.EQ,
  2332. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  2333. },
  2334. RHS: &ast.BinaryExpr{
  2335. RHS: &ast.BinaryExpr{
  2336. OP: ast.AND,
  2337. LHS: &ast.BinaryExpr{
  2338. OP: ast.GT,
  2339. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2340. RHS: &ast.IntegerLiteral{Val: 20},
  2341. },
  2342. RHS: &ast.BinaryExpr{
  2343. OP: ast.LT,
  2344. LHS: &ast.FieldRef{Name: "hum", StreamName: "tableInPlanner"},
  2345. RHS: &ast.IntegerLiteral{Val: 60},
  2346. },
  2347. },
  2348. OP: ast.AND,
  2349. LHS: &ast.BinaryExpr{
  2350. OP: ast.GT,
  2351. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2352. RHS: &ast.IntegerLiteral{Val: 111},
  2353. },
  2354. },
  2355. },
  2356. },
  2357. },
  2358. }.Init(),
  2359. },
  2360. },
  2361. fields: []ast.Field{
  2362. {
  2363. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2364. Name: "id1",
  2365. AName: "",
  2366. },
  2367. },
  2368. isAggregate: false,
  2369. sendMeta: false,
  2370. }.Init(),
  2371. }, { // 10 meta
  2372. sql: `SELECT temp, meta(id) AS eid,meta(Humidity->Device) AS hdevice FROM src1 WHERE meta(device)="demo2"`,
  2373. p: ProjectPlan{
  2374. baseLogicalPlan: baseLogicalPlan{
  2375. children: []LogicalPlan{
  2376. FilterPlan{
  2377. baseLogicalPlan: baseLogicalPlan{
  2378. children: []LogicalPlan{
  2379. DataSourcePlan{
  2380. name: "src1",
  2381. streamFields: map[string]*ast.JsonStreamField{
  2382. "temp": nil,
  2383. },
  2384. isSchemaless: true,
  2385. streamStmt: streams["src1"],
  2386. metaFields: []string{"Humidity", "device", "id"},
  2387. }.Init(),
  2388. },
  2389. },
  2390. condition: &ast.BinaryExpr{
  2391. LHS: &ast.Call{
  2392. Name: "meta",
  2393. FuncId: 2,
  2394. Args: []ast.Expr{&ast.MetaRef{
  2395. Name: "device",
  2396. StreamName: ast.DefaultStream,
  2397. }},
  2398. },
  2399. OP: ast.EQ,
  2400. RHS: &ast.StringLiteral{
  2401. Val: "demo2",
  2402. },
  2403. },
  2404. }.Init(),
  2405. },
  2406. },
  2407. fields: []ast.Field{
  2408. {
  2409. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2410. Name: "temp",
  2411. AName: "",
  2412. }, {
  2413. Expr: &ast.FieldRef{Name: "eid", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2414. &ast.Call{Name: "meta", FuncId: 0, Args: []ast.Expr{&ast.MetaRef{
  2415. Name: "id",
  2416. StreamName: ast.DefaultStream,
  2417. }}},
  2418. []ast.StreamName{},
  2419. nil,
  2420. )},
  2421. Name: "meta",
  2422. AName: "eid",
  2423. }, {
  2424. Expr: &ast.FieldRef{Name: "hdevice", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2425. &ast.Call{Name: "meta", FuncId: 1, Args: []ast.Expr{
  2426. &ast.BinaryExpr{
  2427. OP: ast.ARROW,
  2428. LHS: &ast.MetaRef{Name: "Humidity", StreamName: ast.DefaultStream},
  2429. RHS: &ast.JsonFieldRef{Name: "Device"},
  2430. },
  2431. }},
  2432. []ast.StreamName{},
  2433. nil,
  2434. )},
  2435. Name: "meta",
  2436. AName: "hdevice",
  2437. },
  2438. },
  2439. isAggregate: false,
  2440. sendMeta: false,
  2441. }.Init(),
  2442. }, { // 11 join with same name field and aliased
  2443. sql: `SELECT src2.hum AS hum1, tableInPlanner.hum AS hum2 FROM src2 INNER JOIN tableInPlanner on id2 = id WHERE hum1 > hum2`,
  2444. p: ProjectPlan{
  2445. baseLogicalPlan: baseLogicalPlan{
  2446. children: []LogicalPlan{
  2447. JoinPlan{
  2448. baseLogicalPlan: baseLogicalPlan{
  2449. children: []LogicalPlan{
  2450. JoinAlignPlan{
  2451. baseLogicalPlan: baseLogicalPlan{
  2452. children: []LogicalPlan{
  2453. DataSourcePlan{
  2454. name: "src2",
  2455. streamFields: map[string]*ast.JsonStreamField{
  2456. "hum": nil,
  2457. "id": nil,
  2458. "id2": nil,
  2459. },
  2460. isSchemaless: true,
  2461. streamStmt: streams["src2"],
  2462. metaFields: []string{},
  2463. }.Init(),
  2464. DataSourcePlan{
  2465. name: "tableInPlanner",
  2466. streamFields: map[string]*ast.JsonStreamField{
  2467. "hum": {
  2468. Type: "bigint",
  2469. },
  2470. "id": {
  2471. Type: "bigint",
  2472. },
  2473. },
  2474. streamStmt: streams["tableInPlanner"],
  2475. metaFields: []string{},
  2476. }.Init(),
  2477. },
  2478. },
  2479. Emitters: []string{"tableInPlanner"},
  2480. }.Init(),
  2481. },
  2482. },
  2483. from: &ast.Table{
  2484. Name: "src2",
  2485. },
  2486. joins: []ast.Join{
  2487. {
  2488. Name: "tableInPlanner",
  2489. Alias: "",
  2490. JoinType: ast.INNER_JOIN,
  2491. Expr: &ast.BinaryExpr{
  2492. RHS: &ast.BinaryExpr{
  2493. OP: ast.EQ,
  2494. LHS: &ast.FieldRef{Name: "id2", StreamName: ast.DefaultStream},
  2495. RHS: &ast.FieldRef{Name: "id", StreamName: ast.DefaultStream},
  2496. },
  2497. OP: ast.AND,
  2498. LHS: &ast.BinaryExpr{
  2499. OP: ast.GT,
  2500. LHS: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2501. &ast.FieldRef{
  2502. Name: "hum",
  2503. StreamName: "src2",
  2504. },
  2505. []ast.StreamName{"src2"},
  2506. &boolFalse,
  2507. )},
  2508. RHS: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2509. &ast.FieldRef{
  2510. Name: "hum",
  2511. StreamName: "tableInPlanner",
  2512. },
  2513. []ast.StreamName{"tableInPlanner"},
  2514. &boolFalse,
  2515. )},
  2516. },
  2517. },
  2518. },
  2519. },
  2520. }.Init(),
  2521. },
  2522. },
  2523. fields: []ast.Field{
  2524. {
  2525. Expr: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2526. &ast.FieldRef{
  2527. Name: "hum",
  2528. StreamName: "src2",
  2529. },
  2530. []ast.StreamName{"src2"},
  2531. &boolFalse,
  2532. )},
  2533. Name: "hum",
  2534. AName: "hum1",
  2535. }, {
  2536. Expr: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2537. &ast.FieldRef{
  2538. Name: "hum",
  2539. StreamName: "tableInPlanner",
  2540. },
  2541. []ast.StreamName{"tableInPlanner"},
  2542. &boolFalse,
  2543. )},
  2544. Name: "hum",
  2545. AName: "hum2",
  2546. },
  2547. },
  2548. isAggregate: false,
  2549. sendMeta: false,
  2550. }.Init(),
  2551. }, { // 12
  2552. sql: `SELECT name->first, name->last FROM src1`,
  2553. p: ProjectPlan{
  2554. baseLogicalPlan: baseLogicalPlan{
  2555. children: []LogicalPlan{
  2556. DataSourcePlan{
  2557. baseLogicalPlan: baseLogicalPlan{},
  2558. name: "src1",
  2559. streamFields: map[string]*ast.JsonStreamField{
  2560. "name": nil,
  2561. },
  2562. isSchemaless: true,
  2563. streamStmt: streams["src1"],
  2564. metaFields: []string{},
  2565. }.Init(),
  2566. },
  2567. },
  2568. fields: []ast.Field{
  2569. {
  2570. Expr: &ast.BinaryExpr{
  2571. OP: ast.ARROW,
  2572. LHS: &ast.FieldRef{StreamName: "src1", Name: "name"},
  2573. RHS: &ast.JsonFieldRef{Name: "first"},
  2574. },
  2575. Name: "kuiper_field_0",
  2576. AName: "",
  2577. }, {
  2578. Expr: &ast.BinaryExpr{
  2579. OP: ast.ARROW,
  2580. LHS: &ast.FieldRef{StreamName: "src1", Name: "name"},
  2581. RHS: &ast.JsonFieldRef{Name: "last"},
  2582. },
  2583. Name: "kuiper_field_1",
  2584. AName: "",
  2585. },
  2586. },
  2587. isAggregate: false,
  2588. sendMeta: false,
  2589. }.Init(),
  2590. },
  2591. }
  2592. fmt.Printf("The test bucket size is %d.\n\n", len(tests))
  2593. for i, tt := range tests {
  2594. stmt, err := xsql.NewParser(strings.NewReader(tt.sql)).Parse()
  2595. if err != nil {
  2596. t.Errorf("%d. %q: error compile sql: %s\n", i, tt.sql, err)
  2597. continue
  2598. }
  2599. p, err := createLogicalPlan(stmt, &api.RuleOption{
  2600. IsEventTime: false,
  2601. LateTol: 0,
  2602. Concurrency: 0,
  2603. BufferLength: 0,
  2604. SendMetaToSink: false,
  2605. Qos: 0,
  2606. CheckpointInterval: 0,
  2607. SendError: true,
  2608. }, store)
  2609. if !reflect.DeepEqual(tt.err, testx.Errstring(err)) {
  2610. t.Errorf("%d. %q: error mismatch:\n exp=%s\n got=%s\n\n", i, tt.sql, tt.err, err)
  2611. } else if !reflect.DeepEqual(tt.p, p) {
  2612. t.Errorf("%d. %q\n\nstmt mismatch:\n\nexp=%#v\n\ngot=%#v\n\n", i, tt.sql, render.AsCode(tt.p), render.AsCode(p))
  2613. }
  2614. }
  2615. }
  2616. func Test_createLogicalPlan4Lookup(t *testing.T) {
  2617. store, err := store.GetKV("stream")
  2618. if err != nil {
  2619. t.Error(err)
  2620. return
  2621. }
  2622. streamSqls := map[string]string{
  2623. "src1": `CREATE STREAM src1 () WITH (DATASOURCE="src1", FORMAT="json", KEY="ts");`,
  2624. "table1": `CREATE TABLE table1 () WITH (DATASOURCE="table1",TYPE="sql", KIND="lookup");`,
  2625. "table2": `CREATE TABLE table2 () WITH (DATASOURCE="table2",TYPE="sql", KIND="lookup");`,
  2626. }
  2627. types := map[string]ast.StreamType{
  2628. "src1": ast.TypeStream,
  2629. "table1": ast.TypeTable,
  2630. "table2": ast.TypeTable,
  2631. }
  2632. for name, sql := range streamSqls {
  2633. s, err := json.Marshal(&xsql.StreamInfo{
  2634. StreamType: types[name],
  2635. Statement: sql,
  2636. })
  2637. if err != nil {
  2638. t.Error(err)
  2639. t.Fail()
  2640. }
  2641. err = store.Set(name, string(s))
  2642. if err != nil {
  2643. t.Error(err)
  2644. t.Fail()
  2645. }
  2646. }
  2647. streams := make(map[string]*ast.StreamStmt)
  2648. for n := range streamSqls {
  2649. streamStmt, err := xsql.GetDataSource(store, n)
  2650. if err != nil {
  2651. t.Errorf("fail to get stream %s, please check if stream is created", n)
  2652. return
  2653. }
  2654. streams[n] = streamStmt
  2655. }
  2656. tests := []struct {
  2657. sql string
  2658. p LogicalPlan
  2659. err string
  2660. }{
  2661. { // 0
  2662. sql: `SELECT src1.a, table1.b FROM src1 INNER JOIN table1 ON src1.id = table1.id`,
  2663. p: ProjectPlan{
  2664. baseLogicalPlan: baseLogicalPlan{
  2665. children: []LogicalPlan{
  2666. LookupPlan{
  2667. baseLogicalPlan: baseLogicalPlan{
  2668. children: []LogicalPlan{
  2669. DataSourcePlan{
  2670. baseLogicalPlan: baseLogicalPlan{},
  2671. name: "src1",
  2672. streamFields: map[string]*ast.JsonStreamField{
  2673. "a": nil,
  2674. },
  2675. isSchemaless: true,
  2676. streamStmt: streams["src1"],
  2677. metaFields: []string{},
  2678. }.Init(),
  2679. },
  2680. },
  2681. joinExpr: ast.Join{
  2682. Name: "table1",
  2683. Alias: "",
  2684. JoinType: ast.INNER_JOIN,
  2685. Expr: &ast.BinaryExpr{
  2686. OP: ast.EQ,
  2687. LHS: &ast.FieldRef{
  2688. StreamName: "src1",
  2689. Name: "id",
  2690. },
  2691. RHS: &ast.FieldRef{
  2692. StreamName: "table1",
  2693. Name: "id",
  2694. },
  2695. },
  2696. },
  2697. keys: []string{"id"},
  2698. fields: []string{"b"},
  2699. valvars: []ast.Expr{
  2700. &ast.FieldRef{
  2701. StreamName: "src1",
  2702. Name: "id",
  2703. },
  2704. },
  2705. options: &ast.Options{
  2706. DATASOURCE: "table1",
  2707. TYPE: "sql",
  2708. KIND: "lookup",
  2709. },
  2710. conditions: nil,
  2711. }.Init(),
  2712. },
  2713. },
  2714. fields: []ast.Field{
  2715. {
  2716. Expr: &ast.FieldRef{
  2717. StreamName: "src1",
  2718. Name: "a",
  2719. },
  2720. Name: "a",
  2721. AName: "",
  2722. },
  2723. {
  2724. Expr: &ast.FieldRef{
  2725. StreamName: "table1",
  2726. Name: "b",
  2727. },
  2728. Name: "b",
  2729. AName: "",
  2730. },
  2731. },
  2732. isAggregate: false,
  2733. sendMeta: false,
  2734. }.Init(),
  2735. },
  2736. { // 1
  2737. sql: `SELECT src1.a, table1.* FROM src1 INNER JOIN table1 ON table1.b > 20 AND src1.c < 40 AND src1.id = table1.id`,
  2738. p: ProjectPlan{
  2739. baseLogicalPlan: baseLogicalPlan{
  2740. children: []LogicalPlan{
  2741. FilterPlan{
  2742. baseLogicalPlan: baseLogicalPlan{
  2743. children: []LogicalPlan{
  2744. LookupPlan{
  2745. baseLogicalPlan: baseLogicalPlan{
  2746. children: []LogicalPlan{
  2747. FilterPlan{
  2748. baseLogicalPlan: baseLogicalPlan{
  2749. children: []LogicalPlan{
  2750. DataSourcePlan{
  2751. baseLogicalPlan: baseLogicalPlan{},
  2752. name: "src1",
  2753. streamFields: map[string]*ast.JsonStreamField{
  2754. "a": nil,
  2755. },
  2756. isSchemaless: true,
  2757. streamStmt: streams["src1"],
  2758. metaFields: []string{},
  2759. }.Init(),
  2760. },
  2761. },
  2762. condition: &ast.BinaryExpr{
  2763. OP: ast.LT,
  2764. LHS: &ast.FieldRef{
  2765. StreamName: "src1",
  2766. Name: "c",
  2767. },
  2768. RHS: &ast.IntegerLiteral{Val: 40},
  2769. },
  2770. }.Init(),
  2771. },
  2772. },
  2773. joinExpr: ast.Join{
  2774. Name: "table1",
  2775. Alias: "",
  2776. JoinType: ast.INNER_JOIN,
  2777. Expr: &ast.BinaryExpr{
  2778. OP: ast.AND,
  2779. RHS: &ast.BinaryExpr{
  2780. OP: ast.EQ,
  2781. LHS: &ast.FieldRef{
  2782. StreamName: "src1",
  2783. Name: "id",
  2784. },
  2785. RHS: &ast.FieldRef{
  2786. StreamName: "table1",
  2787. Name: "id",
  2788. },
  2789. },
  2790. LHS: &ast.BinaryExpr{
  2791. OP: ast.AND,
  2792. LHS: &ast.BinaryExpr{
  2793. OP: ast.GT,
  2794. LHS: &ast.FieldRef{
  2795. StreamName: "table1",
  2796. Name: "b",
  2797. },
  2798. RHS: &ast.IntegerLiteral{Val: 20},
  2799. },
  2800. RHS: &ast.BinaryExpr{
  2801. OP: ast.LT,
  2802. LHS: &ast.FieldRef{
  2803. StreamName: "src1",
  2804. Name: "c",
  2805. },
  2806. RHS: &ast.IntegerLiteral{Val: 40},
  2807. },
  2808. },
  2809. },
  2810. },
  2811. keys: []string{"id"},
  2812. valvars: []ast.Expr{
  2813. &ast.FieldRef{
  2814. StreamName: "src1",
  2815. Name: "id",
  2816. },
  2817. },
  2818. options: &ast.Options{
  2819. DATASOURCE: "table1",
  2820. TYPE: "sql",
  2821. KIND: "lookup",
  2822. },
  2823. conditions: &ast.BinaryExpr{
  2824. OP: ast.AND,
  2825. LHS: &ast.BinaryExpr{
  2826. OP: ast.GT,
  2827. LHS: &ast.FieldRef{
  2828. StreamName: "table1",
  2829. Name: "b",
  2830. },
  2831. RHS: &ast.IntegerLiteral{Val: 20},
  2832. },
  2833. RHS: &ast.BinaryExpr{
  2834. OP: ast.LT,
  2835. LHS: &ast.FieldRef{
  2836. StreamName: "src1",
  2837. Name: "c",
  2838. },
  2839. RHS: &ast.IntegerLiteral{Val: 40},
  2840. },
  2841. },
  2842. }.Init(),
  2843. },
  2844. },
  2845. condition: &ast.BinaryExpr{
  2846. OP: ast.GT,
  2847. LHS: &ast.FieldRef{
  2848. StreamName: "table1",
  2849. Name: "b",
  2850. },
  2851. RHS: &ast.IntegerLiteral{Val: 20},
  2852. },
  2853. }.Init(),
  2854. },
  2855. },
  2856. fields: []ast.Field{
  2857. {
  2858. Expr: &ast.FieldRef{
  2859. StreamName: "src1",
  2860. Name: "a",
  2861. },
  2862. Name: "a",
  2863. AName: "",
  2864. },
  2865. {
  2866. Expr: &ast.FieldRef{
  2867. StreamName: "table1",
  2868. Name: "*",
  2869. },
  2870. Name: "*",
  2871. AName: "",
  2872. },
  2873. },
  2874. isAggregate: false,
  2875. sendMeta: false,
  2876. }.Init(),
  2877. },
  2878. { // 2
  2879. sql: `SELECT src1.a, table1.b, table2.c FROM src1 INNER JOIN table1 ON src1.id = table1.id INNER JOIN table2 on table1.id = table2.id`,
  2880. p: ProjectPlan{
  2881. baseLogicalPlan: baseLogicalPlan{
  2882. children: []LogicalPlan{
  2883. LookupPlan{
  2884. baseLogicalPlan: baseLogicalPlan{
  2885. children: []LogicalPlan{
  2886. LookupPlan{
  2887. baseLogicalPlan: baseLogicalPlan{
  2888. children: []LogicalPlan{
  2889. DataSourcePlan{
  2890. baseLogicalPlan: baseLogicalPlan{},
  2891. name: "src1",
  2892. streamFields: map[string]*ast.JsonStreamField{
  2893. "a": nil,
  2894. },
  2895. isSchemaless: true,
  2896. streamStmt: streams["src1"],
  2897. metaFields: []string{},
  2898. }.Init(),
  2899. },
  2900. },
  2901. joinExpr: ast.Join{
  2902. Name: "table1",
  2903. Alias: "",
  2904. JoinType: ast.INNER_JOIN,
  2905. Expr: &ast.BinaryExpr{
  2906. OP: ast.EQ,
  2907. LHS: &ast.FieldRef{
  2908. StreamName: "src1",
  2909. Name: "id",
  2910. },
  2911. RHS: &ast.FieldRef{
  2912. StreamName: "table1",
  2913. Name: "id",
  2914. },
  2915. },
  2916. },
  2917. keys: []string{"id"},
  2918. fields: []string{"b"},
  2919. valvars: []ast.Expr{
  2920. &ast.FieldRef{
  2921. StreamName: "src1",
  2922. Name: "id",
  2923. },
  2924. },
  2925. options: &ast.Options{
  2926. DATASOURCE: "table1",
  2927. TYPE: "sql",
  2928. KIND: "lookup",
  2929. },
  2930. conditions: nil,
  2931. }.Init(),
  2932. },
  2933. },
  2934. joinExpr: ast.Join{
  2935. Name: "table2",
  2936. Alias: "",
  2937. JoinType: ast.INNER_JOIN,
  2938. Expr: &ast.BinaryExpr{
  2939. OP: ast.EQ,
  2940. LHS: &ast.FieldRef{
  2941. StreamName: "table1",
  2942. Name: "id",
  2943. },
  2944. RHS: &ast.FieldRef{
  2945. StreamName: "table2",
  2946. Name: "id",
  2947. },
  2948. },
  2949. },
  2950. keys: []string{"id"},
  2951. fields: []string{"c"},
  2952. valvars: []ast.Expr{
  2953. &ast.FieldRef{
  2954. StreamName: "table1",
  2955. Name: "id",
  2956. },
  2957. },
  2958. options: &ast.Options{
  2959. DATASOURCE: "table2",
  2960. TYPE: "sql",
  2961. KIND: "lookup",
  2962. },
  2963. }.Init(),
  2964. },
  2965. },
  2966. fields: []ast.Field{
  2967. {
  2968. Expr: &ast.FieldRef{
  2969. StreamName: "src1",
  2970. Name: "a",
  2971. },
  2972. Name: "a",
  2973. AName: "",
  2974. },
  2975. {
  2976. Expr: &ast.FieldRef{
  2977. StreamName: "table1",
  2978. Name: "b",
  2979. },
  2980. Name: "b",
  2981. AName: "",
  2982. },
  2983. {
  2984. Expr: &ast.FieldRef{
  2985. StreamName: "table2",
  2986. Name: "c",
  2987. },
  2988. Name: "c",
  2989. AName: "",
  2990. },
  2991. },
  2992. isAggregate: false,
  2993. sendMeta: false,
  2994. }.Init(),
  2995. },
  2996. { // 3
  2997. sql: `SELECT * FROM src1 INNER JOIN table1 ON src1.id = table1.id GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2998. p: ProjectPlan{
  2999. baseLogicalPlan: baseLogicalPlan{
  3000. children: []LogicalPlan{
  3001. LookupPlan{
  3002. baseLogicalPlan: baseLogicalPlan{
  3003. children: []LogicalPlan{
  3004. WindowPlan{
  3005. baseLogicalPlan: baseLogicalPlan{
  3006. children: []LogicalPlan{
  3007. DataSourcePlan{
  3008. baseLogicalPlan: baseLogicalPlan{},
  3009. name: "src1",
  3010. streamStmt: streams["src1"],
  3011. streamFields: map[string]*ast.JsonStreamField{},
  3012. metaFields: []string{},
  3013. isWildCard: true,
  3014. isSchemaless: true,
  3015. }.Init(),
  3016. },
  3017. },
  3018. condition: nil,
  3019. wtype: ast.TUMBLING_WINDOW,
  3020. length: 10000,
  3021. interval: 0,
  3022. limit: 0,
  3023. }.Init(),
  3024. },
  3025. },
  3026. joinExpr: ast.Join{
  3027. Name: "table1",
  3028. Alias: "",
  3029. JoinType: ast.INNER_JOIN,
  3030. Expr: &ast.BinaryExpr{
  3031. OP: ast.EQ,
  3032. LHS: &ast.FieldRef{
  3033. StreamName: "src1",
  3034. Name: "id",
  3035. },
  3036. RHS: &ast.FieldRef{
  3037. StreamName: "table1",
  3038. Name: "id",
  3039. },
  3040. },
  3041. },
  3042. keys: []string{"id"},
  3043. valvars: []ast.Expr{
  3044. &ast.FieldRef{
  3045. StreamName: "src1",
  3046. Name: "id",
  3047. },
  3048. },
  3049. options: &ast.Options{
  3050. DATASOURCE: "table1",
  3051. TYPE: "sql",
  3052. KIND: "lookup",
  3053. },
  3054. conditions: nil,
  3055. }.Init(),
  3056. },
  3057. },
  3058. fields: []ast.Field{
  3059. {
  3060. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  3061. Name: "*",
  3062. AName: "",
  3063. },
  3064. },
  3065. isAggregate: false,
  3066. sendMeta: false,
  3067. }.Init(),
  3068. },
  3069. }
  3070. for i, tt := range tests {
  3071. stmt, err := xsql.NewParser(strings.NewReader(tt.sql)).Parse()
  3072. if err != nil {
  3073. t.Errorf("%d. %q: error compile sql: %s\n", i, tt.sql, err)
  3074. continue
  3075. }
  3076. p, err := createLogicalPlan(stmt, &api.RuleOption{
  3077. IsEventTime: false,
  3078. LateTol: 0,
  3079. Concurrency: 0,
  3080. BufferLength: 0,
  3081. SendMetaToSink: false,
  3082. Qos: 0,
  3083. CheckpointInterval: 0,
  3084. SendError: true,
  3085. }, store)
  3086. if !reflect.DeepEqual(tt.err, testx.Errstring(err)) {
  3087. t.Errorf("%d. %q: error mismatch:\n exp=%s\n got=%s\n\n", i, tt.sql, tt.err, err)
  3088. } else if !reflect.DeepEqual(tt.p, p) {
  3089. t.Errorf("%d. %q\n\nstmt mismatch:\n\nexp=%#v\n\ngot=%#v\n\n", i, tt.sql, render.AsCode(tt.p), render.AsCode(p))
  3090. }
  3091. }
  3092. }
  3093. func TestTransformSourceNode(t *testing.T) {
  3094. testCases := []struct {
  3095. name string
  3096. plan *DataSourcePlan
  3097. node *node.SourceNode
  3098. }{
  3099. {
  3100. name: "normal source node",
  3101. plan: &DataSourcePlan{
  3102. name: "test",
  3103. streamStmt: &ast.StreamStmt{
  3104. StreamType: ast.TypeStream,
  3105. Options: &ast.Options{
  3106. TYPE: "file",
  3107. },
  3108. },
  3109. streamFields: nil,
  3110. allMeta: false,
  3111. metaFields: []string{},
  3112. iet: false,
  3113. isBinary: false,
  3114. },
  3115. node: node.NewSourceNode("test", ast.TypeStream, nil, &ast.Options{
  3116. TYPE: "file",
  3117. }, false),
  3118. },
  3119. }
  3120. for _, tc := range testCases {
  3121. t.Run(tc.name, func(t *testing.T) {
  3122. node, err := transformSourceNode(tc.plan, nil, &api.RuleOption{})
  3123. if err != nil {
  3124. t.Errorf("unexpected error: %v", err)
  3125. return
  3126. }
  3127. if !reflect.DeepEqual(node, tc.node) {
  3128. t.Errorf("unexpected result: got %v, want %v", node, tc.node)
  3129. }
  3130. })
  3131. }
  3132. }