parallel_eager.c 9.6 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307
  1. /* StarPU --- Runtime system for heterogeneous multicore architectures.
  2. *
  3. * Copyright (C) 2011-2013 Université de Bordeaux 1
  4. * Copyright (C) 2011 Télécom-SudParis
  5. * Copyright (C) 2011-2013 INRIA
  6. *
  7. * StarPU is free software; you can redistribute it and/or modify
  8. * it under the terms of the GNU Lesser General Public License as published by
  9. * the Free Software Foundation; either version 2.1 of the License, or (at
  10. * your option) any later version.
  11. *
  12. * StarPU is distributed in the hope that it will be useful, but
  13. * WITHOUT ANY WARRANTY; without even the implied warranty of
  14. * MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE.
  15. *
  16. * See the GNU Lesser General Public License in COPYING.LGPL for more details.
  17. */
  18. #include <sched_policies/fifo_queues.h>
  19. #include <core/detect_combined_workers.h>
  20. #include <starpu_scheduler.h>
  21. #include <core/workers.h>
  22. struct _starpu_peager_data
  23. {
  24. struct _starpu_fifo_taskq *fifo;
  25. struct _starpu_fifo_taskq *local_fifo[STARPU_NMAXWORKERS];
  26. int master_id[STARPU_NMAXWORKERS];
  27. starpu_pthread_mutex_t policy_mutex;
  28. };
  29. #define STARPU_NMAXCOMBINED_WORKERS 520
  30. /* instead of STARPU_NMAXCOMBINED_WORKERS, we should use some "MAX combination .."*/
  31. static int possible_combinations_cnt[STARPU_NMAXWORKERS];
  32. static int possible_combinations[STARPU_NMAXWORKERS][STARPU_NMAXCOMBINED_WORKERS];
  33. static int possible_combinations_size[STARPU_NMAXWORKERS][STARPU_NMAXCOMBINED_WORKERS];
  34. /*!!!!!!! It doesn't work with several contexts because the combined workers are constructed
  35. from the workers available to the program, and not to the context !!!!!!!!!!!!!!!!!!!!!!!
  36. */
  37. static void peager_add_workers(unsigned sched_ctx_id, int *workerids, unsigned nworkers)
  38. {
  39. _starpu_sched_find_worker_combinations(workerids, nworkers);
  40. struct _starpu_peager_data *data = (struct _starpu_peager_data*)starpu_sched_ctx_get_policy_data(sched_ctx_id);
  41. unsigned nbasic_workers = starpu_worker_get_count();
  42. unsigned ncombined_workers= starpu_combined_worker_get_count();
  43. unsigned workerid, i;
  44. /* Find the master of each worker. We first assign the worker as its
  45. * own master, and then iterate over the different worker combinations
  46. * to find the biggest combination containing this worker. */
  47. for(i = 0; i < nworkers; i++)
  48. {
  49. workerid = workerids[i];
  50. starpu_sched_ctx_worker_shares_tasks_lists(workerid, sched_ctx_id);
  51. int cnt = possible_combinations_cnt[workerid]++;
  52. possible_combinations[workerid][cnt] = workerid;
  53. possible_combinations_size[workerid][cnt] = 1;
  54. data->master_id[workerid] = workerid;
  55. }
  56. for (i = 0; i < ncombined_workers; i++)
  57. {
  58. workerid = nbasic_workers + i;
  59. /* Note that we ASSUME that the workers are sorted by size ! */
  60. int *workers;
  61. int size;
  62. starpu_combined_worker_get_description(workerid, &size, &workers);
  63. int master = workers[0];
  64. int j;
  65. for (j = 0; j < size; j++)
  66. {
  67. if (data->master_id[workers[j]] > master)
  68. data->master_id[workers[j]] = master;
  69. int cnt = possible_combinations_cnt[workers[j]]++;
  70. possible_combinations[workers[j]][cnt] = workerid;
  71. possible_combinations_size[workers[j]][cnt] = size;
  72. }
  73. }
  74. for(i = 0; i < nworkers; i++)
  75. {
  76. workerid = workerids[i];
  77. /* slaves pick up tasks from their local queue, their master
  78. * will put tasks directly in that local list when a parallel
  79. * tasks comes. */
  80. data->local_fifo[workerid] = _starpu_create_fifo();
  81. }
  82. #if 0
  83. for(i = 0; i < nworkers; i++)
  84. {
  85. workerid = workerids[i];
  86. fprintf(stderr, "MASTER of %d = %d\n", workerid, master_id[workerid]);
  87. }
  88. #endif
  89. }
  90. static void peager_remove_workers(unsigned sched_ctx_id, int *workerids, unsigned nworkers)
  91. {
  92. struct _starpu_peager_data *data = (struct _starpu_peager_data*)starpu_sched_ctx_get_policy_data(sched_ctx_id);
  93. int workerid;
  94. unsigned i;
  95. for(i = 0; i < nworkers; i++)
  96. {
  97. workerid = workerids[i];
  98. if(!starpu_worker_is_combined_worker(workerid))
  99. _starpu_destroy_fifo(data->local_fifo[workerid]);
  100. }
  101. }
  102. static void initialize_peager_policy(unsigned sched_ctx_id)
  103. {
  104. starpu_sched_ctx_create_worker_collection(sched_ctx_id, STARPU_WORKER_LIST);
  105. struct _starpu_peager_data *data = (struct _starpu_peager_data*)malloc(sizeof(struct _starpu_peager_data));
  106. /* masters pick tasks from that queue */
  107. data->fifo = _starpu_create_fifo();
  108. starpu_sched_ctx_set_policy_data(sched_ctx_id, (void*)data);
  109. STARPU_PTHREAD_MUTEX_INIT(&data->policy_mutex, NULL);
  110. }
  111. static void deinitialize_peager_policy(unsigned sched_ctx_id)
  112. {
  113. /* TODO check that there is no task left in the queue */
  114. struct _starpu_peager_data *data = (struct _starpu_peager_data*)starpu_sched_ctx_get_policy_data(sched_ctx_id);
  115. /* deallocate the job queue */
  116. _starpu_destroy_fifo(data->fifo);
  117. starpu_sched_ctx_delete_worker_collection(sched_ctx_id);
  118. STARPU_PTHREAD_MUTEX_DESTROY(&data->policy_mutex);
  119. free(data);
  120. }
  121. static int push_task_peager_policy(struct starpu_task *task)
  122. {
  123. unsigned sched_ctx_id = task->sched_ctx;
  124. int ret_val = -1;
  125. struct _starpu_peager_data *data = (struct _starpu_peager_data*)starpu_sched_ctx_get_policy_data(sched_ctx_id);
  126. STARPU_PTHREAD_MUTEX_LOCK(&data->policy_mutex);
  127. ret_val = _starpu_fifo_push_task(data->fifo, task);
  128. starpu_push_task_end(task);
  129. STARPU_PTHREAD_MUTEX_UNLOCK(&data->policy_mutex);
  130. /*if there are no tasks block */
  131. /* wake people waiting for a task */
  132. int worker = -1;
  133. struct starpu_worker_collection *workers = starpu_sched_ctx_get_worker_collection(sched_ctx_id);
  134. struct starpu_sched_ctx_iterator it;
  135. if(workers->init_iterator)
  136. workers->init_iterator(workers, &it);
  137. while(workers->has_next(workers, &it))
  138. {
  139. worker = workers->get_next(workers, &it);
  140. int master = data->master_id[worker];
  141. /* If this is not a CPU or a MIC, then the worker simply grabs tasks from the fifo */
  142. if ((!starpu_worker_is_combined_worker(worker) &&
  143. starpu_worker_get_type(worker) != STARPU_MIC_WORKER &&
  144. starpu_worker_get_type(worker) != STARPU_CPU_WORKER)
  145. || (master == worker))
  146. {
  147. starpu_pthread_mutex_t *sched_mutex;
  148. starpu_pthread_cond_t *sched_cond;
  149. starpu_worker_get_sched_condition(worker, &sched_mutex, &sched_cond);
  150. STARPU_PTHREAD_MUTEX_LOCK(sched_mutex);
  151. STARPU_PTHREAD_COND_SIGNAL(sched_cond);
  152. STARPU_PTHREAD_MUTEX_UNLOCK(sched_mutex);
  153. }
  154. }
  155. return ret_val;
  156. }
  157. static struct starpu_task *pop_task_peager_policy(unsigned sched_ctx_id)
  158. {
  159. struct _starpu_peager_data *data = (struct _starpu_peager_data*)starpu_sched_ctx_get_policy_data(sched_ctx_id);
  160. int workerid = starpu_worker_get_id();
  161. /* If this is not a CPU or a MIC, then the worker simply grabs tasks from the fifo */
  162. if (starpu_worker_get_type(workerid) != STARPU_CPU_WORKER && starpu_worker_get_type(workerid) != STARPU_MIC_WORKER)
  163. {
  164. struct starpu_task *task = NULL;
  165. STARPU_PTHREAD_MUTEX_LOCK(&data->policy_mutex);
  166. task = _starpu_fifo_pop_task(data->fifo, workerid);
  167. STARPU_PTHREAD_MUTEX_UNLOCK(&data->policy_mutex);
  168. return task;
  169. }
  170. int master = data->master_id[workerid];
  171. //_STARPU_DEBUG("workerid:%d, master:%d\n",workerid,master);
  172. if (master == workerid)
  173. {
  174. /* The worker is a master */
  175. struct starpu_task *task = NULL;
  176. STARPU_PTHREAD_MUTEX_LOCK(&data->policy_mutex);
  177. task = _starpu_fifo_pop_task(data->fifo, workerid);
  178. STARPU_PTHREAD_MUTEX_UNLOCK(&data->policy_mutex);
  179. if (!task)
  180. return NULL;
  181. /* Find the largest compatible worker combination */
  182. int best_size = -1;
  183. int best_workerid = -1;
  184. int i;
  185. for (i = 0; i < possible_combinations_cnt[master]; i++)
  186. {
  187. if (possible_combinations_size[workerid][i] > best_size)
  188. {
  189. int combined_worker = possible_combinations[workerid][i];
  190. if (starpu_combined_worker_can_execute_task(combined_worker, task, 0))
  191. {
  192. best_size = possible_combinations_size[workerid][i];
  193. best_workerid = combined_worker;
  194. }
  195. }
  196. }
  197. /* In case nobody can execute this task, we let the master
  198. * worker take it anyway, so that it can discard it afterward.
  199. * */
  200. if (best_workerid == -1)
  201. return task;
  202. /* Is this a basic worker or a combined worker ? */
  203. int nbasic_workers = (int)starpu_worker_get_count();
  204. int is_basic_worker = (best_workerid < nbasic_workers);
  205. if (is_basic_worker)
  206. {
  207. /* The master is alone */
  208. return task;
  209. }
  210. else
  211. {
  212. starpu_parallel_task_barrier_init(task, best_workerid);
  213. int worker_size = 0;
  214. int *combined_workerid;
  215. starpu_combined_worker_get_description(best_workerid, &worker_size, &combined_workerid);
  216. /* Dispatch task aliases to the different slaves */
  217. for (i = 1; i < worker_size; i++)
  218. {
  219. struct starpu_task *alias = starpu_task_dup(task);
  220. int local_worker = combined_workerid[i];
  221. starpu_pthread_mutex_t *sched_mutex;
  222. starpu_pthread_cond_t *sched_cond;
  223. starpu_worker_get_sched_condition(local_worker, &sched_mutex, &sched_cond);
  224. STARPU_PTHREAD_MUTEX_LOCK(sched_mutex);
  225. _starpu_fifo_push_task(data->local_fifo[local_worker], alias);
  226. STARPU_PTHREAD_COND_SIGNAL(sched_cond);
  227. STARPU_PTHREAD_MUTEX_UNLOCK(sched_mutex);
  228. }
  229. /* The master also manipulated an alias */
  230. struct starpu_task *master_alias = starpu_task_dup(task);
  231. return master_alias;
  232. }
  233. }
  234. else
  235. {
  236. /* The worker is a slave */
  237. return _starpu_fifo_pop_task(data->local_fifo[workerid], workerid);
  238. }
  239. }
  240. struct starpu_sched_policy _starpu_sched_peager_policy =
  241. {
  242. .init_sched = initialize_peager_policy,
  243. .deinit_sched = deinitialize_peager_policy,
  244. .add_workers = peager_add_workers,
  245. .remove_workers = peager_remove_workers,
  246. .push_task = push_task_peager_policy,
  247. .pop_task = pop_task_peager_policy,
  248. .pre_exec_hook = NULL,
  249. .post_exec_hook = NULL,
  250. .pop_every_task = NULL,
  251. .policy_name = "peager",
  252. .policy_description = "parallel eager policy"
  253. };