@@ -85,16 +85,16 @@ func (e *CTEExec) Open(ctx context.Context) (err error) {
85
85
defer e .producer .resTbl .Unlock ()
86
86
87
87
if e .producer .checkAndUpdateCorColHashCode () {
88
- e .producer .reset ()
89
- if err = e . producer . reopenTbls (); err != nil {
88
+ err = e .producer .reset ()
89
+ if err != nil {
90
90
return err
91
91
}
92
92
}
93
93
if e .producer .openErr != nil {
94
94
return e .producer .openErr
95
95
}
96
- if ! e .producer .opened {
97
- if err = e .producer .openProducer (ctx , e ); err != nil {
96
+ if ! e .producer .hasCTEResult () && ! e . producer . executorOpened {
97
+ if err = e .producer .openProducerExecutor (ctx , e ); err != nil {
98
98
return err
99
99
}
100
100
}
@@ -105,8 +105,14 @@ func (e *CTEExec) Open(ctx context.Context) (err error) {
105
105
func (e * CTEExec ) Next (ctx context.Context , req * chunk.Chunk ) (err error ) {
106
106
e .producer .resTbl .Lock ()
107
107
defer e .producer .resTbl .Unlock ()
108
- if ! e .producer .resTbl .Done () {
109
- if err = e .producer .produce (ctx ); err != nil {
108
+ if ! e .producer .hasCTEResult () {
109
+ // in case that another CTEExec call close without generate CTE result.
110
+ if ! e .producer .executorOpened {
111
+ if err = e .producer .openProducerExecutor (ctx , e ); err != nil {
112
+ return err
113
+ }
114
+ }
115
+ if err = e .producer .genCTEResult (ctx ); err != nil {
110
116
return err
111
117
}
112
118
}
@@ -128,18 +134,23 @@ func (e *CTEExec) Close() (firstErr error) {
128
134
func () {
129
135
e .producer .resTbl .Lock ()
130
136
defer e .producer .resTbl .Unlock ()
131
- if ! e .producer .closed {
137
+ if e .producer .executorOpened {
132
138
failpoint .Inject ("mock_cte_exec_panic_avoid_deadlock" , func (v failpoint.Value ) {
133
139
if ok := v .(bool ); ok {
134
140
panic ("mock mem oom panic" )
135
141
}
136
142
})
137
- // closeProducer () only close seedExec and recursiveExec, will not touch resTbl.
138
- // It means you can still read resTbl after call closeProducer ().
139
- // You can even call all three functions(openProducer/produce/closeProducer ) in CTEExec.Next().
143
+ // closeProducerExecutor () only close seedExec and recursiveExec, will not touch resTbl.
144
+ // It means you can still read resTbl after call closeProducerExecutor ().
145
+ // You can even call all three functions(openProducerExecutor/genCTEResult/closeProducerExecutor ) in CTEExec.Next().
140
146
// Separating these three function calls is only to follow the abstraction of the volcano model.
141
- err := e .producer .closeProducer ()
147
+ err := e .producer .closeProducerExecutor ()
142
148
firstErr = setFirstErr (firstErr , err , "close cte producer error" )
149
+ if ! e .producer .hasCTEResult () {
150
+ // CTE result is not generated, in this case, we reset it
151
+ err = e .producer .reset ()
152
+ firstErr = setFirstErr (firstErr , err , "close cte producer error" )
153
+ }
143
154
}
144
155
}()
145
156
err := e .baseExecutor .Close ()
@@ -154,10 +165,10 @@ func (e *CTEExec) reset() {
154
165
}
155
166
156
167
type cteProducer struct {
157
- // opened should be false when not open or open fail(a.k.a. openErr != nil)
158
- opened bool
159
- produced bool
160
- closed bool
168
+ // executorOpened is used to indicate whether the executor(seedExec/recursiveExec) is opened.
169
+ // when executorOpened is true, the executor is opened, otherwise it means the executor is
170
+ // not opened or is already closed.
171
+ executorOpened bool
161
172
162
173
// cteProducer is shared by multiple operators, so if the first operator tries to open
163
174
// and got error, the second should return open error directly instead of open again.
@@ -196,14 +207,10 @@ type cteProducer struct {
196
207
corColHashCodes [][]byte
197
208
}
198
209
199
- func (p * cteProducer ) openProducer (ctx context.Context , cteExec * CTEExec ) (err error ) {
210
+ func (p * cteProducer ) openProducerExecutor (ctx context.Context , cteExec * CTEExec ) (err error ) {
200
211
defer func () {
201
212
p .openErr = err
202
- if err == nil {
203
- p .opened = true
204
- } else {
205
- p .opened = false
206
- }
213
+ p .executorOpened = true
207
214
}()
208
215
if p .seedExec == nil {
209
216
return errors .New ("seedExec for CTEExec is nil" )
@@ -246,7 +253,7 @@ func (p *cteProducer) openProducer(ctx context.Context, cteExec *CTEExec) (err e
246
253
return nil
247
254
}
248
255
249
- func (p * cteProducer ) closeProducer () (firstErr error ) {
256
+ func (p * cteProducer ) closeProducerExecutor () (firstErr error ) {
250
257
err := p .seedExec .Close ()
251
258
firstErr = setFirstErr (firstErr , err , "close seedExec err" )
252
259
if p .recursiveExec != nil {
@@ -264,7 +271,7 @@ func (p *cteProducer) closeProducer() (firstErr error) {
264
271
// because ExplainExec still needs tracker to get mem usage info.
265
272
p .memTracker = nil
266
273
p .diskTracker = nil
267
- p .closed = true
274
+ p .executorOpened = false
268
275
return
269
276
}
270
277
@@ -331,7 +338,13 @@ func (p *cteProducer) nextChunkLimit(cteExec *CTEExec, req *chunk.Chunk) error {
331
338
return nil
332
339
}
333
340
334
- func (p * cteProducer ) produce (ctx context.Context ) (err error ) {
341
+ func (p * cteProducer ) hasCTEResult () bool {
342
+ return p .resTbl .Done ()
343
+ }
344
+
345
+ // genCTEResult generates the result of CTE, and stores the result in resTbl.
346
+ // This is a synchronous function, which means it will block until the result is generated.
347
+ func (p * cteProducer ) genCTEResult (ctx context.Context ) (err error ) {
335
348
if p .resTbl .Error () != nil {
336
349
return p .resTbl .Error ()
337
350
}
@@ -524,14 +537,18 @@ func (p *cteProducer) setupTblsForNewIteration() (err error) {
524
537
return nil
525
538
}
526
539
527
- func (p * cteProducer ) reset () {
540
+ func (p * cteProducer ) reset () error {
528
541
p .curIter = 0
529
542
p .hashTbl = nil
530
-
531
- p .opened = false
543
+ p .executorOpened = false
532
544
p .openErr = nil
533
- p .produced = false
534
- p .closed = false
545
+
546
+ // Normally we need to setup tracker after calling Reopen(),
547
+ // But reopen resTbl means we need to call genCTEResult() again, it will setup tracker.
548
+ if err := p .resTbl .Reopen (); err != nil {
549
+ return err
550
+ }
551
+ return p .iterInTbl .Reopen ()
535
552
}
536
553
537
554
func (p * cteProducer ) resetTracker () {
@@ -545,18 +562,6 @@ func (p *cteProducer) resetTracker() {
545
562
}
546
563
}
547
564
548
- func (p * cteProducer ) reopenTbls () (err error ) {
549
- if p .isDistinct {
550
- p .hashTbl = newConcurrentMapHashTable ()
551
- }
552
- // Normally we need to setup tracker after calling Reopen(),
553
- // But reopen resTbl means we need to call produce() again, it will setup tracker.
554
- if err := p .resTbl .Reopen (); err != nil {
555
- return err
556
- }
557
- return p .iterInTbl .Reopen ()
558
- }
559
-
560
565
// Check if tbl meets the requirement of limit.
561
566
func (p * cteProducer ) limitDone (tbl cteutil.Storage ) bool {
562
567
return p .hasLimit && uint64 (tbl .NumRows ()) >= p .limitEnd
0 commit comments