@@ -6,12 +6,15 @@ use futures::{
6
6
stream:: { self , BoxStream } ,
7
7
Stream , StreamExt ,
8
8
} ;
9
- use log:: debug;
9
+ use log:: { debug, warn } ;
10
10
11
11
use crate :: {
12
12
ec:: EcSchema ,
13
- hdfs:: connection:: { DatanodeConnection , Op } ,
14
- proto:: { common, hdfs} ,
13
+ hdfs:: connection:: { DatanodeConnection , Op , DATANODE_CACHE } ,
14
+ proto:: {
15
+ common,
16
+ hdfs:: { self , BlockOpResponseProto } ,
17
+ } ,
15
18
HdfsError , Result ,
16
19
} ;
17
20
@@ -32,6 +35,55 @@ pub(crate) fn get_block_stream(
32
35
}
33
36
}
34
37
38
+ /// Connects to a DataNode to do a read, attempting to used cached connections.
39
+ async fn connect_and_send (
40
+ url : & str ,
41
+ block : & hdfs:: ExtendedBlockProto ,
42
+ token : common:: TokenProto ,
43
+ offset : u64 ,
44
+ len : u64 ,
45
+ ) -> Result < ( DatanodeConnection , BlockOpResponseProto ) > {
46
+ let mut remaining_attempts = 2 ;
47
+ while remaining_attempts > 0 {
48
+ if let Some ( mut conn) = DATANODE_CACHE . get ( url) {
49
+ let message = hdfs:: OpReadBlockProto {
50
+ header : conn. build_header ( block, Some ( token. clone ( ) ) ) ,
51
+ offset,
52
+ len,
53
+ send_checksums : Some ( true ) ,
54
+ ..Default :: default ( )
55
+ } ;
56
+ debug ! ( "Block read op request {:?}" , & message) ;
57
+ match conn. send ( Op :: ReadBlock , & message) . await {
58
+ Ok ( response) => {
59
+ debug ! ( "Block read op response {:?}" , response) ;
60
+ return Ok ( ( conn, response) ) ;
61
+ }
62
+ Err ( e) => {
63
+ warn ! ( "Failed to use cached connection: {:?}" , e) ;
64
+ }
65
+ }
66
+ } else {
67
+ break ;
68
+ }
69
+ remaining_attempts -= 1 ;
70
+ }
71
+ let mut conn = DatanodeConnection :: connect ( url) . await ?;
72
+
73
+ let message = hdfs:: OpReadBlockProto {
74
+ header : conn. build_header ( block, Some ( token) ) ,
75
+ offset,
76
+ len,
77
+ send_checksums : Some ( true ) ,
78
+ ..Default :: default ( )
79
+ } ;
80
+
81
+ debug ! ( "Block read op request {:?}" , & message) ;
82
+ let response = conn. send ( Op :: ReadBlock , & message) . await ?;
83
+ debug ! ( "Block read op response {:?}" , response) ;
84
+ Ok ( ( conn, response) )
85
+ }
86
+
35
87
struct ReplicatedBlockStream {
36
88
block : hdfs:: LocatedBlockProto ,
37
89
offset : usize ,
@@ -63,24 +115,18 @@ impl ReplicatedBlockStream {
63
115
) ) ;
64
116
}
65
117
}
66
- let datanode = & self . block . locs [ self . current_replica ] . id ;
67
- let mut connection =
68
- DatanodeConnection :: connect ( & format ! ( "{}:{}" , datanode. ip_addr, datanode. xfer_port) )
69
- . await ?;
70
-
71
- let message = hdfs:: OpReadBlockProto {
72
- header : connection. build_header ( & self . block . b , Some ( self . block . block_token . clone ( ) ) ) ,
73
- offset : self . offset as u64 ,
74
- len : self . len as u64 ,
75
- send_checksums : Some ( true ) ,
76
- ..Default :: default ( )
77
- } ;
78
118
79
- debug ! ( "Block read op request {:?}" , & message) ;
119
+ let datanode = & self . block . locs [ self . current_replica ] . id ;
120
+ let datanode_url = format ! ( "{}:{}" , datanode. ip_addr, datanode. xfer_port) ;
80
121
81
- connection. send ( Op :: ReadBlock , & message) . await ?;
82
- let response = connection. read_block_op_response ( ) . await ?;
83
- debug ! ( "Block read op response {:?}" , response) ;
122
+ let ( connection, response) = connect_and_send (
123
+ & datanode_url,
124
+ & self . block . b ,
125
+ self . block . block_token . clone ( ) ,
126
+ self . offset as u64 ,
127
+ self . len as u64 ,
128
+ )
129
+ . await ?;
84
130
85
131
if response. status ( ) != hdfs:: Status :: Success {
86
132
return Err ( HdfsError :: DataTransferError ( response. message ( ) . to_string ( ) ) ) ;
@@ -96,13 +142,20 @@ impl ReplicatedBlockStream {
96
142
if self . connection . is_none ( ) {
97
143
self . select_next_datanode ( ) . await ?;
98
144
}
99
- let conn = self . connection . as_mut ( ) . unwrap ( ) ;
100
145
101
146
if self . len == 0 {
147
+ let mut conn = self . connection . take ( ) . unwrap ( ) ;
148
+
149
+ // Read the final empty packet
150
+ conn. read_packet ( ) . await ?;
151
+
102
152
conn. send_read_success ( ) . await ?;
153
+ DATANODE_CACHE . release ( conn) ;
103
154
return Ok ( None ) ;
104
155
}
105
156
157
+ let conn = self . connection . as_mut ( ) . unwrap ( ) ;
158
+
106
159
let packet = conn. read_packet ( ) . await ?;
107
160
108
161
let packet_offset = if self . offset > packet. header . offset_in_block as usize {
@@ -336,29 +389,22 @@ impl StripedBlockStream {
336
389
return Ok ( ( ) ) ;
337
390
}
338
391
339
- let mut conn =
340
- DatanodeConnection :: connect ( & format ! ( "{}:{}" , datanode. ip_addr, datanode. xfer_port) )
341
- . await ?;
342
-
343
- let message = hdfs:: OpReadBlockProto {
344
- header : conn. build_header ( block, Some ( token. clone ( ) ) ) ,
345
- offset : offset as u64 ,
346
- len : len as u64 ,
347
- send_checksums : Some ( true ) ,
348
- ..Default :: default ( )
349
- } ;
350
- debug ! ( "Block read op request {:?}" , & message) ;
351
-
352
- conn. send ( Op :: ReadBlock , & message) . await ?;
353
- let response = conn. read_block_op_response ( ) . await ?;
354
- debug ! ( "Block read op response {:?}" , response) ;
392
+ let datanode_url = format ! ( "{}:{}" , datanode. ip_addr, datanode. xfer_port) ;
393
+ let ( mut connection, response) = connect_and_send (
394
+ & datanode_url,
395
+ block,
396
+ token. clone ( ) ,
397
+ offset as u64 ,
398
+ len as u64 ,
399
+ )
400
+ . await ?;
355
401
356
402
if response. status ( ) != hdfs:: Status :: Success {
357
403
return Err ( HdfsError :: DataTransferError ( response. message ( ) . to_string ( ) ) ) ;
358
404
}
359
405
360
406
// First handle the offset into the first packet
361
- let mut packet = conn . read_packet ( ) . await ?;
407
+ let mut packet = connection . read_packet ( ) . await ?;
362
408
let packet_offset = offset - packet. header . offset_in_block as usize ;
363
409
let data_len = packet. header . data_len as usize - packet_offset;
364
410
let data_to_read = usize:: min ( data_len, len) ;
@@ -368,7 +414,7 @@ impl StripedBlockStream {
368
414
buf. put ( packet_data. slice ( packet_offset..( packet_offset + data_to_read) ) ) ;
369
415
370
416
while data_left > 0 {
371
- packet = conn . read_packet ( ) . await ?;
417
+ packet = connection . read_packet ( ) . await ?;
372
418
// TODO: Error checking
373
419
let data_to_read = usize:: min ( data_left, packet. header . data_len as usize ) ;
374
420
buf. put (
@@ -380,7 +426,9 @@ impl StripedBlockStream {
380
426
}
381
427
382
428
// There should be one last empty packet after we are done
383
- conn. read_packet ( ) . await ?;
429
+ connection. read_packet ( ) . await ?;
430
+ connection. send_read_success ( ) . await ?;
431
+ DATANODE_CACHE . release ( connection) ;
384
432
385
433
Ok ( ( ) )
386
434
}
0 commit comments